mirror of
				https://github.com/ClusterCockpit/cc-metric-store.git
				synced 2025-10-24 23:05:07 +02:00 
			
		
		
		
	Continue restructuring. Intermediate state.
This commit is contained in:
		
							
								
								
									
										597
									
								
								internal/memorystore/archive.go
									
									
									
									
									
										Normal file
									
								
							
							
						
						
									
										597
									
								
								internal/memorystore/archive.go
									
									
									
									
									
										Normal file
									
								
							| @@ -0,0 +1,597 @@ | ||||
| package memorystore | ||||
|  | ||||
| import ( | ||||
| 	"archive/zip" | ||||
| 	"bufio" | ||||
| 	"encoding/json" | ||||
| 	"errors" | ||||
| 	"fmt" | ||||
| 	"io" | ||||
| 	"io/fs" | ||||
| 	"log" | ||||
| 	"os" | ||||
| 	"path" | ||||
| 	"path/filepath" | ||||
| 	"runtime" | ||||
| 	"sort" | ||||
| 	"strconv" | ||||
| 	"strings" | ||||
| 	"sync" | ||||
| 	"sync/atomic" | ||||
| ) | ||||
|  | ||||
| // Whenever changed, update MarshalJSON as well! | ||||
| type CheckpointMetrics struct { | ||||
| 	Frequency int64   `json:"frequency"` | ||||
| 	Start     int64   `json:"start"` | ||||
| 	Data      []Float `json:"data"` | ||||
| } | ||||
|  | ||||
| // As `Float` implements a custom MarshalJSON() function, | ||||
| // serializing an array of such types has more overhead | ||||
| // than one would assume (because of extra allocations, interfaces and so on). | ||||
| func (cm *CheckpointMetrics) MarshalJSON() ([]byte, error) { | ||||
| 	buf := make([]byte, 0, 128+len(cm.Data)*8) | ||||
| 	buf = append(buf, `{"frequency":`...) | ||||
| 	buf = strconv.AppendInt(buf, cm.Frequency, 10) | ||||
| 	buf = append(buf, `,"start":`...) | ||||
| 	buf = strconv.AppendInt(buf, cm.Start, 10) | ||||
| 	buf = append(buf, `,"data":[`...) | ||||
| 	for i, x := range cm.Data { | ||||
| 		if i != 0 { | ||||
| 			buf = append(buf, ',') | ||||
| 		} | ||||
| 		if x.IsNaN() { | ||||
| 			buf = append(buf, `null`...) | ||||
| 		} else { | ||||
| 			buf = strconv.AppendFloat(buf, float64(x), 'f', 1, 32) | ||||
| 		} | ||||
| 	} | ||||
| 	buf = append(buf, `]}`...) | ||||
| 	return buf, nil | ||||
| } | ||||
|  | ||||
| type CheckpointFile struct { | ||||
| 	From     int64                         `json:"from"` | ||||
| 	To       int64                         `json:"to"` | ||||
| 	Metrics  map[string]*CheckpointMetrics `json:"metrics"` | ||||
| 	Children map[string]*CheckpointFile    `json:"children"` | ||||
| } | ||||
|  | ||||
| var ErrNoNewData error = errors.New("all data already archived") | ||||
|  | ||||
| var NumWorkers int = 4 | ||||
|  | ||||
| func init() { | ||||
| 	maxWorkers := 10 | ||||
| 	NumWorkers = runtime.NumCPU()/2 + 1 | ||||
| 	if NumWorkers > maxWorkers { | ||||
| 		NumWorkers = maxWorkers | ||||
| 	} | ||||
| } | ||||
|  | ||||
| // Metrics stored at the lowest 2 levels are not stored away (root and cluster)! | ||||
| // On a per-host basis a new JSON file is created. I have no idea if this will scale. | ||||
| // The good thing: Only a host at a time is locked, so this function can run | ||||
| // in parallel to writes/reads. | ||||
| func (m *MemoryStore) ToCheckpoint(dir string, from, to int64) (int, error) { | ||||
| 	levels := make([]*Level, 0) | ||||
| 	selectors := make([][]string, 0) | ||||
| 	m.root.lock.RLock() | ||||
| 	for sel1, l1 := range m.root.children { | ||||
| 		l1.lock.RLock() | ||||
| 		for sel2, l2 := range l1.children { | ||||
| 			levels = append(levels, l2) | ||||
| 			selectors = append(selectors, []string{sel1, sel2}) | ||||
| 		} | ||||
| 		l1.lock.RUnlock() | ||||
| 	} | ||||
| 	m.root.lock.RUnlock() | ||||
|  | ||||
| 	type workItem struct { | ||||
| 		level    *Level | ||||
| 		dir      string | ||||
| 		selector []string | ||||
| 	} | ||||
|  | ||||
| 	n, errs := int32(0), int32(0) | ||||
|  | ||||
| 	var wg sync.WaitGroup | ||||
| 	wg.Add(NumWorkers) | ||||
| 	work := make(chan workItem, NumWorkers*2) | ||||
| 	for worker := 0; worker < NumWorkers; worker++ { | ||||
| 		go func() { | ||||
| 			defer wg.Done() | ||||
|  | ||||
| 			for workItem := range work { | ||||
| 				if err := workItem.level.toCheckpoint(workItem.dir, from, to, m); err != nil { | ||||
| 					if err == ErrNoNewData { | ||||
| 						continue | ||||
| 					} | ||||
|  | ||||
| 					log.Printf("error while checkpointing %#v: %s", workItem.selector, err.Error()) | ||||
| 					atomic.AddInt32(&errs, 1) | ||||
| 				} else { | ||||
| 					atomic.AddInt32(&n, 1) | ||||
| 				} | ||||
| 			} | ||||
| 		}() | ||||
| 	} | ||||
|  | ||||
| 	for i := 0; i < len(levels); i++ { | ||||
| 		dir := path.Join(dir, path.Join(selectors[i]...)) | ||||
| 		work <- workItem{ | ||||
| 			level:    levels[i], | ||||
| 			dir:      dir, | ||||
| 			selector: selectors[i], | ||||
| 		} | ||||
| 	} | ||||
|  | ||||
| 	close(work) | ||||
| 	wg.Wait() | ||||
|  | ||||
| 	if errs > 0 { | ||||
| 		return int(n), fmt.Errorf("%d errors happend while creating checkpoints (%d successes)", errs, n) | ||||
| 	} | ||||
| 	return int(n), nil | ||||
| } | ||||
|  | ||||
| func (l *Level) toCheckpointFile(from, to int64, m *MemoryStore) (*CheckpointFile, error) { | ||||
| 	l.lock.RLock() | ||||
| 	defer l.lock.RUnlock() | ||||
|  | ||||
| 	retval := &CheckpointFile{ | ||||
| 		From:     from, | ||||
| 		To:       to, | ||||
| 		Metrics:  make(map[string]*CheckpointMetrics), | ||||
| 		Children: make(map[string]*CheckpointFile), | ||||
| 	} | ||||
|  | ||||
| 	for metric, minfo := range m.Metrics { | ||||
| 		b := l.metrics[minfo.offset] | ||||
| 		if b == nil { | ||||
| 			continue | ||||
| 		} | ||||
|  | ||||
| 		allArchived := true | ||||
| 		b.iterFromTo(from, to, func(b *buffer) error { | ||||
| 			if !b.archived { | ||||
| 				allArchived = false | ||||
| 			} | ||||
| 			return nil | ||||
| 		}) | ||||
|  | ||||
| 		if allArchived { | ||||
| 			continue | ||||
| 		} | ||||
|  | ||||
| 		data := make([]Float, (to-from)/b.frequency+1) | ||||
| 		data, start, end, err := b.read(from, to, data) | ||||
| 		if err != nil { | ||||
| 			return nil, err | ||||
| 		} | ||||
|  | ||||
| 		for i := int((end - start) / b.frequency); i < len(data); i++ { | ||||
| 			data[i] = NaN | ||||
| 		} | ||||
|  | ||||
| 		retval.Metrics[metric] = &CheckpointMetrics{ | ||||
| 			Frequency: b.frequency, | ||||
| 			Start:     start, | ||||
| 			Data:      data, | ||||
| 		} | ||||
| 	} | ||||
|  | ||||
| 	for name, child := range l.children { | ||||
| 		val, err := child.toCheckpointFile(from, to, m) | ||||
| 		if err != nil { | ||||
| 			return nil, err | ||||
| 		} | ||||
|  | ||||
| 		if val != nil { | ||||
| 			retval.Children[name] = val | ||||
| 		} | ||||
| 	} | ||||
|  | ||||
| 	if len(retval.Children) == 0 && len(retval.Metrics) == 0 { | ||||
| 		return nil, nil | ||||
| 	} | ||||
|  | ||||
| 	return retval, nil | ||||
| } | ||||
|  | ||||
| func (l *Level) toCheckpoint(dir string, from, to int64, m *MemoryStore) error { | ||||
| 	cf, err := l.toCheckpointFile(from, to, m) | ||||
| 	if err != nil { | ||||
| 		return err | ||||
| 	} | ||||
|  | ||||
| 	if cf == nil { | ||||
| 		return ErrNoNewData | ||||
| 	} | ||||
|  | ||||
| 	filepath := path.Join(dir, fmt.Sprintf("%d.json", from)) | ||||
| 	f, err := os.OpenFile(filepath, os.O_CREATE|os.O_WRONLY, 0o644) | ||||
| 	if err != nil && os.IsNotExist(err) { | ||||
| 		err = os.MkdirAll(dir, 0o755) | ||||
| 		if err == nil { | ||||
| 			f, err = os.OpenFile(filepath, os.O_CREATE|os.O_WRONLY, 0o644) | ||||
| 		} | ||||
| 	} | ||||
| 	if err != nil { | ||||
| 		return err | ||||
| 	} | ||||
| 	defer f.Close() | ||||
|  | ||||
| 	bw := bufio.NewWriter(f) | ||||
| 	if err = json.NewEncoder(bw).Encode(cf); err != nil { | ||||
| 		return err | ||||
| 	} | ||||
|  | ||||
| 	return bw.Flush() | ||||
| } | ||||
|  | ||||
| // Metrics stored at the lowest 2 levels are not loaded (root and cluster)! | ||||
| // This function can only be called once and before the very first write or read. | ||||
| // Different host's data is loaded to memory in parallel. | ||||
| func (m *MemoryStore) FromCheckpoint(dir string, from int64) (int, error) { | ||||
| 	var wg sync.WaitGroup | ||||
| 	work := make(chan [2]string, NumWorkers) | ||||
| 	n, errs := int32(0), int32(0) | ||||
|  | ||||
| 	wg.Add(NumWorkers) | ||||
| 	for worker := 0; worker < NumWorkers; worker++ { | ||||
| 		go func() { | ||||
| 			defer wg.Done() | ||||
| 			for host := range work { | ||||
| 				lvl := m.root.findLevelOrCreate(host[:], len(m.Metrics)) | ||||
| 				nn, err := lvl.fromCheckpoint(filepath.Join(dir, host[0], host[1]), from, m) | ||||
| 				if err != nil { | ||||
| 					log.Fatalf("error while loading checkpoints: %s", err.Error()) | ||||
| 					atomic.AddInt32(&errs, 1) | ||||
| 				} | ||||
| 				atomic.AddInt32(&n, int32(nn)) | ||||
| 			} | ||||
| 		}() | ||||
| 	} | ||||
|  | ||||
| 	i := 0 | ||||
| 	clustersDir, err := os.ReadDir(dir) | ||||
| 	for _, clusterDir := range clustersDir { | ||||
| 		if !clusterDir.IsDir() { | ||||
| 			err = errors.New("expected only directories at first level of checkpoints/ directory") | ||||
| 			goto done | ||||
| 		} | ||||
|  | ||||
| 		hostsDir, e := os.ReadDir(filepath.Join(dir, clusterDir.Name())) | ||||
| 		if e != nil { | ||||
| 			err = e | ||||
| 			goto done | ||||
| 		} | ||||
|  | ||||
| 		for _, hostDir := range hostsDir { | ||||
| 			if !hostDir.IsDir() { | ||||
| 				err = errors.New("expected only directories at second level of checkpoints/ directory") | ||||
| 				goto done | ||||
| 			} | ||||
|  | ||||
| 			i++ | ||||
| 			if i%NumWorkers == 0 && i > 100 { | ||||
| 				// Forcing garbage collection runs here regulary during the loading of checkpoints | ||||
| 				// will decrease the total heap size after loading everything back to memory is done. | ||||
| 				// While loading data, the heap will grow fast, so the GC target size will double | ||||
| 				// almost always. By forcing GCs here, we can keep it growing more slowly so that | ||||
| 				// at the end, less memory is wasted. | ||||
| 				runtime.GC() | ||||
| 			} | ||||
|  | ||||
| 			work <- [2]string{clusterDir.Name(), hostDir.Name()} | ||||
| 		} | ||||
| 	} | ||||
| done: | ||||
| 	close(work) | ||||
| 	wg.Wait() | ||||
|  | ||||
| 	if err != nil { | ||||
| 		return int(n), err | ||||
| 	} | ||||
|  | ||||
| 	if errs > 0 { | ||||
| 		return int(n), fmt.Errorf("%d errors happend while creating checkpoints (%d successes)", errs, n) | ||||
| 	} | ||||
| 	return int(n), nil | ||||
| } | ||||
|  | ||||
| func (l *Level) loadFile(cf *CheckpointFile, m *MemoryStore) error { | ||||
| 	for name, metric := range cf.Metrics { | ||||
| 		n := len(metric.Data) | ||||
| 		b := &buffer{ | ||||
| 			frequency: metric.Frequency, | ||||
| 			start:     metric.Start, | ||||
| 			data:      metric.Data[0:n:n], // Space is wasted here :( | ||||
| 			prev:      nil, | ||||
| 			next:      nil, | ||||
| 			archived:  true, | ||||
| 		} | ||||
| 		b.close() | ||||
|  | ||||
| 		minfo, ok := m.Metrics[name] | ||||
| 		if !ok { | ||||
| 			continue | ||||
| 			// return errors.New("Unkown metric: " + name) | ||||
| 		} | ||||
|  | ||||
| 		prev := l.metrics[minfo.offset] | ||||
| 		if prev == nil { | ||||
| 			l.metrics[minfo.offset] = b | ||||
| 		} else { | ||||
| 			if prev.start > b.start { | ||||
| 				return errors.New("wooops") | ||||
| 			} | ||||
|  | ||||
| 			b.prev = prev | ||||
| 			prev.next = b | ||||
| 		} | ||||
| 		l.metrics[minfo.offset] = b | ||||
| 	} | ||||
|  | ||||
| 	if len(cf.Children) > 0 && l.children == nil { | ||||
| 		l.children = make(map[string]*Level) | ||||
| 	} | ||||
|  | ||||
| 	for sel, childCf := range cf.Children { | ||||
| 		child, ok := l.children[sel] | ||||
| 		if !ok { | ||||
| 			child = &Level{ | ||||
| 				metrics:  make([]*buffer, len(m.Metrics)), | ||||
| 				children: nil, | ||||
| 			} | ||||
| 			l.children[sel] = child | ||||
| 		} | ||||
|  | ||||
| 		if err := child.loadFile(childCf, m); err != nil { | ||||
| 			return err | ||||
| 		} | ||||
| 	} | ||||
|  | ||||
| 	return nil | ||||
| } | ||||
|  | ||||
| func (l *Level) fromCheckpoint(dir string, from int64, m *MemoryStore) (int, error) { | ||||
| 	direntries, err := os.ReadDir(dir) | ||||
| 	if err != nil { | ||||
| 		if os.IsNotExist(err) { | ||||
| 			return 0, nil | ||||
| 		} | ||||
|  | ||||
| 		return 0, err | ||||
| 	} | ||||
|  | ||||
| 	jsonFiles := make([]fs.DirEntry, 0) | ||||
| 	filesLoaded := 0 | ||||
| 	for _, e := range direntries { | ||||
| 		if e.IsDir() { | ||||
| 			child := &Level{ | ||||
| 				metrics:  make([]*buffer, len(m.Metrics)), | ||||
| 				children: make(map[string]*Level), | ||||
| 			} | ||||
|  | ||||
| 			files, err := child.fromCheckpoint(path.Join(dir, e.Name()), from, m) | ||||
| 			filesLoaded += files | ||||
| 			if err != nil { | ||||
| 				return filesLoaded, err | ||||
| 			} | ||||
|  | ||||
| 			l.children[e.Name()] = child | ||||
| 		} else if strings.HasSuffix(e.Name(), ".json") { | ||||
| 			jsonFiles = append(jsonFiles, e) | ||||
| 		} else { | ||||
| 			return filesLoaded, errors.New("unexpected file: " + dir + "/" + e.Name()) | ||||
| 		} | ||||
| 	} | ||||
|  | ||||
| 	files, err := findFiles(jsonFiles, from, true) | ||||
| 	if err != nil { | ||||
| 		return filesLoaded, err | ||||
| 	} | ||||
|  | ||||
| 	for _, filename := range files { | ||||
| 		f, err := os.Open(path.Join(dir, filename)) | ||||
| 		if err != nil { | ||||
| 			return filesLoaded, err | ||||
| 		} | ||||
| 		defer f.Close() | ||||
|  | ||||
| 		br := bufio.NewReader(f) | ||||
| 		cf := &CheckpointFile{} | ||||
| 		if err = json.NewDecoder(br).Decode(cf); err != nil { | ||||
| 			return filesLoaded, err | ||||
| 		} | ||||
|  | ||||
| 		if cf.To != 0 && cf.To < from { | ||||
| 			continue | ||||
| 		} | ||||
|  | ||||
| 		if err = l.loadFile(cf, m); err != nil { | ||||
| 			return filesLoaded, err | ||||
| 		} | ||||
|  | ||||
| 		filesLoaded += 1 | ||||
| 	} | ||||
|  | ||||
| 	return filesLoaded, nil | ||||
| } | ||||
|  | ||||
| // This will probably get very slow over time! | ||||
| // A solution could be some sort of an index file in which all other files | ||||
| // and the timespan they contain is listed. | ||||
| func findFiles(direntries []fs.DirEntry, t int64, findMoreRecentFiles bool) ([]string, error) { | ||||
| 	nums := map[string]int64{} | ||||
| 	for _, e := range direntries { | ||||
| 		ts, err := strconv.ParseInt(strings.TrimSuffix(e.Name(), ".json"), 10, 64) | ||||
| 		if err != nil { | ||||
| 			return nil, err | ||||
| 		} | ||||
| 		nums[e.Name()] = ts | ||||
| 	} | ||||
|  | ||||
| 	sort.Slice(direntries, func(i, j int) bool { | ||||
| 		a, b := direntries[i], direntries[j] | ||||
| 		return nums[a.Name()] < nums[b.Name()] | ||||
| 	}) | ||||
|  | ||||
| 	filenames := make([]string, 0) | ||||
| 	for i := 0; i < len(direntries); i++ { | ||||
| 		e := direntries[i] | ||||
| 		ts1 := nums[e.Name()] | ||||
|  | ||||
| 		if findMoreRecentFiles && t <= ts1 || i == len(direntries)-1 { | ||||
| 			filenames = append(filenames, e.Name()) | ||||
| 			continue | ||||
| 		} | ||||
|  | ||||
| 		enext := direntries[i+1] | ||||
| 		ts2 := nums[enext.Name()] | ||||
|  | ||||
| 		if findMoreRecentFiles { | ||||
| 			if ts1 < t && t < ts2 { | ||||
| 				filenames = append(filenames, e.Name()) | ||||
| 			} | ||||
| 		} else { | ||||
| 			if ts2 < t { | ||||
| 				filenames = append(filenames, e.Name()) | ||||
| 			} | ||||
| 		} | ||||
| 	} | ||||
|  | ||||
| 	return filenames, nil | ||||
| } | ||||
|  | ||||
| // ZIP all checkpoint files older than `from` together and write them to the `archiveDir`, | ||||
| // deleting them from the `checkpointsDir`. | ||||
| func ArchiveCheckpoints(checkpointsDir, archiveDir string, from int64, deleteInstead bool) (int, error) { | ||||
| 	entries1, err := os.ReadDir(checkpointsDir) | ||||
| 	if err != nil { | ||||
| 		return 0, err | ||||
| 	} | ||||
|  | ||||
| 	type workItem struct { | ||||
| 		cdir, adir    string | ||||
| 		cluster, host string | ||||
| 	} | ||||
|  | ||||
| 	var wg sync.WaitGroup | ||||
| 	n, errs := int32(0), int32(0) | ||||
| 	work := make(chan workItem, NumWorkers) | ||||
|  | ||||
| 	wg.Add(NumWorkers) | ||||
| 	for worker := 0; worker < NumWorkers; worker++ { | ||||
| 		go func() { | ||||
| 			defer wg.Done() | ||||
| 			for workItem := range work { | ||||
| 				m, err := archiveCheckpoints(workItem.cdir, workItem.adir, from, deleteInstead) | ||||
| 				if err != nil { | ||||
| 					log.Printf("error while archiving %s/%s: %s", workItem.cluster, workItem.host, err.Error()) | ||||
| 					atomic.AddInt32(&errs, 1) | ||||
| 				} | ||||
| 				atomic.AddInt32(&n, int32(m)) | ||||
| 			} | ||||
| 		}() | ||||
| 	} | ||||
|  | ||||
| 	for _, de1 := range entries1 { | ||||
| 		entries2, e := os.ReadDir(filepath.Join(checkpointsDir, de1.Name())) | ||||
| 		if e != nil { | ||||
| 			err = e | ||||
| 		} | ||||
|  | ||||
| 		for _, de2 := range entries2 { | ||||
| 			cdir := filepath.Join(checkpointsDir, de1.Name(), de2.Name()) | ||||
| 			adir := filepath.Join(archiveDir, de1.Name(), de2.Name()) | ||||
| 			work <- workItem{ | ||||
| 				adir: adir, cdir: cdir, | ||||
| 				cluster: de1.Name(), host: de2.Name(), | ||||
| 			} | ||||
| 		} | ||||
| 	} | ||||
|  | ||||
| 	close(work) | ||||
| 	wg.Wait() | ||||
|  | ||||
| 	if err != nil { | ||||
| 		return int(n), err | ||||
| 	} | ||||
|  | ||||
| 	if errs > 0 { | ||||
| 		return int(n), fmt.Errorf("%d errors happend while archiving (%d successes)", errs, n) | ||||
| 	} | ||||
| 	return int(n), nil | ||||
| } | ||||
|  | ||||
| // Helper function for `ArchiveCheckpoints`. | ||||
| func archiveCheckpoints(dir string, archiveDir string, from int64, deleteInstead bool) (int, error) { | ||||
| 	entries, err := os.ReadDir(dir) | ||||
| 	if err != nil { | ||||
| 		return 0, err | ||||
| 	} | ||||
|  | ||||
| 	files, err := findFiles(entries, from, false) | ||||
| 	if err != nil { | ||||
| 		return 0, err | ||||
| 	} | ||||
|  | ||||
| 	if deleteInstead { | ||||
| 		n := 0 | ||||
| 		for _, checkpoint := range files { | ||||
| 			filename := filepath.Join(dir, checkpoint) | ||||
| 			if err = os.Remove(filename); err != nil { | ||||
| 				return n, err | ||||
| 			} | ||||
| 			n += 1 | ||||
| 		} | ||||
| 		return n, nil | ||||
| 	} | ||||
|  | ||||
| 	filename := filepath.Join(archiveDir, fmt.Sprintf("%d.zip", from)) | ||||
| 	f, err := os.OpenFile(filename, os.O_CREATE|os.O_WRONLY, 0o644) | ||||
| 	if err != nil && os.IsNotExist(err) { | ||||
| 		err = os.MkdirAll(archiveDir, 0o755) | ||||
| 		if err == nil { | ||||
| 			f, err = os.OpenFile(filename, os.O_CREATE|os.O_WRONLY, 0o644) | ||||
| 		} | ||||
| 	} | ||||
| 	if err != nil { | ||||
| 		return 0, err | ||||
| 	} | ||||
| 	defer f.Close() | ||||
| 	bw := bufio.NewWriter(f) | ||||
| 	defer bw.Flush() | ||||
| 	zw := zip.NewWriter(bw) | ||||
| 	defer zw.Close() | ||||
|  | ||||
| 	n := 0 | ||||
| 	for _, checkpoint := range files { | ||||
| 		filename := filepath.Join(dir, checkpoint) | ||||
| 		r, err := os.Open(filename) | ||||
| 		if err != nil { | ||||
| 			return n, err | ||||
| 		} | ||||
| 		defer r.Close() | ||||
|  | ||||
| 		w, err := zw.Create(checkpoint) | ||||
| 		if err != nil { | ||||
| 			return n, err | ||||
| 		} | ||||
|  | ||||
| 		if _, err = io.Copy(w, r); err != nil { | ||||
| 			return n, err | ||||
| 		} | ||||
|  | ||||
| 		if err = os.Remove(filename); err != nil { | ||||
| 			return n, err | ||||
| 		} | ||||
| 		n += 1 | ||||
| 	} | ||||
|  | ||||
| 	return n, nil | ||||
| } | ||||
							
								
								
									
										241
									
								
								internal/memorystore/buffer.go
									
									
									
									
									
										Normal file
									
								
							
							
						
						
									
										241
									
								
								internal/memorystore/buffer.go
									
									
									
									
									
										Normal file
									
								
							| @@ -0,0 +1,241 @@ | ||||
| package memorystore | ||||
|  | ||||
| import ( | ||||
| 	"errors" | ||||
| 	"sync" | ||||
|  | ||||
| 	"github.com/ClusterCockpit/cc-metric-store/internal/util" | ||||
| ) | ||||
|  | ||||
| // Default buffer capacity. | ||||
| // `buffer.data` will only ever grow up to it's capacity and a new link | ||||
| // in the buffer chain will be created if needed so that no copying | ||||
| // of data or reallocation needs to happen on writes. | ||||
| const ( | ||||
| 	BUFFER_CAP int = 512 | ||||
| ) | ||||
|  | ||||
| // So that we can reuse allocations | ||||
| var bufferPool sync.Pool = sync.Pool{ | ||||
| 	New: func() interface{} { | ||||
| 		return &buffer{ | ||||
| 			data: make([]util.Float, 0, BUFFER_CAP), | ||||
| 		} | ||||
| 	}, | ||||
| } | ||||
|  | ||||
| var ( | ||||
| 	ErrNoData           error = errors.New("no data for this metric/level") | ||||
| 	ErrDataDoesNotAlign error = errors.New("data from lower granularities does not align") | ||||
| ) | ||||
|  | ||||
| // Each metric on each level has it's own buffer. | ||||
| // This is where the actual values go. | ||||
| // If `cap(data)` is reached, a new buffer is created and | ||||
| // becomes the new head of a buffer list. | ||||
| type buffer struct { | ||||
| 	frequency  int64        // Time between two "slots" | ||||
| 	start      int64        // Timestamp of when `data[0]` was written. | ||||
| 	data       []util.Float // The slice should never reallocacte as `cap(data)` is respected. | ||||
| 	prev, next *buffer      // `prev` contains older data, `next` newer data. | ||||
| 	archived   bool         // If true, this buffer is already archived | ||||
|  | ||||
| 	closed bool | ||||
| 	/* | ||||
| 		statisticts struct { | ||||
| 			samples int | ||||
| 			min     Float | ||||
| 			max     Float | ||||
| 			avg     Float | ||||
| 		} | ||||
| 	*/ | ||||
| } | ||||
|  | ||||
| func newBuffer(ts, freq int64) *buffer { | ||||
| 	b := bufferPool.Get().(*buffer) | ||||
| 	b.frequency = freq | ||||
| 	b.start = ts - (freq / 2) | ||||
| 	b.prev = nil | ||||
| 	b.next = nil | ||||
| 	b.archived = false | ||||
| 	b.closed = false | ||||
| 	b.data = b.data[:0] | ||||
| 	return b | ||||
| } | ||||
|  | ||||
| // If a new buffer was created, the new head is returnd. | ||||
| // Otherwise, the existing buffer is returnd. | ||||
| // Normaly, only "newer" data should be written, but if the value would | ||||
| // end up in the same buffer anyways it is allowed. | ||||
| func (b *buffer) write(ts int64, value util.Float) (*buffer, error) { | ||||
| 	if ts < b.start { | ||||
| 		return nil, errors.New("cannot write value to buffer from past") | ||||
| 	} | ||||
|  | ||||
| 	// idx := int((ts - b.start + (b.frequency / 3)) / b.frequency) | ||||
| 	idx := int((ts - b.start) / b.frequency) | ||||
| 	if idx >= cap(b.data) { | ||||
| 		newbuf := newBuffer(ts, b.frequency) | ||||
| 		newbuf.prev = b | ||||
| 		b.next = newbuf | ||||
| 		b.close() | ||||
| 		b = newbuf | ||||
| 		idx = 0 | ||||
| 	} | ||||
|  | ||||
| 	// Overwriting value or writing value from past | ||||
| 	if idx < len(b.data) { | ||||
| 		b.data[idx] = value | ||||
| 		return b, nil | ||||
| 	} | ||||
|  | ||||
| 	// Fill up unwritten slots with NaN | ||||
| 	for i := len(b.data); i < idx; i++ { | ||||
| 		b.data = append(b.data, util.NaN) | ||||
| 	} | ||||
|  | ||||
| 	b.data = append(b.data, value) | ||||
| 	return b, nil | ||||
| } | ||||
|  | ||||
| func (b *buffer) end() int64 { | ||||
| 	return b.firstWrite() + int64(len(b.data))*b.frequency | ||||
| } | ||||
|  | ||||
| func (b *buffer) firstWrite() int64 { | ||||
| 	return b.start + (b.frequency / 2) | ||||
| } | ||||
|  | ||||
| func (b *buffer) close() {} | ||||
|  | ||||
| /* | ||||
| func (b *buffer) close() { | ||||
| 	if b.closed { | ||||
| 		return | ||||
| 	} | ||||
|  | ||||
| 	b.closed = true | ||||
| 	n, sum, min, max := 0, 0., math.MaxFloat64, -math.MaxFloat64 | ||||
| 	for _, x := range b.data { | ||||
| 		if x.IsNaN() { | ||||
| 			continue | ||||
| 		} | ||||
|  | ||||
| 		n += 1 | ||||
| 		f := float64(x) | ||||
| 		sum += f | ||||
| 		min = math.Min(min, f) | ||||
| 		max = math.Max(max, f) | ||||
| 	} | ||||
|  | ||||
| 	b.statisticts.samples = n | ||||
| 	if n > 0 { | ||||
| 		b.statisticts.avg = Float(sum / float64(n)) | ||||
| 		b.statisticts.min = Float(min) | ||||
| 		b.statisticts.max = Float(max) | ||||
| 	} else { | ||||
| 		b.statisticts.avg = NaN | ||||
| 		b.statisticts.min = NaN | ||||
| 		b.statisticts.max = NaN | ||||
| 	} | ||||
| } | ||||
| */ | ||||
|  | ||||
| // func interpolate(idx int, data []Float) Float { | ||||
| // 	if idx == 0 || idx+1 == len(data) { | ||||
| // 		return NaN | ||||
| // 	} | ||||
| // 	return (data[idx-1] + data[idx+1]) / 2.0 | ||||
| // } | ||||
|  | ||||
| // Return all known values from `from` to `to`. Gaps of information are represented as NaN. | ||||
| // Simple linear interpolation is done between the two neighboring cells if possible. | ||||
| // If values at the start or end are missing, instead of NaN values, the second and thrid | ||||
| // return values contain the actual `from`/`to`. | ||||
| // This function goes back the buffer chain if `from` is older than the currents buffer start. | ||||
| // The loaded values are added to `data` and `data` is returned, possibly with a shorter length. | ||||
| // If `data` is not long enough to hold all values, this function will panic! | ||||
| func (b *buffer) read(from, to int64, data []util.Float) ([]util.Float, int64, int64, error) { | ||||
| 	if from < b.firstWrite() { | ||||
| 		if b.prev != nil { | ||||
| 			return b.prev.read(from, to, data) | ||||
| 		} | ||||
| 		from = b.firstWrite() | ||||
| 	} | ||||
|  | ||||
| 	var i int = 0 | ||||
| 	var t int64 = from | ||||
| 	for ; t < to; t += b.frequency { | ||||
| 		idx := int((t - b.start) / b.frequency) | ||||
| 		if idx >= cap(b.data) { | ||||
| 			if b.next == nil { | ||||
| 				break | ||||
| 			} | ||||
| 			b = b.next | ||||
| 			idx = 0 | ||||
| 		} | ||||
|  | ||||
| 		if idx >= len(b.data) { | ||||
| 			if b.next == nil || to <= b.next.start { | ||||
| 				break | ||||
| 			} | ||||
| 			data[i] += util.NaN | ||||
| 		} else if t < b.start { | ||||
| 			data[i] += util.NaN | ||||
| 			// } else if b.data[idx].IsNaN() { | ||||
| 			// 	data[i] += interpolate(idx, b.data) | ||||
| 		} else { | ||||
| 			data[i] += b.data[idx] | ||||
| 		} | ||||
| 		i++ | ||||
| 	} | ||||
|  | ||||
| 	return data[:i], from, t, nil | ||||
| } | ||||
|  | ||||
| // Returns true if this buffer needs to be freed. | ||||
| func (b *buffer) free(t int64) (delme bool, n int) { | ||||
| 	if b.prev != nil { | ||||
| 		delme, m := b.prev.free(t) | ||||
| 		n += m | ||||
| 		if delme { | ||||
| 			b.prev.next = nil | ||||
| 			if cap(b.prev.data) == BUFFER_CAP { | ||||
| 				bufferPool.Put(b.prev) | ||||
| 			} | ||||
| 			b.prev = nil | ||||
| 		} | ||||
| 	} | ||||
|  | ||||
| 	end := b.end() | ||||
| 	if end < t { | ||||
| 		return true, n + 1 | ||||
| 	} | ||||
|  | ||||
| 	return false, n | ||||
| } | ||||
|  | ||||
| // Call `callback` on every buffer that contains data in the range from `from` to `to`. | ||||
| func (b *buffer) iterFromTo(from, to int64, callback func(b *buffer) error) error { | ||||
| 	if b == nil { | ||||
| 		return nil | ||||
| 	} | ||||
|  | ||||
| 	if err := b.prev.iterFromTo(from, to, callback); err != nil { | ||||
| 		return err | ||||
| 	} | ||||
|  | ||||
| 	if from <= b.end() && b.start <= to { | ||||
| 		return callback(b) | ||||
| 	} | ||||
|  | ||||
| 	return nil | ||||
| } | ||||
|  | ||||
| func (b *buffer) count() int64 { | ||||
| 	res := int64(len(b.data)) | ||||
| 	if b.prev != nil { | ||||
| 		res += b.prev.count() | ||||
| 	} | ||||
| 	return res | ||||
| } | ||||
							
								
								
									
										107
									
								
								internal/memorystore/debug.go
									
									
									
									
									
										Normal file
									
								
							
							
						
						
									
										107
									
								
								internal/memorystore/debug.go
									
									
									
									
									
										Normal file
									
								
							| @@ -0,0 +1,107 @@ | ||||
| package memorystore | ||||
|  | ||||
| import ( | ||||
| 	"bufio" | ||||
| 	"fmt" | ||||
| 	"strconv" | ||||
| ) | ||||
|  | ||||
| func (b *buffer) debugDump(buf []byte) []byte { | ||||
| 	if b.prev != nil { | ||||
| 		buf = b.prev.debugDump(buf) | ||||
| 	} | ||||
|  | ||||
| 	start, len, end := b.start, len(b.data), b.start+b.frequency*int64(len(b.data)) | ||||
| 	buf = append(buf, `{"start":`...) | ||||
| 	buf = strconv.AppendInt(buf, start, 10) | ||||
| 	buf = append(buf, `,"len":`...) | ||||
| 	buf = strconv.AppendInt(buf, int64(len), 10) | ||||
| 	buf = append(buf, `,"end":`...) | ||||
| 	buf = strconv.AppendInt(buf, end, 10) | ||||
| 	if b.archived { | ||||
| 		buf = append(buf, `,"saved":true`...) | ||||
| 	} | ||||
| 	if b.next != nil { | ||||
| 		buf = append(buf, `},`...) | ||||
| 	} else { | ||||
| 		buf = append(buf, `}`...) | ||||
| 	} | ||||
| 	return buf | ||||
| } | ||||
|  | ||||
| func (l *Level) debugDump(m *MemoryStore, w *bufio.Writer, lvlname string, buf []byte, depth int) ([]byte, error) { | ||||
| 	l.lock.RLock() | ||||
| 	defer l.lock.RUnlock() | ||||
| 	for i := 0; i < depth; i++ { | ||||
| 		buf = append(buf, '\t') | ||||
| 	} | ||||
| 	buf = append(buf, '"') | ||||
| 	buf = append(buf, lvlname...) | ||||
| 	buf = append(buf, "\":{\n"...) | ||||
| 	depth += 1 | ||||
| 	objitems := 0 | ||||
| 	for name, mc := range m.Metrics { | ||||
| 		if b := l.metrics[mc.offset]; b != nil { | ||||
| 			for i := 0; i < depth; i++ { | ||||
| 				buf = append(buf, '\t') | ||||
| 			} | ||||
|  | ||||
| 			buf = append(buf, '"') | ||||
| 			buf = append(buf, name...) | ||||
| 			buf = append(buf, `":[`...) | ||||
| 			buf = b.debugDump(buf) | ||||
| 			buf = append(buf, "],\n"...) | ||||
| 			objitems++ | ||||
| 		} | ||||
| 	} | ||||
|  | ||||
| 	for name, lvl := range l.children { | ||||
| 		_, err := w.Write(buf) | ||||
| 		if err != nil { | ||||
| 			return nil, err | ||||
| 		} | ||||
|  | ||||
| 		buf = buf[0:0] | ||||
| 		buf, err = lvl.debugDump(m, w, name, buf, depth) | ||||
| 		if err != nil { | ||||
| 			return nil, err | ||||
| 		} | ||||
|  | ||||
| 		buf = append(buf, ',', '\n') | ||||
| 		objitems++ | ||||
| 	} | ||||
|  | ||||
| 	// remove final `,`: | ||||
| 	if objitems > 0 { | ||||
| 		buf = append(buf[0:len(buf)-1], '\n') | ||||
| 	} | ||||
|  | ||||
| 	depth -= 1 | ||||
| 	for i := 0; i < depth; i++ { | ||||
| 		buf = append(buf, '\t') | ||||
| 	} | ||||
| 	buf = append(buf, '}') | ||||
| 	return buf, nil | ||||
| } | ||||
|  | ||||
| func (m *MemoryStore) DebugDump(w *bufio.Writer, selector []string) error { | ||||
| 	lvl := m.root.findLevel(selector) | ||||
| 	if lvl == nil { | ||||
| 		return fmt.Errorf("not found: %#v", selector) | ||||
| 	} | ||||
|  | ||||
| 	buf := make([]byte, 0, 2048) | ||||
| 	buf = append(buf, "{"...) | ||||
|  | ||||
| 	buf, err := lvl.debugDump(m, w, "data", buf, 0) | ||||
| 	if err != nil { | ||||
| 		return err | ||||
| 	} | ||||
|  | ||||
| 	buf = append(buf, "}\n"...) | ||||
| 	if _, err = w.Write(buf); err != nil { | ||||
| 		return err | ||||
| 	} | ||||
|  | ||||
| 	return w.Flush() | ||||
| } | ||||
							
								
								
									
										183
									
								
								internal/memorystore/level.go
									
									
									
									
									
										Normal file
									
								
							
							
						
						
									
										183
									
								
								internal/memorystore/level.go
									
									
									
									
									
										Normal file
									
								
							| @@ -0,0 +1,183 @@ | ||||
| package memorystore | ||||
|  | ||||
| import ( | ||||
| 	"sync" | ||||
| 	"unsafe" | ||||
|  | ||||
| 	"github.com/ClusterCockpit/cc-metric-store/internal/util" | ||||
| ) | ||||
|  | ||||
| // Could also be called "node" as this forms a node in a tree structure. | ||||
| // Called Level because "node" might be confusing here. | ||||
| // Can be both a leaf or a inner node. In this tree structue, inner nodes can | ||||
| // also hold data (in `metrics`). | ||||
| type Level struct { | ||||
| 	lock     sync.RWMutex | ||||
| 	metrics  []*buffer         // Every level can store metrics. | ||||
| 	children map[string]*Level // Lower levels. | ||||
| } | ||||
|  | ||||
| // Find the correct level for the given selector, creating it if | ||||
| // it does not exist. Example selector in the context of the | ||||
| // ClusterCockpit could be: []string{ "emmy", "host123", "cpu0" }. | ||||
| // This function would probably benefit a lot from `level.children` beeing a `sync.Map`? | ||||
| func (l *Level) findLevelOrCreate(selector []string, nMetrics int) *Level { | ||||
| 	if len(selector) == 0 { | ||||
| 		return l | ||||
| 	} | ||||
|  | ||||
| 	// Allow concurrent reads: | ||||
| 	l.lock.RLock() | ||||
| 	var child *Level | ||||
| 	var ok bool | ||||
| 	if l.children == nil { | ||||
| 		// Children map needs to be created... | ||||
| 		l.lock.RUnlock() | ||||
| 	} else { | ||||
| 		child, ok := l.children[selector[0]] | ||||
| 		l.lock.RUnlock() | ||||
| 		if ok { | ||||
| 			return child.findLevelOrCreate(selector[1:], nMetrics) | ||||
| 		} | ||||
| 	} | ||||
|  | ||||
| 	// The level does not exist, take write lock for unqiue access: | ||||
| 	l.lock.Lock() | ||||
| 	// While this thread waited for the write lock, another thread | ||||
| 	// could have created the child node. | ||||
| 	if l.children != nil { | ||||
| 		child, ok = l.children[selector[0]] | ||||
| 		if ok { | ||||
| 			l.lock.Unlock() | ||||
| 			return child.findLevelOrCreate(selector[1:], nMetrics) | ||||
| 		} | ||||
| 	} | ||||
|  | ||||
| 	child = &Level{ | ||||
| 		metrics:  make([]*buffer, nMetrics), | ||||
| 		children: nil, | ||||
| 	} | ||||
|  | ||||
| 	if l.children != nil { | ||||
| 		l.children[selector[0]] = child | ||||
| 	} else { | ||||
| 		l.children = map[string]*Level{selector[0]: child} | ||||
| 	} | ||||
| 	l.lock.Unlock() | ||||
| 	return child.findLevelOrCreate(selector[1:], nMetrics) | ||||
| } | ||||
|  | ||||
| func (l *Level) free(t int64) (int, error) { | ||||
| 	l.lock.Lock() | ||||
| 	defer l.lock.Unlock() | ||||
|  | ||||
| 	n := 0 | ||||
| 	for i, b := range l.metrics { | ||||
| 		if b != nil { | ||||
| 			delme, m := b.free(t) | ||||
| 			n += m | ||||
| 			if delme { | ||||
| 				if cap(b.data) == BUFFER_CAP { | ||||
| 					bufferPool.Put(b) | ||||
| 				} | ||||
| 				l.metrics[i] = nil | ||||
| 			} | ||||
| 		} | ||||
| 	} | ||||
|  | ||||
| 	for _, l := range l.children { | ||||
| 		m, err := l.free(t) | ||||
| 		n += m | ||||
| 		if err != nil { | ||||
| 			return n, err | ||||
| 		} | ||||
| 	} | ||||
|  | ||||
| 	return n, nil | ||||
| } | ||||
|  | ||||
| func (l *Level) sizeInBytes() int64 { | ||||
| 	l.lock.RLock() | ||||
| 	defer l.lock.RUnlock() | ||||
| 	size := int64(0) | ||||
|  | ||||
| 	for _, b := range l.metrics { | ||||
| 		if b != nil { | ||||
| 			size += b.count() * int64(unsafe.Sizeof(util.Float(0))) | ||||
| 		} | ||||
| 	} | ||||
|  | ||||
| 	return size | ||||
| } | ||||
|  | ||||
| func (l *Level) findLevel(selector []string) *Level { | ||||
| 	if len(selector) == 0 { | ||||
| 		return l | ||||
| 	} | ||||
|  | ||||
| 	l.lock.RLock() | ||||
| 	defer l.lock.RUnlock() | ||||
|  | ||||
| 	lvl := l.children[selector[0]] | ||||
| 	if lvl == nil { | ||||
| 		return nil | ||||
| 	} | ||||
|  | ||||
| 	return lvl.findLevel(selector[1:]) | ||||
| } | ||||
|  | ||||
| func (l *Level) findBuffers(selector Selector, offset int, f func(b *buffer) error) error { | ||||
| 	l.lock.RLock() | ||||
| 	defer l.lock.RUnlock() | ||||
|  | ||||
| 	if len(selector) == 0 { | ||||
| 		b := l.metrics[offset] | ||||
| 		if b != nil { | ||||
| 			return f(b) | ||||
| 		} | ||||
|  | ||||
| 		for _, lvl := range l.children { | ||||
| 			err := lvl.findBuffers(nil, offset, f) | ||||
| 			if err != nil { | ||||
| 				return err | ||||
| 			} | ||||
| 		} | ||||
| 		return nil | ||||
| 	} | ||||
|  | ||||
| 	sel := selector[0] | ||||
| 	if len(sel.String) != 0 && l.children != nil { | ||||
| 		lvl, ok := l.children[sel.String] | ||||
| 		if ok { | ||||
| 			err := lvl.findBuffers(selector[1:], offset, f) | ||||
| 			if err != nil { | ||||
| 				return err | ||||
| 			} | ||||
| 		} | ||||
| 		return nil | ||||
| 	} | ||||
|  | ||||
| 	if sel.Group != nil && l.children != nil { | ||||
| 		for _, key := range sel.Group { | ||||
| 			lvl, ok := l.children[key] | ||||
| 			if ok { | ||||
| 				err := lvl.findBuffers(selector[1:], offset, f) | ||||
| 				if err != nil { | ||||
| 					return err | ||||
| 				} | ||||
| 			} | ||||
| 		} | ||||
| 		return nil | ||||
| 	} | ||||
|  | ||||
| 	if sel.Any && l.children != nil { | ||||
| 		for _, lvl := range l.children { | ||||
| 			if err := lvl.findBuffers(selector[1:], offset, f); err != nil { | ||||
| 				return err | ||||
| 			} | ||||
| 		} | ||||
| 		return nil | ||||
| 	} | ||||
|  | ||||
| 	return nil | ||||
| } | ||||
							
								
								
									
										222
									
								
								internal/memorystore/memorystore.go
									
									
									
									
									
										Normal file
									
								
							
							
						
						
									
										222
									
								
								internal/memorystore/memorystore.go
									
									
									
									
									
										Normal file
									
								
							| @@ -0,0 +1,222 @@ | ||||
| package memorystore | ||||
|  | ||||
| import ( | ||||
| 	"errors" | ||||
| 	"log" | ||||
| 	"sync" | ||||
|  | ||||
| 	"github.com/ClusterCockpit/cc-metric-store/internal/config" | ||||
| 	"github.com/ClusterCockpit/cc-metric-store/internal/util" | ||||
| ) | ||||
|  | ||||
| var ( | ||||
| 	singleton  sync.Once | ||||
| 	msInstance *MemoryStore | ||||
| ) | ||||
|  | ||||
| type Metric struct { | ||||
| 	Name         string | ||||
| 	Value        util.Float | ||||
| 	MetricConfig config.MetricConfig | ||||
| } | ||||
|  | ||||
| type MemoryStore struct { | ||||
| 	root    Level // root of the tree structure | ||||
| 	Metrics map[string]config.MetricConfig | ||||
| } | ||||
|  | ||||
| // Create a new, initialized instance of a MemoryStore. | ||||
| // Will panic if values in the metric configurations are invalid. | ||||
| func Init(metrics map[string]config.MetricConfig) { | ||||
| 	singleton.Do(func() { | ||||
| 		offset := 0 | ||||
| 		for key, cfg := range metrics { | ||||
| 			if cfg.Frequency == 0 { | ||||
| 				panic("invalid frequency") | ||||
| 			} | ||||
|  | ||||
| 			metrics[key] = config.MetricConfig{ | ||||
| 				Frequency:   cfg.Frequency, | ||||
| 				Aggregation: cfg.Aggregation, | ||||
| 				Offset:      offset, | ||||
| 			} | ||||
| 			offset += 1 | ||||
| 		} | ||||
|  | ||||
| 		msInstance = &MemoryStore{ | ||||
| 			root: Level{ | ||||
| 				metrics:  make([]*buffer, len(metrics)), | ||||
| 				children: make(map[string]*Level), | ||||
| 			}, | ||||
| 			Metrics: metrics, | ||||
| 		} | ||||
| 	}) | ||||
| } | ||||
|  | ||||
| func GetMemoryStore() *MemoryStore { | ||||
| 	if msInstance == nil { | ||||
| 		log.Fatalf("MemoryStore not initialized!") | ||||
| 	} | ||||
|  | ||||
| 	return msInstance | ||||
| } | ||||
|  | ||||
| // Write all values in `metrics` to the level specified by `selector` for time `ts`. | ||||
| // Look at `findLevelOrCreate` for how selectors work. | ||||
| func (m *MemoryStore) Write(selector []string, ts int64, metrics []Metric) error { | ||||
| 	var ok bool | ||||
| 	for i, metric := range metrics { | ||||
| 		if metric.MetricConfig.Frequency == 0 { | ||||
| 			metric.MetricConfig, ok = m.Metrics[metric.Name] | ||||
| 			if !ok { | ||||
| 				metric.MetricConfig.Frequency = 0 | ||||
| 			} | ||||
| 			metrics[i] = metric | ||||
| 		} | ||||
| 	} | ||||
|  | ||||
| 	return m.WriteToLevel(&m.root, selector, ts, metrics) | ||||
| } | ||||
|  | ||||
| func (m *MemoryStore) GetLevel(selector []string) *Level { | ||||
| 	return m.root.findLevelOrCreate(selector, len(m.Metrics)) | ||||
| } | ||||
|  | ||||
| // Assumes that `minfo` in `metrics` is filled in! | ||||
| func (m *MemoryStore) WriteToLevel(l *Level, selector []string, ts int64, metrics []Metric) error { | ||||
| 	l = l.findLevelOrCreate(selector, len(m.Metrics)) | ||||
| 	l.lock.Lock() | ||||
| 	defer l.lock.Unlock() | ||||
|  | ||||
| 	for _, metric := range metrics { | ||||
| 		if metric.MetricConfig.Frequency == 0 { | ||||
| 			continue | ||||
| 		} | ||||
|  | ||||
| 		b := l.metrics[metric.MetricConfig.Offset] | ||||
| 		if b == nil { | ||||
| 			// First write to this metric and level | ||||
| 			b = newBuffer(ts, metric.MetricConfig.Frequency) | ||||
| 			l.metrics[metric.MetricConfig.Offset] = b | ||||
| 		} | ||||
|  | ||||
| 		nb, err := b.write(ts, metric.Value) | ||||
| 		if err != nil { | ||||
| 			return err | ||||
| 		} | ||||
|  | ||||
| 		// Last write created a new buffer... | ||||
| 		if b != nb { | ||||
| 			l.metrics[metric.MetricConfig.Offset] = nb | ||||
| 		} | ||||
| 	} | ||||
| 	return nil | ||||
| } | ||||
|  | ||||
| // Returns all values for metric `metric` from `from` to `to` for the selected level(s). | ||||
| // If the level does not hold the metric itself, the data will be aggregated recursively from the children. | ||||
| // The second and third return value are the actual from/to for the data. Those can be different from | ||||
| // the range asked for if no data was available. | ||||
| func (m *MemoryStore) Read(selector Selector, metric string, from, to int64) ([]util.Float, int64, int64, error) { | ||||
| 	if from > to { | ||||
| 		return nil, 0, 0, errors.New("invalid time range") | ||||
| 	} | ||||
|  | ||||
| 	minfo, ok := m.Metrics[metric] | ||||
| 	if !ok { | ||||
| 		return nil, 0, 0, errors.New("unkown metric: " + metric) | ||||
| 	} | ||||
|  | ||||
| 	n, data := 0, make([]util.Float, (to-from)/minfo.Frequency+1) | ||||
| 	err := m.root.findBuffers(selector, minfo.Offset, func(b *buffer) error { | ||||
| 		cdata, cfrom, cto, err := b.read(from, to, data) | ||||
| 		if err != nil { | ||||
| 			return err | ||||
| 		} | ||||
|  | ||||
| 		if n == 0 { | ||||
| 			from, to = cfrom, cto | ||||
| 		} else if from != cfrom || to != cto || len(data) != len(cdata) { | ||||
| 			missingfront, missingback := int((from-cfrom)/minfo.Frequency), int((to-cto)/minfo.Frequency) | ||||
| 			if missingfront != 0 { | ||||
| 				return ErrDataDoesNotAlign | ||||
| 			} | ||||
|  | ||||
| 			newlen := len(cdata) - missingback | ||||
| 			if newlen < 1 { | ||||
| 				return ErrDataDoesNotAlign | ||||
| 			} | ||||
| 			cdata = cdata[0:newlen] | ||||
| 			if len(cdata) != len(data) { | ||||
| 				return ErrDataDoesNotAlign | ||||
| 			} | ||||
|  | ||||
| 			from, to = cfrom, cto | ||||
| 		} | ||||
|  | ||||
| 		data = cdata | ||||
| 		n += 1 | ||||
| 		return nil | ||||
| 	}) | ||||
|  | ||||
| 	if err != nil { | ||||
| 		return nil, 0, 0, err | ||||
| 	} else if n == 0 { | ||||
| 		return nil, 0, 0, errors.New("metric or host not found") | ||||
| 	} else if n > 1 { | ||||
| 		if minfo.Aggregation == config.AvgAggregation { | ||||
| 			normalize := 1. / util.Float(n) | ||||
| 			for i := 0; i < len(data); i++ { | ||||
| 				data[i] *= normalize | ||||
| 			} | ||||
| 		} else if minfo.Aggregation != config.SumAggregation { | ||||
| 			return nil, 0, 0, errors.New("invalid aggregation") | ||||
| 		} | ||||
| 	} | ||||
|  | ||||
| 	return data, from, to, nil | ||||
| } | ||||
|  | ||||
| // Release all buffers for the selected level and all its children that contain only | ||||
| // values older than `t`. | ||||
| func (m *MemoryStore) Free(selector []string, t int64) (int, error) { | ||||
| 	return m.GetLevel(selector).free(t) | ||||
| } | ||||
|  | ||||
| func (m *MemoryStore) FreeAll() error { | ||||
| 	for k := range m.root.children { | ||||
| 		delete(m.root.children, k) | ||||
| 	} | ||||
|  | ||||
| 	return nil | ||||
| } | ||||
|  | ||||
| func (m *MemoryStore) SizeInBytes() int64 { | ||||
| 	return m.root.sizeInBytes() | ||||
| } | ||||
|  | ||||
| // Given a selector, return a list of all children of the level selected. | ||||
| func (m *MemoryStore) ListChildren(selector []string) []string { | ||||
| 	lvl := &m.root | ||||
| 	for lvl != nil && len(selector) != 0 { | ||||
| 		lvl.lock.RLock() | ||||
| 		next := lvl.children[selector[0]] | ||||
| 		lvl.lock.RUnlock() | ||||
| 		lvl = next | ||||
| 		selector = selector[1:] | ||||
| 	} | ||||
|  | ||||
| 	if lvl == nil { | ||||
| 		return nil | ||||
| 	} | ||||
|  | ||||
| 	lvl.lock.RLock() | ||||
| 	defer lvl.lock.RUnlock() | ||||
|  | ||||
| 	children := make([]string, 0, len(lvl.children)) | ||||
| 	for child := range lvl.children { | ||||
| 		children = append(children, child) | ||||
| 	} | ||||
|  | ||||
| 	return children | ||||
| } | ||||
							
								
								
									
										51
									
								
								internal/memorystore/selector.go
									
									
									
									
									
										Normal file
									
								
							
							
						
						
									
										51
									
								
								internal/memorystore/selector.go
									
									
									
									
									
										Normal file
									
								
							| @@ -0,0 +1,51 @@ | ||||
| package memorystore | ||||
|  | ||||
| import ( | ||||
| 	"encoding/json" | ||||
| 	"errors" | ||||
| ) | ||||
|  | ||||
| type SelectorElement struct { | ||||
| 	Any    bool | ||||
| 	String string | ||||
| 	Group  []string | ||||
| } | ||||
|  | ||||
| func (se *SelectorElement) UnmarshalJSON(input []byte) error { | ||||
| 	if input[0] == '"' { | ||||
| 		if err := json.Unmarshal(input, &se.String); err != nil { | ||||
| 			return err | ||||
| 		} | ||||
|  | ||||
| 		if se.String == "*" { | ||||
| 			se.Any = true | ||||
| 			se.String = "" | ||||
| 		} | ||||
|  | ||||
| 		return nil | ||||
| 	} | ||||
|  | ||||
| 	if input[0] == '[' { | ||||
| 		return json.Unmarshal(input, &se.Group) | ||||
| 	} | ||||
|  | ||||
| 	return errors.New("the Go SelectorElement type can only be a string or an array of strings") | ||||
| } | ||||
|  | ||||
| func (se *SelectorElement) MarshalJSON() ([]byte, error) { | ||||
| 	if se.Any { | ||||
| 		return []byte("\"*\""), nil | ||||
| 	} | ||||
|  | ||||
| 	if se.String != "" { | ||||
| 		return json.Marshal(se.String) | ||||
| 	} | ||||
|  | ||||
| 	if se.Group != nil { | ||||
| 		return json.Marshal(se.Group) | ||||
| 	} | ||||
|  | ||||
| 	return nil, errors.New("a Go Selector must be a non-empty string or a non-empty slice of strings") | ||||
| } | ||||
|  | ||||
| type Selector []SelectorElement | ||||
							
								
								
									
										120
									
								
								internal/memorystore/stats.go
									
									
									
									
									
										Normal file
									
								
							
							
						
						
									
										120
									
								
								internal/memorystore/stats.go
									
									
									
									
									
										Normal file
									
								
							| @@ -0,0 +1,120 @@ | ||||
| package memorystore | ||||
|  | ||||
| import ( | ||||
| 	"errors" | ||||
| 	"math" | ||||
|  | ||||
| 	"github.com/ClusterCockpit/cc-metric-store/internal/config" | ||||
| 	"github.com/ClusterCockpit/cc-metric-store/internal/util" | ||||
| ) | ||||
|  | ||||
| type Stats struct { | ||||
| 	Samples int | ||||
| 	Avg     util.Float | ||||
| 	Min     util.Float | ||||
| 	Max     util.Float | ||||
| } | ||||
|  | ||||
| func (b *buffer) stats(from, to int64) (Stats, int64, int64, error) { | ||||
| 	if from < b.start { | ||||
| 		if b.prev != nil { | ||||
| 			return b.prev.stats(from, to) | ||||
| 		} | ||||
| 		from = b.start | ||||
| 	} | ||||
|  | ||||
| 	// TODO: Check if b.closed and if so and the full buffer is queried, | ||||
| 	// use b.statistics instead of iterating over the buffer. | ||||
|  | ||||
| 	samples := 0 | ||||
| 	sum, min, max := 0.0, math.MaxFloat32, -math.MaxFloat32 | ||||
|  | ||||
| 	var t int64 | ||||
| 	for t = from; t < to; t += b.frequency { | ||||
| 		idx := int((t - b.start) / b.frequency) | ||||
| 		if idx >= cap(b.data) { | ||||
| 			b = b.next | ||||
| 			if b == nil { | ||||
| 				break | ||||
| 			} | ||||
| 			idx = 0 | ||||
| 		} | ||||
|  | ||||
| 		if t < b.start || idx >= len(b.data) { | ||||
| 			continue | ||||
| 		} | ||||
|  | ||||
| 		xf := float64(b.data[idx]) | ||||
| 		if math.IsNaN(xf) { | ||||
| 			continue | ||||
| 		} | ||||
|  | ||||
| 		samples += 1 | ||||
| 		sum += xf | ||||
| 		min = math.Min(min, xf) | ||||
| 		max = math.Max(max, xf) | ||||
| 	} | ||||
|  | ||||
| 	return Stats{ | ||||
| 		Samples: samples, | ||||
| 		Avg:     util.Float(sum) / util.Float(samples), | ||||
| 		Min:     util.Float(min), | ||||
| 		Max:     util.Float(max), | ||||
| 	}, from, t, nil | ||||
| } | ||||
|  | ||||
| // Returns statistics for the requested metric on the selected node/level. | ||||
| // Data is aggregated to the selected level the same way as in `MemoryStore.Read`. | ||||
| // If `Stats.Samples` is zero, the statistics should not be considered as valid. | ||||
| func (m *MemoryStore) Stats(selector Selector, metric string, from, to int64) (*Stats, int64, int64, error) { | ||||
| 	if from > to { | ||||
| 		return nil, 0, 0, errors.New("invalid time range") | ||||
| 	} | ||||
|  | ||||
| 	minfo, ok := m.Metrics[metric] | ||||
| 	if !ok { | ||||
| 		return nil, 0, 0, errors.New("unkown metric: " + metric) | ||||
| 	} | ||||
|  | ||||
| 	n, samples := 0, 0 | ||||
| 	avg, min, max := util.Float(0), math.MaxFloat32, -math.MaxFloat32 | ||||
| 	err := m.root.findBuffers(selector, minfo.Offset, func(b *buffer) error { | ||||
| 		stats, cfrom, cto, err := b.stats(from, to) | ||||
| 		if err != nil { | ||||
| 			return err | ||||
| 		} | ||||
|  | ||||
| 		if n == 0 { | ||||
| 			from, to = cfrom, cto | ||||
| 		} else if from != cfrom || to != cto { | ||||
| 			return ErrDataDoesNotAlign | ||||
| 		} | ||||
|  | ||||
| 		samples += stats.Samples | ||||
| 		avg += stats.Avg | ||||
| 		min = math.Min(min, float64(stats.Min)) | ||||
| 		max = math.Max(max, float64(stats.Max)) | ||||
| 		n += 1 | ||||
| 		return nil | ||||
| 	}) | ||||
| 	if err != nil { | ||||
| 		return nil, 0, 0, err | ||||
| 	} | ||||
|  | ||||
| 	if n == 0 { | ||||
| 		return nil, 0, 0, ErrNoData | ||||
| 	} | ||||
|  | ||||
| 	if minfo.Aggregation == config.AvgAggregation { | ||||
| 		avg /= util.Float(n) | ||||
| 	} else if n > 1 && minfo.Aggregation != config.SumAggregation { | ||||
| 		return nil, 0, 0, errors.New("invalid aggregation") | ||||
| 	} | ||||
|  | ||||
| 	return &Stats{ | ||||
| 		Samples: samples, | ||||
| 		Avg:     avg, | ||||
| 		Min:     util.Float(min), | ||||
| 		Max:     util.Float(max), | ||||
| 	}, from, to, nil | ||||
| } | ||||
		Reference in New Issue
	
	Block a user