Skip to content

Commit 4264547

Browse files
committed
db: improve metrics and logs to measure all filesystem ops
Enhance the log writer to instrument all filesystem operations done for writing the log file in the latency histogram (create, write, sync, etc.).
1 parent 6f00a88 commit 4264547

File tree

12 files changed

+171
-73
lines changed

12 files changed

+171
-73
lines changed

commit_test.go

Lines changed: 0 additions & 3 deletions
Original file line numberDiff line numberDiff line change
@@ -20,7 +20,6 @@ import (
2020
"github.com/cockroachdb/pebble/internal/invariants"
2121
"github.com/cockroachdb/pebble/record"
2222
"github.com/cockroachdb/pebble/vfs"
23-
"github.com/prometheus/client_golang/prometheus"
2423
"github.com/stretchr/testify/require"
2524
)
2625

@@ -249,7 +248,6 @@ func TestCommitPipelineWALClose(t *testing.T) {
249248
}
250249
p := newCommitPipeline(testEnv)
251250
wal = record.NewLogWriter(sf, 0 /* logNum */, record.LogWriterConfig{
252-
WALFsyncLatency: prometheus.NewHistogram(prometheus.HistogramOpts{}),
253251
QueueSemChan: p.logSyncQSem,
254252
WriteWALSyncOffsets: func() bool { return false },
255253
})
@@ -388,7 +386,6 @@ func BenchmarkCommitPipeline(b *testing.B) {
388386
p := newCommitPipeline(nullCommitEnv)
389387
wal = record.NewLogWriter(io.Discard, 0, /* logNum */
390388
record.LogWriterConfig{
391-
WALFsyncLatency: prometheus.NewHistogram(prometheus.HistogramOpts{}),
392389
QueueSemChan: p.logSyncQSem,
393390
WriteWALSyncOffsets: func() bool { return false },
394391
})

db.go

Lines changed: 4 additions & 11 deletions
Original file line numberDiff line numberDiff line change
@@ -35,14 +35,12 @@ import (
3535
"github.com/cockroachdb/pebble/objstorage"
3636
"github.com/cockroachdb/pebble/objstorage/remote"
3737
"github.com/cockroachdb/pebble/rangekey"
38-
"github.com/cockroachdb/pebble/record"
3938
"github.com/cockroachdb/pebble/sstable"
4039
"github.com/cockroachdb/pebble/sstable/blob"
4140
"github.com/cockroachdb/pebble/sstable/block"
4241
"github.com/cockroachdb/pebble/vfs/atomicfs"
4342
"github.com/cockroachdb/pebble/wal"
4443
"github.com/cockroachdb/tokenbucket"
45-
"github.com/prometheus/client_golang/prometheus"
4644
)
4745

4846
const (
@@ -378,13 +376,7 @@ type DB struct {
378376
// commitPipeline.mu and DB.mu to be held when rotating the WAL/memtable
379377
// (i.e. makeRoomForWrite). Can be nil.
380378
writer wal.Writer
381-
metrics struct {
382-
// fsyncLatency has its own internal synchronization, and is not
383-
// protected by mu.
384-
fsyncLatency prometheus.Histogram
385-
// Updated whenever a wal.Writer is closed.
386-
record.LogWriterMetrics
387-
}
379+
metrics WALMetrics
388380
}
389381

390382
mem struct {
@@ -1996,8 +1988,9 @@ func (d *DB) Metrics() *Metrics {
19961988
blobStats, _ := d.mu.versions.latest.blobFiles.Stats()
19971989
d.mu.versions.logUnlock()
19981990

1999-
metrics.LogWriter.FsyncLatency = d.mu.log.metrics.fsyncLatency
2000-
if err := metrics.LogWriter.Merge(&d.mu.log.metrics.LogWriterMetrics); err != nil {
1991+
metrics.WALMetrics.PrimaryFileOpLatency = d.mu.log.metrics.PrimaryFileOpLatency
1992+
metrics.WALMetrics.SecondaryFileOpLatency = d.mu.log.metrics.SecondaryFileOpLatency
1993+
if err := metrics.WALMetrics.Merge(&d.mu.log.metrics.LogWriterMetrics); err != nil {
20011994
d.opts.Logger.Errorf("metrics error: %s", err)
20021995
}
20031996
metrics.Flush.WriteThroughput = d.mu.compact.flushWriteThroughput

metrics.go

Lines changed: 23 additions & 4 deletions
Original file line numberDiff line numberDiff line change
@@ -368,10 +368,7 @@ type Metrics struct {
368368
// deletion. These can be relevant if free disk space is unexplainably low.
369369
DeletePacer deletepacer.Metrics
370370

371-
LogWriter struct {
372-
FsyncLatency prometheus.Histogram
373-
record.LogWriterMetrics
374-
}
371+
WALMetrics WALMetrics
375372

376373
CategoryStats []block.CategoryStatsAggregate
377374

@@ -526,6 +523,28 @@ func (cm *CompressionMetrics) MergeWith(o *CompressionMetrics) {
526523
cm.Zstd.Add(o.Zstd)
527524
}
528525

526+
// DirectoryContext identifies which WAL directory an operation targets
527+
type DirectoryContext int
528+
529+
const (
530+
// DirectoryPrimary indicates operation targets the primary WAL directory
531+
DirectoryPrimary DirectoryContext = iota
532+
// DirectorySecondary indicates operation targets the secondary WAL directory
533+
DirectorySecondary
534+
// DirectoryUnknown indicates directory context is unknown or not applicable
535+
DirectoryUnknown
536+
)
537+
538+
// WALMetrics contains directory-specific latency histograms for all WAL operations
539+
type WALMetrics struct {
540+
// PrimaryFileOpLatency tracks all file operations for the primary directory
541+
PrimaryFileOpLatency prometheus.Histogram
542+
// SecondaryFileOpLatency tracks all file operations for the secondary directory
543+
SecondaryFileOpLatency prometheus.Histogram
544+
// Updated whenever a wal.Writer is closed
545+
record.LogWriterMetrics
546+
}
547+
529548
var (
530549
// FsyncLatencyBuckets are prometheus histogram buckets suitable for a histogram
531550
// that records latencies for fsyncs.

open.go

Lines changed: 21 additions & 6 deletions
Original file line numberDiff line numberDiff line change
@@ -274,9 +274,8 @@ func Open(dirname string, opts *Options) (db *DB, err error) {
274274
d.mu.mem.queue = append(d.mu.mem.queue, entry)
275275
}
276276

277-
d.mu.log.metrics.fsyncLatency = prometheus.NewHistogram(prometheus.HistogramOpts{
278-
Buckets: FsyncLatencyBuckets,
279-
})
277+
// Initialize WAL metrics structure (histograms populated below)
278+
d.mu.log.metrics = WALMetrics{}
280279

281280
walOpts := wal.Options{
282281
Primary: rs.dirs.WALPrimary,
@@ -287,17 +286,33 @@ func Open(dirname string, opts *Options) (db *DB, err error) {
287286
BytesPerSync: opts.WALBytesPerSync,
288287
PreallocateSize: d.walPreallocateSize,
289288
MinSyncInterval: opts.WALMinSyncInterval,
290-
FsyncLatency: d.mu.log.metrics.fsyncLatency,
291289
QueueSemChan: d.commit.logSyncQSem,
292290
Logger: opts.Logger,
293291
EventListener: walEventListenerAdaptor{l: opts.EventListener},
294292
WriteWALSyncOffsets: func() bool { return d.FormatMajorVersion() >= FormatWALSyncChunks },
295293
}
294+
295+
// Create and assign WAL file operation histograms
296+
walPrimaryFileOpHistogram := prometheus.NewHistogram(prometheus.HistogramOpts{
297+
Buckets: FsyncLatencyBuckets,
298+
})
299+
d.mu.log.metrics.PrimaryFileOpLatency = walPrimaryFileOpHistogram
300+
walOpts.PrimaryFileOpHistogram = walPrimaryFileOpHistogram
301+
302+
// Configure failover-specific histograms and options
296303
if !opts.ReadOnly && opts.WALFailover != nil {
297-
walOpts.FailoverOptions = opts.WALFailover.FailoverOptions
298-
walOpts.FailoverWriteAndSyncLatency = prometheus.NewHistogram(prometheus.HistogramOpts{
304+
walSecondaryFileOpHistogram := prometheus.NewHistogram(prometheus.HistogramOpts{
305+
Buckets: FsyncLatencyBuckets,
306+
})
307+
d.mu.log.metrics.SecondaryFileOpLatency = walSecondaryFileOpHistogram
308+
walOpts.SecondaryFileOpHistogram = walSecondaryFileOpHistogram
309+
310+
walFailoverWriteAndSyncHistogram := prometheus.NewHistogram(prometheus.HistogramOpts{
299311
Buckets: FsyncLatencyBuckets,
300312
})
313+
walOpts.FailoverWriteAndSyncLatency = walFailoverWriteAndSyncHistogram
314+
315+
walOpts.FailoverOptions = opts.WALFailover.FailoverOptions
301316
}
302317

303318
walManager, err := wal.Init(walOpts, rs.walsReplay)

record/log_writer.go

Lines changed: 34 additions & 9 deletions
Original file line numberDiff line numberDiff line change
@@ -455,8 +455,10 @@ type LogWriter struct {
455455
err error
456456
// minSyncInterval is the minimum duration between syncs.
457457
minSyncInterval durationFunc
458-
fsyncLatency prometheus.Histogram
459-
pending []*block
458+
459+
// WAL file operation latency histogram
460+
walFileOpHistogram WALFileOpHistogram
461+
pending []*block
460462
// Pushing and popping from pendingSyncs does not require flusher mutex to
461463
// be held.
462464
pendingSyncs pendingSyncs
@@ -486,10 +488,17 @@ type LogWriter struct {
486488
emitFragment func(n int, p []byte) (remainingP []byte)
487489
}
488490

491+
// WALFileOpHistogram is a prometheus histogram for tracking WAL file operation latencies
492+
// (create, write, fsync, close, stat, opendir).
493+
type WALFileOpHistogram = prometheus.Histogram
494+
489495
// LogWriterConfig is a struct used for configuring new LogWriters
490496
type LogWriterConfig struct {
491497
WALMinSyncInterval durationFunc
492-
WALFsyncLatency prometheus.Histogram
498+
499+
// WAL file operation latency histogram for this directory
500+
WALFileOpHistogram WALFileOpHistogram
501+
493502
// QueueSemChan is an optional channel to pop from when popping from
494503
// LogWriter.flusher.syncQueue. It functions as a semaphore that prevents
495504
// the syncQueue from overflowing (which will cause a panic). All production
@@ -581,7 +590,7 @@ func NewLogWriter(
581590

582591
f := &r.flusher
583592
f.minSyncInterval = logWriterConfig.WALMinSyncInterval
584-
f.fsyncLatency = logWriterConfig.WALFsyncLatency
593+
f.walFileOpHistogram = logWriterConfig.WALFileOpHistogram
585594

586595
go func() {
587596
pprof.Do(context.Background(), walSyncLabels, r.flushLoop)
@@ -725,9 +734,9 @@ func (w *LogWriter) flushLoop(context.Context) {
725734
writtenOffset += uint64(len(data))
726735
synced, syncLatency, bytesWritten, err := w.flushPending(data, pending, snap)
727736
f.Lock()
728-
if synced && f.fsyncLatency != nil {
737+
if synced && f.walFileOpHistogram != nil {
729738
w.syncedOffset.Store(writtenOffset)
730-
f.fsyncLatency.Observe(float64(syncLatency))
739+
f.walFileOpHistogram.Observe(float64(syncLatency))
731740
}
732741
f.err = err
733742
if f.err != nil {
@@ -784,7 +793,12 @@ func (w *LogWriter) flushPending(
784793
}
785794
if n := len(data); err == nil && n > 0 {
786795
bytesWritten += int64(n)
796+
// Measure write latency
797+
writeStart := crtime.NowMono()
787798
_, err = w.w.Write(data)
799+
if w.flusher.walFileOpHistogram != nil {
800+
w.flusher.walFileOpHistogram.Observe(float64(writeStart.Elapsed()))
801+
}
788802
}
789803

790804
synced = !snap.empty()
@@ -811,7 +825,13 @@ func (w *LogWriter) syncWithLatency() (time.Duration, error) {
811825
}
812826

813827
func (w *LogWriter) flushBlock(b *block) error {
814-
if _, err := w.w.Write(b.buf[b.flushed:]); err != nil {
828+
// Measure write latency for block flush
829+
writeStart := crtime.NowMono()
830+
_, err := w.w.Write(b.buf[b.flushed:])
831+
if w.flusher.walFileOpHistogram != nil {
832+
w.flusher.walFileOpHistogram.Observe(float64(writeStart.Elapsed()))
833+
}
834+
if err != nil {
815835
return err
816836
}
817837
b.written.Store(0)
@@ -885,8 +905,8 @@ func (w *LogWriter) closeInternal(lastQueuedRecord PendingSyncIndex) error {
885905
syncLatency, err = w.syncWithLatency()
886906
}
887907
f.Lock()
888-
if err == nil && f.fsyncLatency != nil {
889-
f.fsyncLatency.Observe(float64(syncLatency))
908+
if err == nil && f.walFileOpHistogram != nil {
909+
f.walFileOpHistogram.Observe(float64(syncLatency))
890910
}
891911
free := w.free.blocks
892912
f.Unlock()
@@ -897,7 +917,12 @@ func (w *LogWriter) closeInternal(lastQueuedRecord PendingSyncIndex) error {
897917
w.pendingSyncsBackingIndex.externalSyncQueueCallback(lastQueuedRecord, err)
898918
}
899919
if w.c != nil {
920+
// Measure close latency
921+
closeStart := crtime.NowMono()
900922
cerr := w.c.Close()
923+
if w.flusher.walFileOpHistogram != nil {
924+
w.flusher.walFileOpHistogram.Observe(float64(closeStart.Elapsed()))
925+
}
901926
w.c = nil
902927
err = firstError(err, cerr)
903928
}

record/log_writer_test.go

Lines changed: 2 additions & 15 deletions
Original file line numberDiff line numberDiff line change
@@ -141,7 +141,6 @@ func TestSyncError(t *testing.T) {
141141

142142
injectedErr := errors.New("injected error")
143143
w := NewLogWriter(syncErrorFile{f, injectedErr}, 0, LogWriterConfig{
144-
WALFsyncLatency: prometheus.NewHistogram(prometheus.HistogramOpts{}),
145144
WriteWALSyncOffsets: func() bool { return false },
146145
})
147146

@@ -184,7 +183,6 @@ func (f *syncFile) Sync() error {
184183
func TestSyncRecord(t *testing.T) {
185184
f := &syncFile{}
186185
w := NewLogWriter(f, 0, LogWriterConfig{
187-
WALFsyncLatency: prometheus.NewHistogram(prometheus.HistogramOpts{}),
188186
WriteWALSyncOffsets: func() bool { return false },
189187
})
190188

@@ -212,7 +210,6 @@ func TestSyncRecordWithSignalChan(t *testing.T) {
212210
semChan <- struct{}{}
213211
}
214212
w := NewLogWriter(f, 0, LogWriterConfig{
215-
WALFsyncLatency: prometheus.NewHistogram(prometheus.HistogramOpts{}),
216213
QueueSemChan: semChan,
217214
WriteWALSyncOffsets: func() bool { return false },
218215
})
@@ -264,7 +261,6 @@ func TestMinSyncInterval(t *testing.T) {
264261
WALMinSyncInterval: func() time.Duration {
265262
return minSyncInterval
266263
},
267-
WALFsyncLatency: prometheus.NewHistogram(prometheus.HistogramOpts{}),
268264
WriteWALSyncOffsets: func() bool { return false },
269265
})
270266

@@ -337,7 +333,6 @@ func TestMinSyncIntervalClose(t *testing.T) {
337333
WALMinSyncInterval: func() time.Duration {
338334
return minSyncInterval
339335
},
340-
WALFsyncLatency: prometheus.NewHistogram(prometheus.HistogramOpts{}),
341336
WriteWALSyncOffsets: func() bool { return false },
342337
})
343338

@@ -389,7 +384,6 @@ func TestMetricsWithoutSync(t *testing.T) {
389384
f := &syncFileWithWait{}
390385
f.writeWG.Add(1)
391386
w := NewLogWriter(f, 0, LogWriterConfig{
392-
WALFsyncLatency: prometheus.NewHistogram(prometheus.HistogramOpts{}),
393387
WriteWALSyncOffsets: func() bool { return false },
394388
})
395389
offset, err := w.SyncRecord([]byte("hello"), nil, nil)
@@ -435,10 +429,9 @@ func TestMetricsWithSync(t *testing.T) {
435429
})
436430

437431
w := NewLogWriter(f, 0, LogWriterConfig{
438-
WALFsyncLatency: syncLatencyMicros,
439432
WriteWALSyncOffsets: func() bool { return false },
440-
},
441-
)
433+
WALFileOpHistogram: syncLatencyMicros,
434+
})
442435
var wg sync.WaitGroup
443436
wg.Add(100)
444437
for i := 0; i < 100; i++ {
@@ -542,7 +535,6 @@ func TestQueueWALBlocks(t *testing.T) {
542535
return nil
543536
}))
544537
w := NewLogWriter(f, 0, LogWriterConfig{
545-
WALFsyncLatency: prometheus.NewHistogram(prometheus.HistogramOpts{}),
546538
WriteWALSyncOffsets: func() bool { return false },
547539
})
548540
const numBlocks = 1024
@@ -651,7 +643,6 @@ func TestSyncRecordGeneralized(t *testing.T) {
651643
lastSync := int64(-1)
652644
cbChan := make(chan struct{}, 2)
653645
w := NewLogWriter(f, 0, LogWriterConfig{
654-
WALFsyncLatency: prometheus.NewHistogram(prometheus.HistogramOpts{}),
655646
ExternalSyncQueueCallback: func(doneSync PendingSyncIndex, err error) {
656647
require.NoError(t, err)
657648
require.Equal(t, lastSync+1, doneSync.Index)
@@ -703,7 +694,6 @@ func TestSyncRecordGeneralizedWithCloseError(t *testing.T) {
703694
lastSync := int64(-1)
704695
cbChan := make(chan struct{}, 2)
705696
w := NewLogWriter(f, 0, LogWriterConfig{
706-
WALFsyncLatency: prometheus.NewHistogram(prometheus.HistogramOpts{}),
707697
ExternalSyncQueueCallback: func(doneSync PendingSyncIndex, err error) {
708698
if doneSync.Index == 1 {
709699
require.Error(t, err)
@@ -744,7 +734,6 @@ func TestSyncRecordGeneralizedWithCloseError(t *testing.T) {
744734
func writeWALSyncRecords(t *testing.T, numRecords int, recordSizes []int) *syncFile {
745735
f := &syncFile{}
746736
w := NewLogWriter(f, 1, LogWriterConfig{
747-
WALFsyncLatency: prometheus.NewHistogram(prometheus.HistogramOpts{}),
748737
WriteWALSyncOffsets: func() bool { return true },
749738
})
750739
var syncErr error
@@ -853,7 +842,6 @@ func BenchmarkQueueWALBlocks(b *testing.B) {
853842
return nil
854843
}))
855844
w := NewLogWriter(f, 0, LogWriterConfig{
856-
WALFsyncLatency: prometheus.NewHistogram(prometheus.HistogramOpts{}),
857845
WriteWALSyncOffsets: func() bool { return false },
858846
})
859847

@@ -886,7 +874,6 @@ func BenchmarkWriteWALBlocksAllocs(b *testing.B) {
886874
b.StopTimer()
887875
f := vfstest.DiscardFile
888876
w := NewLogWriter(f, 0, LogWriterConfig{
889-
WALFsyncLatency: prometheus.NewHistogram(prometheus.HistogramOpts{}),
890877
ExternalSyncQueueCallback: func(doneSync PendingSyncIndex, err error) {},
891878
WriteWALSyncOffsets: func() bool { return false },
892879
})

0 commit comments

Comments
 (0)