diff --git a/pkg/gen/stringer.bzl b/pkg/gen/stringer.bzl index c0e3accd8f1b..dec031ce3f08 100644 --- a/pkg/gen/stringer.bzl +++ b/pkg/gen/stringer.bzl @@ -44,7 +44,6 @@ STRINGER_SRCS = [ "//pkg/sql:nodestatus_string.go", "//pkg/sql:txneventtype_string.go", "//pkg/sql:txntype_string.go", - "//pkg/storage:resourcelimitreached_string.go", "//pkg/util/encoding:type_string.go", "//pkg/util/timeutil/pgdate:field_string.go", "//pkg/workload/schemachange:optype_string.go", diff --git a/pkg/kv/kvserver/batcheval/BUILD.bazel b/pkg/kv/kvserver/batcheval/BUILD.bazel index 1c5328479297..9ad5de919cb2 100644 --- a/pkg/kv/kvserver/batcheval/BUILD.bazel +++ b/pkg/kv/kvserver/batcheval/BUILD.bazel @@ -83,7 +83,6 @@ go_library( "//pkg/util/log", "//pkg/util/mon", "//pkg/util/protoutil", - "//pkg/util/timeutil", "//pkg/util/tracing", "//pkg/util/uuid", "@com_github_cockroachdb_errors//:errors", diff --git a/pkg/kv/kvserver/batcheval/cmd_export.go b/pkg/kv/kvserver/batcheval/cmd_export.go index 546e8dcfe6ca..624d303fb54e 100644 --- a/pkg/kv/kvserver/batcheval/cmd_export.go +++ b/pkg/kv/kvserver/batcheval/cmd_export.go @@ -23,7 +23,6 @@ import ( "github.com/cockroachdb/cockroach/pkg/storage" "github.com/cockroachdb/cockroach/pkg/util/hlc" "github.com/cockroachdb/cockroach/pkg/util/log" - "github.com/cockroachdb/cockroach/pkg/util/timeutil" "github.com/cockroachdb/cockroach/pkg/util/tracing" "github.com/cockroachdb/errors" ) @@ -62,21 +61,6 @@ var ExportRequestMaxAllowedFileSizeOverage = settings.RegisterByteSizeSetting( 64<<20, /* 64 MiB */ ).WithPublic() -// exportRequestMaxIterationTime controls time spent by export request iterating -// over data in underlying storage. This threshold preventing export request from -// holding locks for too long and preventing non mvcc operations from progressing. -// If request takes longer than this threshold it would stop and return already -// collected data and allow caller to use resume span to continue. -var exportRequestMaxIterationTime = settings.RegisterDurationSetting( - settings.TenantWritable, - "kv.bulk_sst.max_request_time", - "if set, limits amount of time spent in export requests; "+ - "if export request can not finish within allocated time it will resume from the point it stopped in "+ - "subsequent request", - // Feature is disabled by default. - 0, -) - func init() { RegisterReadOnlyCommand(roachpb.Export, declareKeysExport, evalExport) } @@ -165,8 +149,6 @@ func evalExport( maxSize = targetSize + uint64(allowedOverage) } - maxRunTime := exportRequestMaxIterationTime.Get(&cArgs.EvalCtx.ClusterSettings().SV) - var maxIntents uint64 if m := storage.MaxIntentsPerWriteIntentError.Get(&cArgs.EvalCtx.ClusterSettings().SV); m > 0 { maxIntents = uint64(m) @@ -191,7 +173,6 @@ func evalExport( MaxSize: maxSize, MaxIntents: maxIntents, StopMidKey: args.SplitMidKey, - ResourceLimiter: storage.NewResourceLimiter(storage.ResourceLimiterOptions{MaxRunTime: maxRunTime}, timeutil.DefaultTimeSource{}), } var summary roachpb.BulkOpSummary var resume storage.MVCCKey diff --git a/pkg/storage/BUILD.bazel b/pkg/storage/BUILD.bazel index d4ff71159f06..7b3116416b16 100644 --- a/pkg/storage/BUILD.bazel +++ b/pkg/storage/BUILD.bazel @@ -33,7 +33,6 @@ go_library( "point_synthesizing_iter.go", "read_as_of_iterator.go", "replicas_storage.go", - "resource_limiter.go", "row_counter.go", "slice.go", "slice_go1.9.go", @@ -42,7 +41,6 @@ go_library( "store_properties.go", "temp_engine.go", "verifying_iterator.go", - ":gen-resourcelimitreached-stringer", # keep ], importpath = "github.com/cockroachdb/cockroach/pkg/storage", visibility = ["//visibility:public"], @@ -128,7 +126,6 @@ go_test( "pebble_mvcc_scanner_test.go", "pebble_test.go", "read_as_of_iterator_test.go", - "resource_limiter_test.go", "sst_test.go", "sst_writer_test.go", "temp_engine_test.go", @@ -187,10 +184,4 @@ go_test( ], ) -stringer( - name = "gen-resourcelimitreached-stringer", - src = "resource_limiter.go", - typ = "ResourceLimitReached", -) - get_x_data(name = "get_x_data") diff --git a/pkg/storage/mvcc.go b/pkg/storage/mvcc.go index 10d8d99a9b04..06b9a4b9446a 100644 --- a/pkg/storage/mvcc.go +++ b/pkg/storage/mvcc.go @@ -5996,8 +5996,6 @@ func mvccExportToWriter( paginated := opts.TargetSize > 0 hasElasticCPULimiter := elasticCPUHandle != nil - hasTimeBasedResourceLimiter := opts.ResourceLimiter != nil - // trackKeyBoundary is true if we need to know whether the // iteration has proceeded to a new key. // @@ -6008,8 +6006,7 @@ func mvccExportToWriter( // key boundaries if we may return from our iteration before // the EndKey. This can happen if the user has requested // paginated results, or if we hit a resource limit. - trackKeyBoundary := opts.ExportAllRevisions && (paginated || hasTimeBasedResourceLimiter || hasElasticCPULimiter) - + trackKeyBoundary := opts.ExportAllRevisions && (paginated || hasElasticCPULimiter) firstIteration := true // skipTombstones controls whether we include tombstones. // @@ -6084,30 +6081,6 @@ func mvccExportToWriter( // of starvation. Otherwise operations could spin indefinitely. firstIteration = false } else { - // TODO(irfansharif): Remove this time-based resource limiter once - // enabling elastic CPU limiting by default. There needs to be a - // compelling reason to need two mechanisms. - if opts.ResourceLimiter != nil { - // In happy day case we want to only stop at key boundaries as it allows callers to use - // produced sst's directly. But if we can't find key boundary within reasonable number of - // iterations we would split mid key. - // To achieve that we use soft and hard thresholds in limiter. Once soft limit is reached - // we would start searching for key boundary and return as soon as it is reached. If we - // can't find it before hard limit is reached and caller requested mid key stop we would - // immediately return. - limit := opts.ResourceLimiter.IsExhausted() - // We can stop at key once any threshold is reached or force stop at hard limit if midkey - // split is allowed. - if limit >= ResourceLimitReachedSoft && isNewKey || limit == ResourceLimitReachedHard && opts.StopMidKey { - // Reached iteration limit, stop with resume span - resumeKey = unsafeKey.Clone() - if isNewKey { - resumeKey.Timestamp = hlc.Timestamp{} - } - break - } - } - // Check if we're over our allotted CPU time + on a key boundary (we // prefer callers being able to use SSTs directly). Going over limit is // accounted for in admission control by penalizing the subsequent @@ -6368,10 +6341,6 @@ type MVCCExportOptions struct { // cause problems with multiplexed iteration using NewSSTIterator(), nor when // ingesting the SSTs via `AddSSTable`. StopMidKey bool - // ResourceLimiter limits how long iterator could run until it exhausts allocated - // resources. Export queries limiter in its iteration loop to break out once - // resources are exhausted. - ResourceLimiter ResourceLimiter // FingerprintOptions controls how fingerprints are generated // when using MVCCExportFingerprint. FingerprintOptions MVCCExportFingerprintOptions diff --git a/pkg/storage/mvcc_test.go b/pkg/storage/mvcc_test.go index 570c1cc52d13..3a2f78926a25 100644 --- a/pkg/storage/mvcc_test.go +++ b/pkg/storage/mvcc_test.go @@ -5894,81 +5894,6 @@ func TestWillOverflow(t *testing.T) { } } -func TestMVCCExportToSSTResourceLimits(t *testing.T) { - defer leaktest.AfterTest(t)() - defer log.Scope(t).Close(t) - - engine := createTestPebbleEngine() - defer engine.Close() - - limits := dataLimits{ - minKey: 0, - maxKey: 1000, - minTimestamp: hlc.Timestamp{WallTime: 100000}, - maxTimestamp: hlc.Timestamp{WallTime: 200000}, - tombstoneChance: 0.01, - } - generateData(t, engine, limits, (limits.maxKey-limits.minKey)*10) - - // Outer loop runs tests on subsets of mvcc dataset. - for _, query := range []queryLimits{ - { - minKey: 0, - maxKey: 1000, - minTimestamp: hlc.Timestamp{WallTime: 100000}, - maxTimestamp: hlc.Timestamp{WallTime: 200000}, - latest: false, - }, - { - minKey: 200, - maxKey: 800, - minTimestamp: hlc.Timestamp{WallTime: 100000}, - maxTimestamp: hlc.Timestamp{WallTime: 200000}, - latest: false, - }, - { - minKey: 0, - maxKey: 1000, - minTimestamp: hlc.Timestamp{WallTime: 150000}, - maxTimestamp: hlc.Timestamp{WallTime: 175000}, - latest: false, - }, - { - minKey: 0, - maxKey: 1000, - minTimestamp: hlc.Timestamp{WallTime: 100000}, - maxTimestamp: hlc.Timestamp{WallTime: 200000}, - latest: true, - }, - } { - t.Run(fmt.Sprintf("minKey=%d,maxKey=%d,minTs=%v,maxTs=%v,latest=%t", query.minKey, query.maxKey, query.minTimestamp, query.maxTimestamp, query.latest), - func(t *testing.T) { - matchingData := exportAllData(t, engine, query) - // Inner loop exercises various thresholds to see that we always progress and respect soft - // and hard limits. - for _, resources := range []resourceLimits{ - // soft threshold under version count, high threshold above - {softThreshold: 5, hardThreshold: 20}, - // soft threshold above version count - {softThreshold: 15, hardThreshold: 30}, - // low threshold to check we could always progress - {softThreshold: 0, hardThreshold: 0}, - // equal thresholds to check we force breaks mid keys - {softThreshold: 15, hardThreshold: 15}, - // very high hard thresholds to eliminate mid key breaking completely - {softThreshold: 5, hardThreshold: math.MaxInt64}, - // very high thresholds to eliminate breaking completely - {softThreshold: math.MaxInt64, hardThreshold: math.MaxInt64}, - } { - t.Run(fmt.Sprintf("softThreshold=%d,hardThreshold=%d", resources.softThreshold, resources.hardThreshold), - func(t *testing.T) { - assertDataEqual(t, engine, matchingData, query, resources) - }) - } - }) - } -} - // TestMVCCExportToSSTExhaustedAtStart is a regression test for a bug // in which mis-handling of resume spans would cause MVCCExportToSST // to return an empty resume key in cases where the resource limiters @@ -6113,7 +6038,6 @@ func TestMVCCExportToSSTExhaustedAtStart(t *testing.T) { StartTS: minTimestamp, EndTS: maxTimestamp, ExportAllRevisions: true, - ResourceLimiter: nil, StopMidKey: false, } @@ -6150,90 +6074,8 @@ func TestMVCCExportToSSTExhaustedAtStart(t *testing.T) { require.Equal(t, 3, len(chunk)) }) - t.Run("resource limit exhausted", - func(t *testing.T) { - engine := createTestPebbleEngine() - defer engine.Close() - - // Construct a data set that contains 4 - // tombstones followed by 1 non-tombstone. - // - // We expect that MVCCExportToSST with - // ExportAllRevisions set to false and with no - // start timestamp will elide the tombstones. - rng := rand.New(rand.NewSource(timeutil.Now().Unix())) - timestamp := minTimestamp.Add(rand.Int63n(maxTimestamp.WallTime-minTimestamp.WallTime), 0) - require.NoError(t, engine.PutMVCC(MVCCKey{Key: testKey(6), Timestamp: timestamp}, MVCCValue{}), "write data to test storage") - require.NoError(t, engine.PutMVCC(MVCCKey{Key: testKey(7), Timestamp: timestamp}, MVCCValue{}), "write data to test storage") - require.NoError(t, engine.PutMVCC(MVCCKey{Key: testKey(8), Timestamp: timestamp}, MVCCValue{}), "write data to test storage") - value := MVCCValue{Value: roachpb.MakeValueFromBytes(randutil.RandBytes(rng, 256))} - require.NoError(t, engine.PutMVCC(MVCCKey{Key: testKey(9), Timestamp: timestamp}, value), "write data to test storage") - require.NoError(t, engine.Flush(), "Flush engine data") - - data := exportAllData(t, engine, queryLimits{ - minKey: minKey, - maxKey: maxKey, - // Tombstones are only elided when - // StartTS isn't set on the export - // request. - minTimestamp: hlc.Timestamp{}, - maxTimestamp: maxTimestamp, - latest: true, - }) - - firstCall := true - assertExportEqualWithOptions(t, context.Background(), engine, data, MVCCExportOptions{ - StartKey: MVCCKey{Key: testKey(minKey), Timestamp: minTimestamp}, - EndKey: testKey(maxKey), - // No StartTS to ensure that - // tombstones are elided. - EndTS: maxTimestamp, - ExportAllRevisions: false, - // The ResourceLimiter will - // return ResourceLimitReached - // on the very first call. - ResourceLimiter: &callbackResourceLimiter{ - func() ResourceLimitReached { - if firstCall { - firstCall = false - return ResourceLimitReachedHard - } - return ResourceLimitNotReached - }, - }, - }) - }) -} - -type callbackResourceLimiter struct { - cb func() ResourceLimitReached -} - -func (c *callbackResourceLimiter) IsExhausted() ResourceLimitReached { - return c.cb() -} - -var _ ResourceLimiter = &callbackResourceLimiter{} - -type countingResourceLimiter struct { - softCount int64 - hardCount int64 - count int64 -} - -func (l *countingResourceLimiter) IsExhausted() ResourceLimitReached { - l.count++ - if l.count > l.hardCount { - return ResourceLimitReachedHard - } - if l.count > l.softCount { - return ResourceLimitReachedSoft - } - return ResourceLimitNotReached } -var _ ResourceLimiter = &countingResourceLimiter{} - type queryLimits struct { minKey int64 maxKey int64 @@ -6254,11 +6096,6 @@ type dataLimits struct { tombstoneChance float64 } -type resourceLimits struct { - softThreshold int64 - hardThreshold int64 -} - func exportAllData(t *testing.T, engine Engine, limits queryLimits) []MVCCKey { st := cluster.MakeTestingClusterSettings() sstFile := &MemFile{} @@ -6296,50 +6133,6 @@ func sstToKeys(t *testing.T, data []byte) []MVCCKey { return results } -func assertDataEqual( - t *testing.T, engine Engine, data []MVCCKey, query queryLimits, resources resourceLimits, -) { - var ( - err error - key = MVCCKey{Key: testKey(query.minKey), Timestamp: query.minTimestamp} - dataIndex = 0 - ) - for { - // Export chunk - limiter := countingResourceLimiter{softCount: resources.softThreshold, hardCount: resources.hardThreshold} - sstFile := &MemFile{} - st := cluster.MakeTestingClusterSettings() - _, key, err = MVCCExportToSST(context.Background(), st, engine, MVCCExportOptions{ - StartKey: key, - EndKey: testKey(query.maxKey), - StartTS: query.minTimestamp, - EndTS: query.maxTimestamp, - ExportAllRevisions: !query.latest, - StopMidKey: true, - ResourceLimiter: &limiter, - }, sstFile) - require.NoError(t, err, "Failed to export to Sst") - - chunk := sstToKeys(t, sstFile.Data()) - require.LessOrEqual(t, len(chunk), len(data)-dataIndex, "Remaining test data") - for _, key := range chunk { - require.True(t, key.Equal(data[dataIndex]), "Returned key is not equal") - dataIndex++ - } - require.LessOrEqual(t, limiter.count-1, resources.hardThreshold, "Fragment size") - - // Last chunk check. - if len(key.Key) == 0 { - break - } - require.GreaterOrEqual(t, limiter.count-1, resources.softThreshold, "Fragment size") - if resources.hardThreshold == math.MaxInt64 { - require.True(t, key.Timestamp.IsEmpty(), "Should never break mid key on high hard thresholds") - } - } - require.Equal(t, dataIndex, len(data), "Not all expected data was consumed") -} - func generateData(t *testing.T, engine Engine, limits dataLimits, totalEntries int64) { rng := rand.New(rand.NewSource(timeutil.Now().Unix())) for i := int64(0); i < totalEntries; i++ { diff --git a/pkg/storage/resource_limiter.go b/pkg/storage/resource_limiter.go deleted file mode 100644 index 6c8ccca29c5c..000000000000 --- a/pkg/storage/resource_limiter.go +++ /dev/null @@ -1,76 +0,0 @@ -// Copyright 2021 The Cockroach Authors. -// -// Use of this software is governed by the Business Source License -// included in the file licenses/BSL.txt. -// -// As of the Change Date specified in that file, in accordance with -// the Business Source License, use of this software will be governed -// by the Apache License, Version 2.0, included in the file -// licenses/APL.txt. - -package storage - -import ( - "time" - - "github.com/cockroachdb/cockroach/pkg/util/timeutil" -) - -// ResourceLimiterOptions is defining limits for resource limiter to restrict number -// of iterations. -type ResourceLimiterOptions struct { - MaxRunTime time.Duration -} - -// ResourceLimitReached indicates which resource threshold is exceeded. -// Soft threshold is advisory as to stop building results at convenient -// point, hard threshold is when iteration should stop straight away. -type ResourceLimitReached int - -//go:generate stringer -type ResourceLimitReached -const ( - ResourceLimitNotReached ResourceLimitReached = 0 - ResourceLimitReachedSoft ResourceLimitReached = 1 - ResourceLimitReachedHard ResourceLimitReached = 2 -) - -// ResourceLimiter provides a facility to stop cooperative long-running operation. -type ResourceLimiter interface { - // IsExhausted returns true when limited resource is exhausted. Iterator is - // checking the exhaustion status of resource limiter every time it advances - // to the next underlying key value pair. - IsExhausted() ResourceLimitReached -} - -// TimeResourceLimiter provides limiter based on wall clock time. -type TimeResourceLimiter struct { - softMaxRunTime time.Duration - hardMaxRunTime time.Duration - startTime time.Time - ts timeutil.TimeSource -} - -var _ ResourceLimiter = &TimeResourceLimiter{} - -// NewResourceLimiter create new default resource limiter. Current implementation is wall clock time based. -// Timer starts as soon as limiter is created. -// If no limits are specified in opts nil is returned. -func NewResourceLimiter(opts ResourceLimiterOptions, ts timeutil.TimeSource) ResourceLimiter { - if opts.MaxRunTime == 0 { - return nil - } - softTimeLimit := time.Duration(float64(opts.MaxRunTime) * 0.9) - return &TimeResourceLimiter{hardMaxRunTime: opts.MaxRunTime, softMaxRunTime: softTimeLimit, startTime: ts.Now(), ts: ts} -} - -// IsExhausted implements ResourceLimiter interface. -func (l *TimeResourceLimiter) IsExhausted() ResourceLimitReached { - timePassed := l.ts.Since(l.startTime) - if timePassed < l.softMaxRunTime { - return ResourceLimitNotReached - } - if timePassed < l.hardMaxRunTime { - return ResourceLimitReachedSoft - } - return ResourceLimitReachedHard -} diff --git a/pkg/storage/resource_limiter_test.go b/pkg/storage/resource_limiter_test.go deleted file mode 100644 index db7dd318b66d..000000000000 --- a/pkg/storage/resource_limiter_test.go +++ /dev/null @@ -1,33 +0,0 @@ -// Copyright 2021 The Cockroach Authors. -// -// Use of this software is governed by the Business Source License -// included in the file licenses/BSL.txt. -// -// As of the Change Date specified in that file, in accordance with -// the Business Source License, use of this software will be governed -// by the Apache License, Version 2.0, included in the file -// licenses/APL.txt. - -package storage - -import ( - "testing" - "time" - - "github.com/cockroachdb/cockroach/pkg/util/leaktest" - "github.com/cockroachdb/cockroach/pkg/util/timeutil" - "github.com/stretchr/testify/require" -) - -func TestResourceLimiter(t *testing.T) { - defer leaktest.AfterTest(t)() - - clock := timeutil.NewManualTime(timeutil.Now()) - l := NewResourceLimiter(ResourceLimiterOptions{MaxRunTime: time.Minute}, clock) - - require.Equal(t, ResourceLimitNotReached, l.IsExhausted(), "Exhausted when time didn't move") - clock.Advance(time.Second * 59) - require.Equal(t, ResourceLimitReachedSoft, l.IsExhausted(), "Soft limit not reached") - clock.Advance(time.Minute) - require.Equal(t, ResourceLimitReachedHard, l.IsExhausted(), "Hard limit not reached") -} diff --git a/pkg/storage/resourcelimitreached_string.go b/pkg/storage/resourcelimitreached_string.go deleted file mode 100644 index ade245076ff9..000000000000 --- a/pkg/storage/resourcelimitreached_string.go +++ /dev/null @@ -1,25 +0,0 @@ -// Code generated by "stringer"; DO NOT EDIT. - -package storage - -import "strconv" - -func _() { - // An "invalid array index" compiler error signifies that the constant values have changed. - // Re-run the stringer command to generate them again. - var x [1]struct{} - _ = x[ResourceLimitNotReached-0] - _ = x[ResourceLimitReachedSoft-1] - _ = x[ResourceLimitReachedHard-2] -} - -const _ResourceLimitReached_name = "ResourceLimitNotReachedResourceLimitReachedSoftResourceLimitReachedHard" - -var _ResourceLimitReached_index = [...]uint8{0, 23, 47, 71} - -func (i ResourceLimitReached) String() string { - if i < 0 || i >= ResourceLimitReached(len(_ResourceLimitReached_index)-1) { - return "ResourceLimitReached(" + strconv.FormatInt(int64(i), 10) + ")" - } - return _ResourceLimitReached_name[_ResourceLimitReached_index[i]:_ResourceLimitReached_index[i+1]] -}