-
Notifications
You must be signed in to change notification settings - Fork 123
/
carbonserver.go
1542 lines (1361 loc) · 42.2 KB
/
carbonserver.go
1
2
3
4
5
6
7
8
9
10
11
12
13
14
15
16
17
18
19
20
21
22
23
24
25
26
27
28
29
30
31
32
33
34
35
36
37
38
39
40
41
42
43
44
45
46
47
48
49
50
51
52
53
54
55
56
57
58
59
60
61
62
63
64
65
66
67
68
69
70
71
72
73
74
75
76
77
78
79
80
81
82
83
84
85
86
87
88
89
90
91
92
93
94
95
96
97
98
99
100
101
102
103
104
105
106
107
108
109
110
111
112
113
114
115
116
117
118
119
120
121
122
123
124
125
126
127
128
129
130
131
132
133
134
135
136
137
138
139
140
141
142
143
144
145
146
147
148
149
150
151
152
153
154
155
156
157
158
159
160
161
162
163
164
165
166
167
168
169
170
171
172
173
174
175
176
177
178
179
180
181
182
183
184
185
186
187
188
189
190
191
192
193
194
195
196
197
198
199
200
201
202
203
204
205
206
207
208
209
210
211
212
213
214
215
216
217
218
219
220
221
222
223
224
225
226
227
228
229
230
231
232
233
234
235
236
237
238
239
240
241
242
243
244
245
246
247
248
249
250
251
252
253
254
255
256
257
258
259
260
261
262
263
264
265
266
267
268
269
270
271
272
273
274
275
276
277
278
279
280
281
282
283
284
285
286
287
288
289
290
291
292
293
294
295
296
297
298
299
300
301
302
303
304
305
306
307
308
309
310
311
312
313
314
315
316
317
318
319
320
321
322
323
324
325
326
327
328
329
330
331
332
333
334
335
336
337
338
339
340
341
342
343
344
345
346
347
348
349
350
351
352
353
354
355
356
357
358
359
360
361
362
363
364
365
366
367
368
369
370
371
372
373
374
375
376
377
378
379
380
381
382
383
384
385
386
387
388
389
390
391
392
393
394
395
396
397
398
399
400
401
402
403
404
405
406
407
408
409
410
411
412
413
414
415
416
417
418
419
420
421
422
423
424
425
426
427
428
429
430
431
432
433
434
435
436
437
438
439
440
441
442
443
444
445
446
447
448
449
450
451
452
453
454
455
456
457
458
459
460
461
462
463
464
465
466
467
468
469
470
471
472
473
474
475
476
477
478
479
480
481
482
483
484
485
486
487
488
489
490
491
492
493
494
495
496
497
498
499
500
501
502
503
504
505
506
507
508
509
510
511
512
513
514
515
516
517
518
519
520
521
522
523
524
525
526
527
528
529
530
531
532
533
534
535
536
537
538
539
540
541
542
543
544
545
546
547
548
549
550
551
552
553
554
555
556
557
558
559
560
561
562
563
564
565
566
567
568
569
570
571
572
573
574
575
576
577
578
579
580
581
582
583
584
585
586
587
588
589
590
591
592
593
594
595
596
597
598
599
600
601
602
603
604
605
606
607
608
609
610
611
612
613
614
615
616
617
618
619
620
621
622
623
624
625
626
627
628
629
630
631
632
633
634
635
636
637
638
639
640
641
642
643
644
645
646
647
648
649
650
651
652
653
654
655
656
657
658
659
660
661
662
663
664
665
666
667
668
669
670
671
672
673
674
675
676
677
678
679
680
681
682
683
684
685
686
687
688
689
690
691
692
693
694
695
696
697
698
699
700
701
702
703
704
705
706
707
708
709
710
711
712
713
714
715
716
717
718
719
720
721
722
723
724
725
726
727
728
729
730
731
732
733
734
735
736
737
738
739
740
741
742
743
744
745
746
747
748
749
750
751
752
753
754
755
756
757
758
759
760
761
762
763
764
765
766
767
768
769
770
771
772
773
774
775
776
777
778
779
780
781
782
783
784
785
786
787
788
789
790
791
792
793
794
795
796
797
798
799
800
801
802
803
804
805
806
807
808
809
810
811
812
813
814
815
816
817
818
819
820
821
822
823
824
825
826
827
828
829
830
831
832
833
834
835
836
837
838
839
840
841
842
843
844
845
846
847
848
849
850
851
852
853
854
855
856
857
858
859
860
861
862
863
864
865
866
867
868
869
870
871
872
873
874
875
876
877
878
879
880
881
882
883
884
885
886
887
888
889
890
891
892
893
894
895
896
897
898
899
900
901
902
903
904
905
906
907
908
909
910
911
912
913
914
915
916
917
918
919
920
921
922
923
924
925
926
927
928
929
930
931
932
933
934
935
936
937
938
939
940
941
942
943
944
945
946
947
948
949
950
951
952
953
954
955
956
957
958
959
960
961
962
963
964
965
966
967
968
969
970
971
972
973
974
975
976
977
978
979
980
981
982
983
984
985
986
987
988
989
990
991
992
993
994
995
996
997
998
999
1000
/*
* Copyright 2013-2016 Fabian Groffen, Damian Gryski, Vladimir Smirnov
*
* Licensed under the Apache License, Version 2.0 (the "License");
* you may not use this file except in compliance with the License.
* You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package carbonserver
import (
"bufio"
"compress/gzip"
"context"
"encoding/binary"
"errors"
"fmt"
"math"
"net"
"net/http"
_ "net/http/pprof"
"os"
"path/filepath"
"runtime"
"runtime/debug"
"sort"
"strconv"
"strings"
"sync"
"sync/atomic"
"syscall"
"time"
prom "github.com/prometheus/client_golang/prometheus"
"go.uber.org/zap"
"github.com/NYTimes/gziphandler"
"github.com/dgryski/go-expirecache"
"github.com/dgryski/go-trigram"
"github.com/dgryski/httputil"
"github.com/go-graphite/go-carbon/helper"
"github.com/go-graphite/go-carbon/helper/stat"
"github.com/go-graphite/go-carbon/points"
protov3 "github.com/go-graphite/protocol/carbonapi_v3_pb"
"github.com/lomik/zapwriter"
"github.com/syndtr/goleveldb/leveldb"
"github.com/syndtr/goleveldb/leveldb/filter"
"github.com/syndtr/goleveldb/leveldb/opt"
)
type metricStruct struct {
RenderRequests uint64
RenderErrors uint64
NotFound uint64
FindRequests uint64
FindErrors uint64
FindZero uint64
InfoRequests uint64
InfoErrors uint64
ListRequests uint64
ListErrors uint64
DetailsRequests uint64
DetailsErrors uint64
CacheHit uint64
CacheMiss uint64
CacheRequestsTotal uint64
CacheWorkTimeNS uint64
CacheWaitTimeFetchNS uint64
DiskWaitTimeNS uint64
DiskRequests uint64
PointsReturned uint64
MetricsReturned uint64
MetricsKnown uint64
FileScanTimeNS uint64
IndexBuildTimeNS uint64
MetricsFetched uint64
MetricsFound uint64
FetchSize uint64
QueryCacheHit uint64
QueryCacheMiss uint64
FindCacheHit uint64
FindCacheMiss uint64
TrieNodes uint64
TrieFiles uint64
TrieDirs uint64
}
type requestsTimes struct {
sync.RWMutex
list []int64
}
const (
QueryIsPending uint64 = 1 << iota
DataIsAvailable
)
type QueryItem struct {
Data atomic.Value
Flags uint64 // DataIsAvailable or QueryIsPending
QueryFinished chan struct{}
}
// type GlobResponse struct {
type ExpandedGlobResponse struct {
Name string
Files []string
Leafs []bool
Err error
}
var statusCodes = map[string][]uint64{
"combined": make([]uint64, 5),
"find": make([]uint64, 5),
"list": make([]uint64, 5),
"render": make([]uint64, 5),
"details": make([]uint64, 5),
"info": make([]uint64, 5),
"capabilities": make([]uint64, 5),
}
// interface to retrive retention and aggregation
// schema from persister.
type configRetriever interface {
MetricRetentionPeriod(string) (int, bool)
MetricAggrConf(string) (string, float64, bool)
}
type responseWriterWithStatus struct {
http.ResponseWriter
statusCode int
}
func (rw responseWriterWithStatus) statusCodeMajor() int {
return rw.statusCode/100 - 1
}
func newResponseWriterWithStatus(w http.ResponseWriter) *responseWriterWithStatus {
return &responseWriterWithStatus{
w,
http.StatusOK,
}
}
func (w *responseWriterWithStatus) WriteHeader(code int) {
w.statusCode = code
w.ResponseWriter.WriteHeader(code)
}
func (q *QueryItem) FetchOrLock() (interface{}, bool) {
d := q.Data.Load()
if d != nil {
return d, true
}
ok := atomic.CompareAndSwapUint64(&q.Flags, 0, QueryIsPending)
if ok {
// We are the leader now and will be fetching the data
return nil, false
}
select { //nolint:gosimple
// TODO: Add timeout support
case <-q.QueryFinished:
break
}
return q.Data.Load(), true
}
func (q *QueryItem) StoreAbort() {
oldChan := q.QueryFinished
q.QueryFinished = make(chan struct{})
close(oldChan)
atomic.StoreUint64(&q.Flags, 0)
}
func (q *QueryItem) StoreAndUnlock(data interface{}) {
q.Data.Store(data)
atomic.StoreUint64(&q.Flags, DataIsAvailable)
close(q.QueryFinished)
}
type queryCache struct {
ec *expirecache.Cache
}
func (q *queryCache) getQueryItem(k string, size uint64, expire int32) *QueryItem {
emptyQueryItem := &QueryItem{QueryFinished: make(chan struct{})}
return q.ec.GetOrSet(k, emptyQueryItem, size, expire).(*QueryItem)
}
type CarbonserverListener struct {
helper.Stoppable
cacheGet func(key string) []points.Point
readTimeout time.Duration
idleTimeout time.Duration
writeTimeout time.Duration
whisperData string
buckets int
maxGlobs int
failOnMaxGlobs bool
percentiles []int
scanFrequency time.Duration
forceScanChan chan struct{}
metricsAsCounters bool
tcpListener *net.TCPListener
logger *zap.Logger
accessLogger *zap.Logger
internalStatsDir string
flock bool
compressed bool
removeEmptyFile bool
maxMetricsGlobbed int
maxMetricsRendered int
queryCacheEnabled bool
queryCacheSizeMB int
queryCache queryCache
findCacheEnabled bool
findCache queryCache
trigramIndex bool
trieIndex bool
concurrentIndex bool
fileListCache string
realtimeIndex int
newMetricsChan chan string
fileIdx atomic.Value
fileIdxMutex sync.Mutex
metrics *metricStruct
requestsTimes requestsTimes
exitChan chan struct{}
timeBuckets []uint64
cacheGetRecentMetrics func() []map[string]struct{}
whisperGetConfig configRetriever
prometheus prometheus
db *leveldb.DB
}
type prometheus struct {
enabled bool
requests *prom.CounterVec
request func(string, int)
durations prom.Histogram
duration func(time.Duration)
cacheRequests *prom.CounterVec
cacheRequest func(string, bool)
cacheDurations *prom.HistogramVec
cacheDuration func(string, time.Duration)
diskRequests prom.Counter
diskRequest func()
cancelledRequests prom.Counter
cancelledRequest func()
timeoutRequests prom.Counter
timeoutRequest func()
diskWaitDurations prom.Histogram
diskWaitDuration func(time.Duration)
returnedMetrics prom.Counter
returnedMetric func()
returnedPoints prom.Counter
returnedPoint func(int)
}
func (c *CarbonserverListener) InitPrometheus(reg prom.Registerer) {
c.prometheus = prometheus{
enabled: true,
requests: prom.NewCounterVec(
prom.CounterOpts{
Name: "http_requests_total",
Help: "How many HTTP requests processed, partitioned by status code and handler",
},
[]string{"code", "handler"},
),
cacheRequests: prom.NewCounterVec(
prom.CounterOpts{
Name: "cache_requests_total",
Help: "Cache counts, partitioned by type and hit/miss",
},
[]string{"type", "hit"},
),
durations: prom.NewHistogram(
prom.HistogramOpts{
Name: "http_request_duration_seconds_exp",
Help: "Duration of HTTP requests (exponential buckets)",
Buckets: prom.ExponentialBuckets(time.Millisecond.Seconds(), 2.0, 20),
},
),
cacheDurations: prom.NewHistogramVec(
prom.HistogramOpts{
Name: "cache_duration_seconds_exp",
Help: "Time spent in cache (exponential buckets)",
Buckets: prom.ExponentialBuckets(time.Millisecond.Seconds(), 2.0, 20),
},
[]string{"type"},
),
diskRequests: prom.NewCounter(prom.CounterOpts{
Name: "disk_requests_total",
Help: "Number of times disk has been hit",
}),
cancelledRequests: prom.NewCounter(prom.CounterOpts{
Name: "cancelled_requests_total",
Help: "Number of times a request has been cancelled",
}),
timeoutRequests: prom.NewCounter(prom.CounterOpts{
Name: "timeout_requests_total",
Help: "Number of times a request has been timeout",
}),
diskWaitDurations: prom.NewHistogram(
prom.HistogramOpts{
Name: "disk_wait_seconds_exp",
Help: "Duration of disk wait times (exponential buckets)",
Buckets: prom.ExponentialBuckets(time.Millisecond.Seconds(), 2.0, 20),
},
),
returnedMetrics: prom.NewCounter(prom.CounterOpts{
Name: "returned_metrics_total",
Help: "Number of metrics returned",
}),
returnedPoints: prom.NewCounter(prom.CounterOpts{
Name: "returned_points_total",
Help: "Number of points returned",
}),
}
c.prometheus.request = func(endpoint string, code int) {
c.prometheus.requests.WithLabelValues(strconv.Itoa(code), endpoint).Inc()
}
c.prometheus.cacheRequest = func(kind string, hit bool) {
c.prometheus.cacheRequests.WithLabelValues(kind, strconv.FormatBool(hit))
}
c.prometheus.duration = func(t time.Duration) {
c.prometheus.durations.Observe(t.Seconds())
}
c.prometheus.cacheDuration = func(kind string, t time.Duration) {
c.prometheus.cacheDurations.WithLabelValues(kind).Observe(t.Seconds())
}
c.prometheus.diskRequest = func() {
c.prometheus.diskRequests.Inc()
}
c.prometheus.cancelledRequest = func() {
c.prometheus.cancelledRequests.Inc()
}
c.prometheus.timeoutRequest = func() {
c.prometheus.timeoutRequests.Inc()
}
c.prometheus.diskWaitDuration = func(t time.Duration) {
c.prometheus.diskWaitDurations.Observe(t.Seconds())
}
c.prometheus.returnedMetric = func() {
c.prometheus.returnedMetrics.Inc()
}
c.prometheus.returnedPoint = func(i int) {
c.prometheus.returnedPoints.Add(float64(i))
}
reg.MustRegister(c.prometheus.requests)
reg.MustRegister(c.prometheus.cacheRequests)
reg.MustRegister(c.prometheus.cancelledRequests)
reg.MustRegister(c.prometheus.timeoutRequests)
reg.MustRegister(c.prometheus.durations)
reg.MustRegister(c.prometheus.diskRequests)
reg.MustRegister(c.prometheus.diskWaitDurations)
reg.MustRegister(c.prometheus.returnedMetrics)
reg.MustRegister(c.prometheus.returnedPoints)
}
type metricDetailsFlat struct {
*protov3.MetricDetails
Name string
}
type jsonMetricDetailsResponse struct {
Metrics []metricDetailsFlat
FreeSpace uint64
TotalSpace uint64
}
type fileIndex struct {
typ int //nolint:unused,structcheck
idx trigram.Index
files []string
trieIdx *trieIndex
details map[string]*protov3.MetricDetails
accessTimes map[string]int64
freeSpace uint64
totalSpace uint64
}
func NewCarbonserverListener(cacheGetFunc func(key string) []points.Point) *CarbonserverListener {
return &CarbonserverListener{
// Config variables
metrics: &metricStruct{},
metricsAsCounters: false,
cacheGet: cacheGetFunc,
logger: zapwriter.Logger("carbonserver"),
accessLogger: zapwriter.Logger("access"),
findCache: queryCache{ec: expirecache.New(0)},
trigramIndex: true,
percentiles: []int{100, 99, 98, 95, 75, 50},
prometheus: prometheus{
request: func(string, int) {},
duration: func(time.Duration) {},
cacheRequest: func(string, bool) {},
cacheDuration: func(string, time.Duration) {},
diskRequest: func() {},
cancelledRequest: func() {},
timeoutRequest: func() {},
diskWaitDuration: func(time.Duration) {},
returnedMetric: func() {},
returnedPoint: func(int) {},
},
}
}
func (listener *CarbonserverListener) SetWhisperData(whisperData string) {
listener.whisperData = strings.TrimRight(whisperData, "/")
}
func (listener *CarbonserverListener) SetMaxGlobs(maxGlobs int) {
listener.maxGlobs = maxGlobs
}
func (listener *CarbonserverListener) SetFailOnMaxGlobs(failOnMaxGlobs bool) {
listener.failOnMaxGlobs = failOnMaxGlobs
}
func (listener *CarbonserverListener) SetMaxMetricsGlobbed(max int) {
listener.maxMetricsGlobbed = max
}
func (listener *CarbonserverListener) SetMaxMetricsRendered(max int) {
listener.maxMetricsRendered = max
}
func (listener *CarbonserverListener) SetFLock(flock bool) {
listener.flock = flock
}
func (listener *CarbonserverListener) SetBuckets(buckets int) {
listener.buckets = buckets
}
func (listener *CarbonserverListener) SetScanFrequency(scanFrequency time.Duration) {
listener.scanFrequency = scanFrequency
}
func (listener *CarbonserverListener) SetReadTimeout(readTimeout time.Duration) {
listener.readTimeout = readTimeout
}
func (listener *CarbonserverListener) SetIdleTimeout(idleTimeout time.Duration) {
listener.idleTimeout = idleTimeout
}
func (listener *CarbonserverListener) SetWriteTimeout(writeTimeout time.Duration) {
listener.writeTimeout = writeTimeout
}
func (listener *CarbonserverListener) SetCompressed(compressed bool) {
listener.compressed = compressed
}
func (listener *CarbonserverListener) SetRemoveEmptyFile(remove bool) {
listener.removeEmptyFile = remove
}
func (listener *CarbonserverListener) SetMetricsAsCounters(metricsAsCounters bool) {
listener.metricsAsCounters = metricsAsCounters
}
func (listener *CarbonserverListener) SetQueryCacheEnabled(enabled bool) {
listener.queryCacheEnabled = enabled
}
func (listener *CarbonserverListener) SetQueryCacheSizeMB(size int) {
listener.queryCacheSizeMB = size
}
func (listener *CarbonserverListener) SetFindCacheEnabled(enabled bool) {
listener.findCacheEnabled = enabled
}
func (listener *CarbonserverListener) SetTrigramIndex(enabled bool) {
listener.trigramIndex = enabled
}
func (listener *CarbonserverListener) SetTrieIndex(enabled bool) {
listener.trieIndex = enabled
}
func (listener *CarbonserverListener) SetCacheGetMetricsFunc(recentMetricsFunc func() []map[string]struct{}) {
listener.cacheGetRecentMetrics = recentMetricsFunc
}
func (listener *CarbonserverListener) SetConfigRetriever(retriever configRetriever) {
listener.whisperGetConfig = retriever
}
func (listener *CarbonserverListener) SetConcurrentIndex(enabled bool) {
listener.concurrentIndex = enabled
}
func (listener *CarbonserverListener) SetRealtimeIndex(num int) chan string {
listener.realtimeIndex = num
listener.newMetricsChan = make(chan string, num)
return listener.newMetricsChan
}
func (listener *CarbonserverListener) SetFileListCache(path string) {
listener.fileListCache = path
}
func (listener *CarbonserverListener) SetInternalStatsDir(dbPath string) {
listener.internalStatsDir = dbPath
}
func (listener *CarbonserverListener) SetPercentiles(percentiles []int) {
listener.percentiles = percentiles
}
func (listener *CarbonserverListener) CurrentFileIndex() *fileIndex {
p := listener.fileIdx.Load()
if p == nil {
return nil
}
return p.(*fileIndex)
}
func (listener *CarbonserverListener) UpdateFileIndex(fidx *fileIndex) { listener.fileIdx.Store(fidx) }
func (listener *CarbonserverListener) UpdateMetricsAccessTimes(metrics map[string]int64, initial bool) {
idx := listener.CurrentFileIndex()
if idx == nil {
return
}
listener.fileIdxMutex.Lock()
defer listener.fileIdxMutex.Unlock()
batch := new(leveldb.Batch)
for m, t := range metrics {
if _, ok := idx.details[m]; ok {
idx.details[m].RdTime = t
} else {
idx.details[m] = &protov3.MetricDetails{RdTime: t}
}
idx.accessTimes[m] = t
if !initial && listener.db != nil {
buf := make([]byte, 10)
binary.PutVarint(buf, t)
batch.Put([]byte(m), buf)
}
}
if !initial && listener.db != nil {
err := listener.db.Write(batch, nil)
if err != nil {
listener.logger.Info("Error updating database",
zap.Error(err),
)
}
}
}
func (listener *CarbonserverListener) UpdateMetricsAccessTimesByRequest(metrics []string) {
now := time.Now().Unix()
accessTimes := make(map[string]int64)
for _, m := range metrics {
accessTimes[m] = now
}
listener.UpdateMetricsAccessTimes(accessTimes, false)
}
func splitAndInsert(cacheMetricNames map[string]struct{}, newCacheMetricNames []map[string]struct{}) map[string]struct{} {
// splits each new metric from cache-scan and inserts
// into the current cacheMetricNames map
// in: new.metric.name1 --> split by "."
// insert "/new" , "/new/metric", "/new/metric/name1.wsp" into the
// metricsName map. This is inline with the inserts
// during filescan walk
for _, shardAddMap := range newCacheMetricNames {
for newMetric := range shardAddMap {
split := strings.Split(newMetric, ".")
fileName := "/"
for i, seg := range split {
fileName = filepath.Join(fileName, seg)
if i == len(split)-1 {
fileName += ".wsp"
}
if _, ok := cacheMetricNames[fileName]; !ok {
cacheMetricNames[fileName] = struct{}{}
}
}
}
}
return cacheMetricNames
}
func (listener *CarbonserverListener) fileListUpdater(dir string, tick <-chan time.Time, force <-chan struct{}, exit <-chan struct{}) {
cacheMetricNames := make(map[string]struct{})
uloop:
for {
select {
case <-exit:
return
case <-tick:
case <-force:
case m := <-listener.newMetricsChan:
fidx := listener.CurrentFileIndex()
if listener.trieIndex && listener.concurrentIndex && fidx != nil && fidx.trieIdx != nil {
metric := "/" + filepath.Clean(strings.ReplaceAll(m, ".", "/")+".wsp")
if err := fidx.trieIdx.insert(metric); err != nil {
listener.logger.Warn("failed to insert new metrics for realtime indexing", zap.String("metric", metric), zap.Error(err))
}
}
continue uloop
}
if listener.cacheGetRecentMetrics != nil {
// cacheMetricNames maintains all new metric names added in cache
// when cache-scan is enabled in conf
newCacheMetricNames := listener.cacheGetRecentMetrics()
cacheMetricNames = splitAndInsert(cacheMetricNames, newCacheMetricNames)
}
if listener.updateFileList(dir, cacheMetricNames) {
listener.logger.Info("file list updated with cache, starting a new scan immediately")
listener.updateFileList(dir, cacheMetricNames)
}
}
}
func (listener *CarbonserverListener) updateFileList(dir string, cacheMetricNames map[string]struct{}) (readFromCache bool) {
logger := listener.logger.With(zap.String("handler", "fileListUpdated"))
defer func() {
if r := recover(); r != nil {
logger.Error("panic encountered",
zap.Stack("stack"),
zap.Any("error", r),
)
}
}()
var t0 = time.Now()
var fidx = listener.CurrentFileIndex()
var files []string
var filesLen int
var details = make(map[string]*protov3.MetricDetails)
var trieIdx *trieIndex
var metricsKnown uint64
var infos []zap.Field
if listener.trieIndex {
if fidx == nil || !listener.concurrentIndex {
trieIdx = newTrie(".wsp")
} else {
trieIdx = fidx.trieIdx
trieIdx.root.gen++
}
}
// populate index for all the metric names in cache
// the iteration takes place only when cache-scan is enabled in conf
var tcache = time.Now()
var cacheMetricLen = len(cacheMetricNames)
for fileName := range cacheMetricNames {
if listener.trieIndex {
if err := trieIdx.insert(fileName); err != nil {
logger.Error("error populating index from cache indexMap",
zap.Error(err))
}
} else {
files = append(files, fileName)
}
if strings.HasSuffix(fileName, ".wsp") {
metricsKnown++
}
}
cacheIndexRuntime := time.Since(tcache)
// readFromCache hould only occur once at the start of the program
if fidx == nil && listener.fileListCache != "" {
fileListCache, err := newFileListCache(listener.fileListCache, 'r')
if err != nil {
if !os.IsNotExist(err) {
logger.Error("failed to read file list cache", zap.Error(err))
}
} else {
readFromCache = true
for fileListCache.scanner.Scan() {
entry := fileListCache.scanner.Text()
if entry == "" {
continue
}
if err := trieIdx.insert(entry); err != nil {
logger.Error("failed to read from file list cache", zap.Error(err))
readFromCache = false
trieIdx = newTrie(".wsp")
break
}
filesLen++
if strings.HasSuffix(entry, ".wsp") {
metricsKnown++
}
}
if err := fileListCache.close(); err != nil {
logger.Error("failed to close file list cache", zap.Error(err))
}
}
}
if !readFromCache {
var flc *fileListCache
if listener.fileListCache != "" {
var err error
flc, err = newFileListCache(listener.fileListCache, 'w')
if err != nil {
if !os.IsNotExist(err) {
logger.Error("failed to create file list cache", zap.Error(err))
}
} else {
defer func() {
// flc could be reset to nil during filepath walk
if flc != nil {
if err := flc.close(); err != nil {
logger.Error("failed to close flie list cache", zap.Error(err))
}
}
}()
}
}
err := filepath.Walk(dir, func(p string, info os.FileInfo, err error) error {
if err != nil {
logger.Info("error processing", zap.String("path", p), zap.Error(err))
return nil
}
// && len(listener.newMetricsChan) >= cap(listener.newMetricsChan)/2
if listener.trieIndex && listener.concurrentIndex && listener.newMetricsChan != nil {
newMetricsLoop:
for {
select {
case m := <-listener.newMetricsChan:
if err := trieIdx.insert(filepath.Clean(strings.ReplaceAll(m, ".", "/") + ".wsp")); err != nil {
logger.Warn("failed to update realtime trie index", zap.Error(err))
}
default:
break newMetricsLoop
}
}
}
isFullMetric := strings.HasSuffix(info.Name(), ".wsp")
if info.IsDir() || isFullMetric {
trimmedName := strings.TrimPrefix(p, listener.whisperData)
filesLen++
if flc != nil {
if err := flc.write(trimmedName); err != nil {
logger.Error("failed to write to file list cache", zap.Error(err))
if err := flc.close(); err != nil {
logger.Error("failed to close flie list cache", zap.Error(err))
}
flc = nil
}
}
// use cacheMetricNames to check and prevent appending duplicate metrics
// into the index when cacheMetricNamesIndex is enabled
if _, present := cacheMetricNames[trimmedName]; present {
delete(cacheMetricNames, trimmedName)
} else {
if listener.trieIndex {
if err := trieIdx.insert(trimmedName); err != nil {
return fmt.Errorf("updateFileList.trie: %s", err)
}
} else {
files = append(files, trimmedName)
}
if isFullMetric {
metricsKnown++
}
}
if isFullMetric && listener.internalStatsDir != "" {
i := stat.GetStat(info)
trimmedName = strings.ReplaceAll(trimmedName[1:len(trimmedName)-4], "/", ".")
details[trimmedName] = &protov3.MetricDetails{
Size_: i.Size,
ModTime: i.MTime,
ATime: i.ATime,
RealSize: i.RealSize,
}
}
}
return nil
})
if err != nil {
logger.Error("error getting file list",
zap.Error(err),
)
}
}
if listener.concurrentIndex && trieIdx != nil {
trieIdx.prune()
}
var stat syscall.Statfs_t
if err := syscall.Statfs(dir, &stat); err != nil {
logger.Info("error getting FS Stats",
zap.String("dir", dir),
zap.Error(err),
)
return
}
var freeSpace uint64
// diskspace can be negative and Bavail is therefore int64
if stat.Bavail >= 0 { // nolint:staticcheck // skipcq: SCC-SA4003
freeSpace = uint64(stat.Bavail) * uint64(stat.Bsize)
}
totalSpace := stat.Blocks * uint64(stat.Bsize)
fileScanRuntime := time.Since(t0)
atomic.StoreUint64(&listener.metrics.MetricsKnown, metricsKnown)
atomic.AddUint64(&listener.metrics.FileScanTimeNS, uint64(fileScanRuntime.Nanoseconds()))
nfidx := &fileIndex{
details: details,
freeSpace: freeSpace,
totalSpace: totalSpace,
accessTimes: make(map[string]int64),
}
var pruned int
var indexType = "trigram"
var tindex = time.Now()
var indexSize int
if listener.trieIndex {
indexType = "trie"
nfidx.trieIdx = trieIdx
infos = append(
infos,
zap.Int("trie_depth", int(nfidx.trieIdx.depth)),
zap.String("longest_metric", nfidx.trieIdx.longestMetric),
)
if listener.trigramIndex && !listener.concurrentIndex {
start := time.Now()
nfidx.trieIdx.setTrigrams()
infos = append(infos, zap.Duration("set_trigram_time", time.Since(start)))
}
start := time.Now()
count, files, dirs, _, _, _, _, _ := trieIdx.countNodes()
atomic.StoreUint64(&listener.metrics.TrieNodes, uint64(count))
atomic.StoreUint64(&listener.metrics.TrieFiles, uint64(files))
atomic.StoreUint64(&listener.metrics.TrieDirs, uint64(dirs))
infos = append(infos, zap.Duration("trie_count_nodes_time", time.Since(start)))
indexSize = count
} else {
nfidx.files = files
nfidx.idx = trigram.NewIndex(files)
pruned = nfidx.idx.Prune(0.95)
indexSize = len(nfidx.idx)
}
indexingRuntime := time.Since(tindex) // note: no longer meaningful for trie index
atomic.AddUint64(&listener.metrics.IndexBuildTimeNS, uint64(indexingRuntime.Nanoseconds()))
var tl = time.Now()
if fidx != nil && listener.internalStatsDir != "" {
listener.fileIdxMutex.Lock()
for m := range fidx.accessTimes {
if d, ok := details[m]; ok {
d.RdTime = fidx.accessTimes[m]
} else {
delete(fidx.accessTimes, m)
if listener.db != nil {
listener.db.Delete([]byte(m), nil)
}
}
}
nfidx.accessTimes = fidx.accessTimes
listener.fileIdxMutex.Unlock()
}
var rdTimeUpdateRuntime = time.Since(tl)
listener.UpdateFileIndex(nfidx)
infos = append(infos,
zap.Duration("file_scan_runtime", fileScanRuntime),
zap.Duration("indexing_runtime", indexingRuntime),
zap.Duration("rdtime_update_runtime", rdTimeUpdateRuntime),
zap.Duration("cache_index_runtime", cacheIndexRuntime),
zap.Duration("total_runtime", time.Since(t0)),
zap.Int("Files", filesLen),
zap.Int("index_size", indexSize),
zap.Int("pruned_trigrams", pruned),
zap.Int("cache_metric_len_before", cacheMetricLen),
zap.Int("cache_metric_len_after", len(cacheMetricNames)),
zap.Uint64("metrics_known", metricsKnown),
zap.String("index_type", indexType),
zap.Bool("read_from_cache", readFromCache),
)
logger.Info("file list updated", infos...)
return
}
func (listener *CarbonserverListener) expandGlobs(ctx context.Context, query string, resultCh chan<- *ExpandedGlobResponse) {
defer func() {
if err := recover(); err != nil {
resultCh <- &ExpandedGlobResponse{query, nil, nil, fmt.Errorf("%s\n%s", err, debug.Stack())}
}
}()
logger := TraceContextToZap(ctx, listener.logger)
matchedCount := 0
defer func(start time.Time) {
dur := time.Now().Sub(start) //nolint:gosimple
if dur <= time.Second {
return
}
var itype string
if listener.trieIndex {
itype = "trie"
if listener.trigramIndex {
itype = "trie-trigram"
}
} else if listener.trigramIndex {
itype = "trigram"
}
logger.Info("slow_expand_globs", zap.Duration("time", dur), zap.String("query", query), zap.Int("matched_count", matchedCount), zap.String("index_type", itype))
}(time.Now())
if listener.trieIndex && listener.CurrentFileIndex() != nil {
files, leafs, err := listener.expandGlobsTrie(query)
resultCh <- &ExpandedGlobResponse{query, files, leafs, err}
return
}
var useGlob bool
// TODO: Find out why we have set 'useGlob' if 'star == -1'
if star := strings.IndexByte(query, '*'); listener.cacheGetRecentMetrics == nil &&
strings.IndexByte(query, '[') == -1 &&
strings.IndexByte(query, '?') == -1 &&
(star == -1 || star == len(query)-1) {
useGlob = true
}
logger = logger.With(zap.Bool("use_glob", useGlob))
/* things to glob:
* - carbon.relays -> carbon.relays
* - carbon.re -> carbon.relays, carbon.rewhatever
* - carbon.[rz] -> carbon.relays, carbon.zipper
* - carbon.{re,zi} -> carbon.relays, carbon.zipper
* - match is either dir or .wsp file
* unfortunately, filepath.Glob doesn't handle the curly brace
* expansion for us */
query = strings.ReplaceAll(query, ".", "/")
var globs []string
if !strings.HasSuffix(query, "*") {
globs = append(globs, query+".wsp")
logger.Debug("appending file to globs struct",
zap.Strings("globs", globs),
)
}
globs = append(globs, query)
globs, err := listener.expandGlobBraces(globs)
if err != nil {
resultCh <- &ExpandedGlobResponse{query, nil, nil, err}
return
}
fidx := listener.CurrentFileIndex()
var files []string
fallbackToFS := false