@@ -8,27 +8,36 @@ package kvserver_test
8
8
import (
9
9
"context"
10
10
"fmt"
11
+ math "math"
11
12
"net/url"
13
+ "strings"
12
14
"sync"
13
15
"sync/atomic"
14
16
"testing"
15
17
"time"
16
18
17
19
"github.com/cockroachdb/cockroach/pkg/base"
18
20
"github.com/cockroachdb/cockroach/pkg/keys"
21
+ "github.com/cockroachdb/cockroach/pkg/kv/kvclient/kvcoord"
19
22
"github.com/cockroachdb/cockroach/pkg/kv/kvpb"
20
23
"github.com/cockroachdb/cockroach/pkg/kv/kvserver"
21
24
"github.com/cockroachdb/cockroach/pkg/roachpb"
25
+ "github.com/cockroachdb/cockroach/pkg/spanconfig"
26
+ "github.com/cockroachdb/cockroach/pkg/storage/enginepb"
22
27
"github.com/cockroachdb/cockroach/pkg/testutils"
23
28
"github.com/cockroachdb/cockroach/pkg/testutils/pgurlutils"
29
+ "github.com/cockroachdb/cockroach/pkg/testutils/serverutils"
24
30
"github.com/cockroachdb/cockroach/pkg/testutils/skip"
25
31
"github.com/cockroachdb/cockroach/pkg/testutils/sqlutils"
26
32
"github.com/cockroachdb/cockroach/pkg/testutils/testcluster"
33
+ "github.com/cockroachdb/cockroach/pkg/util/hlc"
27
34
"github.com/cockroachdb/cockroach/pkg/util/leaktest"
28
35
"github.com/cockroachdb/cockroach/pkg/util/log"
36
+ "github.com/cockroachdb/cockroach/pkg/util/protoutil"
29
37
"github.com/cockroachdb/cockroach/pkg/util/randutil"
30
38
"github.com/cockroachdb/cockroach/pkg/util/syncutil"
31
39
"github.com/cockroachdb/errors"
40
+ "github.com/dustin/go-humanize"
32
41
"github.com/jackc/pgx/v5"
33
42
"github.com/stretchr/testify/require"
34
43
)
@@ -332,3 +341,233 @@ func TestBackpressureNotAppliedWhenReducingRangeSize(t *testing.T) {
332
341
require .Error (t , <- upsertErrCh )
333
342
})
334
343
}
344
+
345
+ // TestSpanConfigUpdatesDoNotGetBlockByRangeSizeBackpressureOnDefaultRanges
346
+ // verifies that spanconfig updates do not block by backpressure when the
347
+ // `system.span_configurations` table range becomes full, showing the allowlist
348
+ // is working.
349
+ //
350
+ // Test strategy:
351
+ // 1. Configure `system.span_configurations` table range to be a small size (8 KiB).
352
+ // 2. Write many large spanconfig records (2 KiB each) to fill up the range.
353
+ // 3. Verify spanconfig updates do not fail due to backpressure when the range is full,
354
+ // 4. This test recreates the scenario where spanconfig updates do not fail due to
355
+ // backpressure.
356
+ func TestSpanConfigUpdatesDoNotGetBlockByRangeSizeBackpressureOnDefaultRanges (t * testing.T ) {
357
+ defer leaktest .AfterTest (t )()
358
+ defer log .Scope (t ).Close (t )
359
+
360
+ ctx := context .Background ()
361
+
362
+ const (
363
+ overloadMaxRangeBytes = 8 << 10 // 8 KiB, a saner value than default 512 MiB for testing
364
+ overloadMinRangeBytes = 2 << 10 // 2 KiB
365
+ numWrites = 16 // enough to hit backpressure for 8 KiB range & 2 KiB spanconfig
366
+ defaultMaxBytes = 512 << 20 // default max bytes for a range
367
+ )
368
+ s := serverutils .StartServerOnly (t , base.TestServerArgs {})
369
+ defer s .Stopper ().Stop (ctx )
370
+
371
+ store , err := s .GetStores ().(* kvserver.Stores ).GetStore (s .GetFirstStoreID ())
372
+ require .NoError (t , err )
373
+
374
+ waitForSpanConfig := func (t * testing.T , tc serverutils.TestServerInterface ,
375
+ tablePrefix roachpb.Key , expRangeMaxBytes int64 ) {
376
+ testutils .SucceedsSoon (t , func () error {
377
+ _ , r := getFirstStoreReplica (t , tc , tablePrefix )
378
+ conf , err := r .LoadSpanConfig (ctx )
379
+ if err != nil {
380
+ return err
381
+ }
382
+ if conf .RangeMaxBytes != expRangeMaxBytes {
383
+ return fmt .Errorf ("expected RangeMaxBytes %d, got %d" ,
384
+ expRangeMaxBytes , conf .RangeMaxBytes )
385
+ }
386
+ return nil
387
+ })
388
+ }
389
+
390
+ spanConfigTablePrefix := keys .SystemSQLCodec .TablePrefix (
391
+ keys .SpanConfigurationsTableID )
392
+
393
+ t .Logf ("targeting span_configurations table at key: %s (table ID %d)\n " ,
394
+ spanConfigTablePrefix , keys .SpanConfigurationsTableID )
395
+
396
+ scratchKey , err := s .ScratchRange ()
397
+ require .NoError (t , err )
398
+
399
+ testutils .SucceedsSoon (t , func () error {
400
+ repl := store .LookupReplica (roachpb .RKey (scratchKey ))
401
+ if got := repl .GetMaxBytes (ctx ); got != defaultMaxBytes {
402
+ return errors .Errorf (
403
+ "range max bytes values did not start at %d; got %d" ,
404
+ defaultMaxBytes , got )
405
+ }
406
+ return nil
407
+ })
408
+
409
+ systemSpanConfigurationsTableSpan := roachpb.Span {
410
+ Key : spanConfigTablePrefix ,
411
+ EndKey : spanConfigTablePrefix .PrefixEnd (),
412
+ }
413
+
414
+ target := spanconfig .MakeTargetFromSpan (systemSpanConfigurationsTableSpan )
415
+
416
+ systemSpanConfig := roachpb.SpanConfig {
417
+ RangeMaxBytes : overloadMaxRangeBytes ,
418
+ RangeMinBytes : overloadMinRangeBytes ,
419
+ }
420
+
421
+ configBytessdfsdf , err := protoutil .Marshal (& systemSpanConfig )
422
+ require .NoError (t , err )
423
+ t .Logf ("marshalled systemSpanConfig size: %d bytes" , len (configBytessdfsdf ))
424
+
425
+ record , err := spanconfig .MakeRecord (target , systemSpanConfig )
426
+ require .NoError (t , err )
427
+
428
+ kvaccessor := s .SpanConfigKVAccessor ().(spanconfig.KVAccessor )
429
+
430
+ err = kvaccessor .UpdateSpanConfigRecords (
431
+ ctx , []spanconfig.Target {target },
432
+ []spanconfig.Record {record }, hlc .MinTimestamp , hlc .MaxTimestamp )
433
+ require .NoError (t , err )
434
+
435
+ waitForSpanConfig (t , s , spanConfigTablePrefix , overloadMaxRangeBytes )
436
+
437
+ // Check if the range is using our custom config.
438
+ repl := store .LookupReplica (keys .MustAddr (spanConfigTablePrefix ))
439
+ if repl != nil {
440
+ conf , err := repl .LoadSpanConfig (ctx )
441
+ require .NoError (t , err )
442
+ t .Logf ("current range config - RangeMaxBytes: %d bytes (%s), " +
443
+ "RangeMinBytes: %d bytes (%s)" ,
444
+ conf .RangeMaxBytes , humanize .Bytes (uint64 (conf .RangeMaxBytes )),
445
+ conf .RangeMinBytes , humanize .Bytes (uint64 (conf .RangeMinBytes )))
446
+
447
+ }
448
+
449
+ t .Logf ("targeting span_configurations table at key: %s (table ID %d)\n " ,
450
+ spanConfigTablePrefix , keys .SpanConfigurationsTableID )
451
+
452
+ // Create a single target for the scratch range (this will be stored in system.span_configurations)
453
+ scratchTarget := spanconfig .MakeTargetFromSpan (roachpb.Span {
454
+ Key : scratchKey ,
455
+ EndKey : scratchKey .PrefixEnd (),
456
+ })
457
+
458
+ // This is a large spanconfig for a scratch range with relevant fields set
459
+ // to maximum int64 and int32 values. This is done to have a spanconfig that
460
+ // is large enough to trigger backpressure without having to write a million
461
+ // records.
462
+ // We want this config to be relatively large - this is done via setting
463
+ // values to have max values and multiple fields as this config gets
464
+ // marshalled into a protobuf and protobuf uses variant encoding, which
465
+ // means larger values take more bytes to encode.
466
+ spanConfig2KiB := roachpb.SpanConfig { // 2078 bytes ~ 2 KiB.
467
+ RangeMaxBytes : math .MaxInt64 ,
468
+ RangeMinBytes : math .MaxInt64 ,
469
+ GCPolicy : roachpb.GCPolicy {
470
+ TTLSeconds : math .MaxInt32 ,
471
+ ProtectionPolicies : []roachpb.ProtectionPolicy {
472
+ {
473
+ ProtectedTimestamp : hlc .MaxTimestamp ,
474
+ },
475
+ {
476
+ ProtectedTimestamp : hlc .MaxTimestamp ,
477
+ },
478
+ },
479
+ },
480
+ NumReplicas : math .MaxInt32 ,
481
+ GlobalReads : true ,
482
+ NumVoters : math .MaxInt32 ,
483
+ VoterConstraints : []roachpb.ConstraintsConjunction {
484
+ {
485
+ Constraints : []roachpb.Constraint {
486
+ {Key : "max_key" , Value : strings .Repeat ("x" , 1024 )}, // very long constraint value
487
+ },
488
+ },
489
+ },
490
+ LeasePreferences : []roachpb.LeasePreference {
491
+ {
492
+ Constraints : []roachpb.Constraint {
493
+ {Key : "max_key" , Value : strings .Repeat ("y" , 1024 )}, // very long constraint value
494
+ },
495
+ },
496
+ },
497
+ }
498
+
499
+ configBytes , err := protoutil .Marshal (& spanConfig2KiB )
500
+ require .NoError (t , err )
501
+
502
+ require .GreaterOrEqual (t , len (configBytes ), 2048 ,
503
+ "spanConfig2KiB should be at least 2 KiB in size" )
504
+
505
+ // Create a record with the span configuration.
506
+ testRecord , err := spanconfig .MakeRecord (scratchTarget , spanConfig2KiB )
507
+ require .NoError (t , err )
508
+
509
+ // Write span configurations using KVAccessor.
510
+ // We expect this to fail due to backpressure.
511
+ var i int
512
+ for i = 0 ; i < numWrites ; i ++ {
513
+ // Use KVAccessor to update span configurations.
514
+ err = kvaccessor .UpdateSpanConfigRecords (ctx , nil ,
515
+ []spanconfig.Record {testRecord }, hlc .MinTimestamp , hlc .MaxTimestamp )
516
+ if err != nil {
517
+ break
518
+ }
519
+ }
520
+
521
+ // Assert that the operation does not fail due to backpressure.
522
+ require .NoError (t , err ,
523
+ "expected span config writes to not fail due to backpressure, but they did" )
524
+
525
+ systemSpanConfigurationsTableSpanMVCCStats := roachpb.Span {
526
+ Key : keys .SystemSQLCodec .TablePrefix (keys .SpanConfigurationsTableID ),
527
+ EndKey : keys .SystemSQLCodec .TablePrefix (keys .SpanConfigurationsTableID + 1 ),
528
+ }
529
+
530
+ distSender := s .DistSenderI ().(* kvcoord.DistSender )
531
+
532
+ // Track aggregate MVCC stats across all SpanConfigurationsTable ranges
533
+ var aggregateStats enginepb.MVCCStats
534
+ var rangeCount int
535
+
536
+ for key := systemSpanConfigurationsTableSpanMVCCStats .Key ; key .Compare (systemSpanConfigurationsTableSpanMVCCStats .EndKey ) < 0 ; {
537
+ desc , err := distSender .RangeDescriptorCache ().Lookup (ctx , keys .MustAddr (key ))
538
+ require .NoError (t , err )
539
+ d := desc .Desc
540
+
541
+ rangeRepl := store .LookupReplica (d .StartKey )
542
+ if rangeRepl != nil {
543
+ stats := rangeRepl .GetMVCCStats ()
544
+ aggregateStats .Add (stats )
545
+ rangeCount ++
546
+ }
547
+
548
+ // Move to next range.
549
+ key = d .EndKey .AsRawKey ()
550
+ if key .Equal (roachpb .KeyMax ) {
551
+ break
552
+ }
553
+ }
554
+
555
+ require .Greater (t , aggregateStats .Total (), int64 (overloadMaxRangeBytes ))
556
+
557
+ smallSpanConfig := roachpb.SpanConfig {
558
+ GCPolicy : roachpb.GCPolicy {
559
+ TTLSeconds : 0 ,
560
+ },
561
+ }
562
+
563
+ smallSpanconfigRecord , err := spanconfig .MakeRecord (scratchTarget , smallSpanConfig )
564
+ require .NoError (t , err )
565
+
566
+ smallSpanconfigRecordWriteErr := kvaccessor .UpdateSpanConfigRecords (ctx ,
567
+ []spanconfig.Target {scratchTarget }, []spanconfig.Record {smallSpanconfigRecord },
568
+ hlc .MinTimestamp , hlc .MaxTimestamp )
569
+
570
+ require .NoError (t , smallSpanconfigRecordWriteErr ,
571
+ "expected smallSpanconfigRecord write to not fail due to backpressure" )
572
+
573
+ }
0 commit comments