From f2ae78ba7be545274cebc4c3afc9d2a7f5e6d2d6 Mon Sep 17 00:00:00 2001 From: Bilal Akhtar Date: Wed, 20 Dec 2023 16:05:28 -0500 Subject: [PATCH] storage: add setting to seed up consistency checks in tests Previously, we'd fall back to the 3 second consistency checker EventuallyFileOnlySnapshot (EFOS) wait in roachtests, which was slowing all of them down when we ran every replica through the consistency checker in post-test assertions. This change speeds up that consistency check in roachtest post-test assertions by flipping a new cluster setting to speed up EFOS waits for consistency checks after a roachtest finishes. Epic: none Unblocks #116330. Release note: None --- docs/generated/settings/settings.html | 1 + .../roachtestutil/validation_check.go | 1 + pkg/kv/kvserver/consistency_queue_test.go | 2 +- pkg/kv/kvserver/replica_consistency.go | 22 +++++++++++++++++-- pkg/kv/kvserver/replica_consistency_test.go | 8 +++---- pkg/kv/kvserver/replica_test.go | 2 +- pkg/kv/kvserver/spanset/batch.go | 7 ++++-- pkg/storage/engine.go | 6 +++-- pkg/storage/pebble.go | 6 +++-- 9 files changed, 41 insertions(+), 14 deletions(-) diff --git a/docs/generated/settings/settings.html b/docs/generated/settings/settings.html index 1005750dbcae..b221c9030a1e 100644 --- a/docs/generated/settings/settings.html +++ b/docs/generated/settings/settings.html @@ -67,6 +67,7 @@
kv.closed_timestamp.lead_for_global_reads_override
duration0sif nonzero, overrides the lead time that global_read ranges use to publish closed timestampsServerless/Dedicated/Self-Hosted (read-only)
kv.closed_timestamp.side_transport_interval
duration200msthe interval at which the closed timestamp side-transport attempts to advance each range's closed timestamp; set to 0 to disable the side-transportServerless/Dedicated/Self-Hosted (read-only)
kv.closed_timestamp.target_duration
duration3sif nonzero, attempt to provide closed timestamp notifications for timestamps trailing cluster time by approximately this durationServerless/Dedicated/Self-Hosted (read-only) +
kv.consistency_queue.testing_fast_efos_acquisition.enabled
booleanfalseset to true to speed up EventuallyFileOnlySnapshot acquisition/transition for tests at the expense of excessive flushesDedicated/Self-Hosted
kv.log_range_and_node_events.enabled
booleantrueset to true to transactionally log range events (e.g., split, merge, add/remove voter/non-voter) into system.rangelogand node join and restart events into system.eventologDedicated/Self-Hosted
kv.protectedts.reconciliation.interval
duration5m0sthe frequency for reconciling jobs with protected timestamp recordsServerless/Dedicated/Self-Hosted (read-only)
kv.range_split.by_load.enabled
booleantrueallow automatic splits of ranges based on where load is concentratedDedicated/Self-Hosted diff --git a/pkg/cmd/roachtest/roachtestutil/validation_check.go b/pkg/cmd/roachtest/roachtestutil/validation_check.go index f245e936648b..9c13028708e6 100644 --- a/pkg/cmd/roachtest/roachtestutil/validation_check.go +++ b/pkg/cmd/roachtest/roachtestutil/validation_check.go @@ -49,6 +49,7 @@ func CheckReplicaDivergenceOnDB(ctx context.Context, l *logger.Logger, db *gosql // is happening or how to disable it. started := timeutil.Now() rows, err := db.QueryContext(ctx, ` +SET CLUSTER SETTING kv.consistency_queue.testing_fast_efos_acquisition.enabled = true; SET statement_timeout = '20m'; SELECT t.range_id, t.start_key_pretty, t.status, t.detail FROM crdb_internal.check_consistency(false, '', '') as t;`) diff --git a/pkg/kv/kvserver/consistency_queue_test.go b/pkg/kv/kvserver/consistency_queue_test.go index e2a62fabda5a..198e5a1ceab8 100644 --- a/pkg/kv/kvserver/consistency_queue_test.go +++ b/pkg/kv/kvserver/consistency_queue_test.go @@ -405,7 +405,7 @@ func TestCheckConsistencyInconsistent(t *testing.T) { // Compute a checksum over the content of the problematic range. rd, err := kvserver.CalcReplicaDigest(context.Background(), *desc, cpEng, - kvpb.ChecksumMode_CHECK_FULL, quotapool.NewRateLimiter("test", quotapool.Inf(), 0)) + kvpb.ChecksumMode_CHECK_FULL, quotapool.NewRateLimiter("test", quotapool.Inf(), 0), nil /* settings */) require.NoError(t, err) hashes[i] = rd.SHA512[:] } diff --git a/pkg/kv/kvserver/replica_consistency.go b/pkg/kv/kvserver/replica_consistency.go index 35bbccd93c37..b23ed9136819 100644 --- a/pkg/kv/kvserver/replica_consistency.go +++ b/pkg/kv/kvserver/replica_consistency.go @@ -30,6 +30,8 @@ import ( "github.com/cockroachdb/cockroach/pkg/kv/kvserver/stateloader" "github.com/cockroachdb/cockroach/pkg/roachpb" "github.com/cockroachdb/cockroach/pkg/rpc" + "github.com/cockroachdb/cockroach/pkg/settings" + "github.com/cockroachdb/cockroach/pkg/settings/cluster" "github.com/cockroachdb/cockroach/pkg/storage" "github.com/cockroachdb/cockroach/pkg/storage/enginepb" "github.com/cockroachdb/cockroach/pkg/storage/fs" @@ -56,6 +58,15 @@ type replicaChecksum struct { result chan CollectChecksumResponse } +// TestingFastEFOSAcquisition speeds up EFOS WaitForFileOnly() to speed up +// node-wide replica consistency check calls in roachtests. +var TestingFastEFOSAcquisition = settings.RegisterBoolSetting( + settings.SystemOnly, + "kv.consistency_queue.testing_fast_efos_acquisition.enabled", + "set to true to speed up EventuallyFileOnlySnapshot acquisition/transition for tests at the expense of excessive flushes", + false, /* defaultValue */ + settings.WithPublic) + // CheckConsistency runs a consistency check on the range. It first applies a // ComputeChecksum through Raft and then issues CollectChecksum commands to the // other replicas. These are inspected and a CheckConsistencyResponse is assembled. @@ -479,6 +490,7 @@ func CalcReplicaDigest( snap storage.Reader, mode kvpb.ChecksumMode, limiter *quotapool.RateLimiter, + settings *cluster.Settings, ) (*ReplicaDigest, error) { statsOnly := mode == kvpb.ChecksumMode_CHECK_STATS @@ -500,7 +512,13 @@ func CalcReplicaDigest( // both requests are likely sharing the same `limiter` so if too many // requests run concurrently, some of them could time out due to a // combination of this wait and the limiter-induced wait. - if err := efos.WaitForFileOnly(ctx); err != nil { + efosWait := storage.MaxEFOSWait + if settings != nil && TestingFastEFOSAcquisition.Get(&settings.SV) { + if efosWait > 10*time.Millisecond { + efosWait = 10 * time.Millisecond + } + } + if err := efos.WaitForFileOnly(ctx, efosWait); err != nil { return nil, err } } @@ -773,7 +791,7 @@ func (r *Replica) computeChecksumPostApply( ); err != nil { log.Errorf(ctx, "checksum collection did not join: %v", err) } else { - result, err := CalcReplicaDigest(ctx, desc, snap, cc.Mode, r.store.consistencyLimiter) + result, err := CalcReplicaDigest(ctx, desc, snap, cc.Mode, r.store.consistencyLimiter, r.ClusterSettings()) if err != nil { log.Errorf(ctx, "checksum computation failed: %v", err) result = nil diff --git a/pkg/kv/kvserver/replica_consistency_test.go b/pkg/kv/kvserver/replica_consistency_test.go index 73581dbc3aff..45154a2536a7 100644 --- a/pkg/kv/kvserver/replica_consistency_test.go +++ b/pkg/kv/kvserver/replica_consistency_test.go @@ -275,7 +275,7 @@ func TestReplicaChecksumSHA512(t *testing.T) { // Hash the empty state. unlim := quotapool.NewRateLimiter("test", quotapool.Inf(), 0) - rd, err := CalcReplicaDigest(ctx, desc, eng, kvpb.ChecksumMode_CHECK_FULL, unlim) + rd, err := CalcReplicaDigest(ctx, desc, eng, kvpb.ChecksumMode_CHECK_FULL, unlim, nil /* settings */) require.NoError(t, err) fmt.Fprintf(sb, "checksum0: %x\n", rd.SHA512) @@ -314,7 +314,7 @@ func TestReplicaChecksumSHA512(t *testing.T) { require.NoError(t, err) } - rd, err = CalcReplicaDigest(ctx, desc, eng, kvpb.ChecksumMode_CHECK_FULL, unlim) + rd, err = CalcReplicaDigest(ctx, desc, eng, kvpb.ChecksumMode_CHECK_FULL, unlim, nil /* settings */) require.NoError(t, err) fmt.Fprintf(sb, "checksum%d: %x\n", i+1, rd.SHA512) } @@ -335,13 +335,13 @@ func TestReplicaChecksumSHA512(t *testing.T) { txn := &roachpb.Transaction{TxnMeta: enginepb.TxnMeta{ID: txnID}} require.NoError(t, storage.MVCCAcquireLock(ctx, eng, txn, l.str, roachpb.Key(l.key), nil, 0)) - rd, err = CalcReplicaDigest(ctx, desc, eng, kvpb.ChecksumMode_CHECK_FULL, unlim) + rd, err = CalcReplicaDigest(ctx, desc, eng, kvpb.ChecksumMode_CHECK_FULL, unlim, nil /* settings */) require.NoError(t, err) fmt.Fprintf(sb, "checksum%d: %x\n", i+1, rd.SHA512) } // Run another check to obtain stats for the final state. - rd, err = CalcReplicaDigest(ctx, desc, eng, kvpb.ChecksumMode_CHECK_FULL, unlim) + rd, err = CalcReplicaDigest(ctx, desc, eng, kvpb.ChecksumMode_CHECK_FULL, unlim, nil /* settings */) require.NoError(t, err) jsonpb := protoutil.JSONPb{Indent: " "} json, err := jsonpb.Marshal(&rd.RecomputedMS) diff --git a/pkg/kv/kvserver/replica_test.go b/pkg/kv/kvserver/replica_test.go index 339ee867bec7..5d1dc2139bf8 100644 --- a/pkg/kv/kvserver/replica_test.go +++ b/pkg/kv/kvserver/replica_test.go @@ -10425,7 +10425,7 @@ func TestReplicaServersideRefreshes(t *testing.T) { snap := tc.engine.NewSnapshot() defer snap.Close() res, err := CalcReplicaDigest(ctx, *tc.repl.Desc(), tc.engine, kvpb.ChecksumMode_CHECK_FULL, - quotapool.NewRateLimiter("test", quotapool.Inf(), 0)) + quotapool.NewRateLimiter("test", quotapool.Inf(), 0), nil /* settings */) if err != nil { return hlc.Timestamp{}, err } diff --git a/pkg/kv/kvserver/spanset/batch.go b/pkg/kv/kvserver/spanset/batch.go index 0164de8a4ae8..a43371f7b116 100644 --- a/pkg/kv/kvserver/spanset/batch.go +++ b/pkg/kv/kvserver/spanset/batch.go @@ -14,6 +14,7 @@ import ( "bytes" "context" "fmt" + "time" "github.com/cockroachdb/cockroach/pkg/keys" "github.com/cockroachdb/cockroach/pkg/roachpb" @@ -938,6 +939,8 @@ func NewEventuallyFileOnlySnapshot( } // WaitForFileOnly implements the storage.EventuallyFileOnlyReader interface. -func (e *spanSetEFOS) WaitForFileOnly(ctx context.Context) error { - return e.efos.WaitForFileOnly(ctx) +func (e *spanSetEFOS) WaitForFileOnly( + ctx context.Context, gracePeriodBeforeFlush time.Duration, +) error { + return e.efos.WaitForFileOnly(ctx, gracePeriodBeforeFlush) } diff --git a/pkg/storage/engine.go b/pkg/storage/engine.go index c3fd2ba1ead2..cbb74b37fab3 100644 --- a/pkg/storage/engine.go +++ b/pkg/storage/engine.go @@ -632,8 +632,10 @@ type EventuallyFileOnlyReader interface { Reader // WaitForFileOnly blocks the calling goroutine until this reader has // transitioned to a file-only reader that does not pin any in-memory state. - // If an error is returned, this transition did not succeed. - WaitForFileOnly(context.Context) error + // If an error is returned, this transition did not succeed. The Duration + // argument specifies how long to wait for before attempting a flush to + // force a transition to a file-only snapshot. + WaitForFileOnly(ctx context.Context, gracePeriodBeforeFlush time.Duration) error } // Writer is the write interface to an engine's data. diff --git a/pkg/storage/pebble.go b/pkg/storage/pebble.go index 92c54fa4c77e..346f9a0beb49 100644 --- a/pkg/storage/pebble.go +++ b/pkg/storage/pebble.go @@ -3130,8 +3130,10 @@ func (p *pebbleEFOS) MVCCIterate( } // WaitForFileOnly implements the EventuallyFileOnlyReader interface. -func (p *pebbleEFOS) WaitForFileOnly(ctx context.Context) error { - return p.efos.WaitForFileOnlySnapshot(ctx, MaxEFOSWait) +func (p *pebbleEFOS) WaitForFileOnly( + ctx context.Context, gracePeriodBeforeFlush time.Duration, +) error { + return p.efos.WaitForFileOnlySnapshot(ctx, gracePeriodBeforeFlush) } // NewMVCCIterator implements the Reader interface.