Skip to content

Commit

Permalink
Merge #73876
Browse files Browse the repository at this point in the history
73876: *: enable span configs by default r=irfansharif a=irfansharif

We were previously using an opt-in flag to enable span configs; this
commit flips it into an opt-out one. This has the following effects:
- fresh crdb nodes/clusters run with span configs enabled.
- our entire test suite runs using the span configs infrastructure, save
   for a select few written:
     - with the gossipped system config span in mind;
     - using legacy testing harnesses that prevent injection of
        spanconfig dependencies due to circular imports.

We also introduce a more minified form of the opt-out testing flag:
UseSystemConfigSpanForQueues. Using the minified form still enables the
SQL-side of span configs to run (Reconciler, SQLWatcher, SQLTranslator)
but defaults to using the system config span down in KV.

Release note: None

Co-authored-by: irfan sharif <[email protected]>
  • Loading branch information
craig[bot] and irfansharif committed Jan 18, 2022
2 parents 7675946 + 4f6fc15 commit 365b4da
Show file tree
Hide file tree
Showing 57 changed files with 754 additions and 96 deletions.
2 changes: 1 addition & 1 deletion docs/generated/settings/settings-for-tenants.txt
Original file line number Diff line number Diff line change
Expand Up @@ -170,4 +170,4 @@ trace.debug.enable boolean false if set, traces for recent requests can be seen
trace.jaeger.agent string the address of a Jaeger agent to receive traces using the Jaeger UDP Thrift protocol, as <host>:<port>. If no port is specified, 6381 will be used.
trace.opentelemetry.collector string address of an OpenTelemetry trace collector to receive traces using the otel gRPC protocol, as <host>:<port>. If no port is specified, 4317 will be used.
trace.zipkin.collector string the address of a Zipkin instance to receive traces, as <host>:<port>. If no port is specified, 9411 will be used.
version version 21.2-36 set the active cluster version in the format '<major>.<minor>'
version version 21.2-42 set the active cluster version in the format '<major>.<minor>'
2 changes: 1 addition & 1 deletion docs/generated/settings/settings.html
Original file line number Diff line number Diff line change
Expand Up @@ -177,6 +177,6 @@
<tr><td><code>trace.jaeger.agent</code></td><td>string</td><td><code></code></td><td>the address of a Jaeger agent to receive traces using the Jaeger UDP Thrift protocol, as <host>:<port>. If no port is specified, 6381 will be used.</td></tr>
<tr><td><code>trace.opentelemetry.collector</code></td><td>string</td><td><code></code></td><td>address of an OpenTelemetry trace collector to receive traces using the otel gRPC protocol, as <host>:<port>. If no port is specified, 4317 will be used.</td></tr>
<tr><td><code>trace.zipkin.collector</code></td><td>string</td><td><code></code></td><td>the address of a Zipkin instance to receive traces, as <host>:<port>. If no port is specified, 9411 will be used.</td></tr>
<tr><td><code>version</code></td><td>version</td><td><code>21.2-36</code></td><td>set the active cluster version in the format '<major>.<minor>'</td></tr>
<tr><td><code>version</code></td><td>version</td><td><code>21.2-42</code></td><td>set the active cluster version in the format '<major>.<minor>'</td></tr>
</tbody>
</table>
11 changes: 8 additions & 3 deletions pkg/base/test_server_args.go
Original file line number Diff line number Diff line change
Expand Up @@ -139,9 +139,14 @@ type TestServerArgs struct {
// If set, a TraceDir is initialized at the provided path.
TraceDir string

// If set, the span configs infrastructure will be enabled. This is
// equivalent to setting COCKROACH_EXPERIMENTAL_SPAN_CONFIGS.
EnableSpanConfigs bool
// DisableSpanConfigs disables the use of the span configs infrastructure
// (in favor of the gossiped system config span). It's equivalent to setting
// COCKROACH_DISABLE_SPAN_CONFIGS, and is only intended for tests written
// with the system config span in mind.
//
// TODO(irfansharif): Remove all uses of this when we rip out the system
// config span.
DisableSpanConfigs bool
}

// TestClusterArgs contains the parameters one can set when creating a test
Expand Down
1 change: 1 addition & 0 deletions pkg/ccl/backupccl/BUILD.bazel
Original file line number Diff line number Diff line change
Expand Up @@ -199,6 +199,7 @@ go_test(
"//pkg/security/securitytest",
"//pkg/server",
"//pkg/settings/cluster",
"//pkg/spanconfig",
"//pkg/sql",
"//pkg/sql/catalog",
"//pkg/sql/catalog/catalogkeys",
Expand Down
7 changes: 6 additions & 1 deletion pkg/ccl/backupccl/backup_test.go
Original file line number Diff line number Diff line change
Expand Up @@ -1933,7 +1933,12 @@ func TestBackupRestoreControlJob(t *testing.T) {
// force every call to update
defer jobs.TestingSetProgressThresholds()()

serverArgs := base.TestServerArgs{Knobs: base.TestingKnobs{JobsTestingKnobs: jobs.NewTestingKnobsWithShortIntervals()}}
serverArgs := base.TestServerArgs{
DisableSpanConfigs: true,
Knobs: base.TestingKnobs{
JobsTestingKnobs: jobs.NewTestingKnobsWithShortIntervals(),
},
}
// Disable external processing of mutations so that the final check of
// crdb_internal.tables is guaranteed to not be cleaned up. Although this
// was never observed by a stress test, it is here for safety.
Expand Down
16 changes: 14 additions & 2 deletions pkg/ccl/backupccl/full_cluster_backup_restore_test.go
Original file line number Diff line number Diff line change
Expand Up @@ -23,6 +23,7 @@ import (
"github.com/cockroachdb/cockroach/pkg/jobs/jobspb"
"github.com/cockroachdb/cockroach/pkg/keys"
"github.com/cockroachdb/cockroach/pkg/security"
"github.com/cockroachdb/cockroach/pkg/spanconfig"
"github.com/cockroachdb/cockroach/pkg/sql"
"github.com/cockroachdb/cockroach/pkg/sql/catalog/catalogkv"
"github.com/cockroachdb/cockroach/pkg/sql/catalog/systemschema"
Expand All @@ -45,9 +46,20 @@ func TestFullClusterBackup(t *testing.T) {
defer leaktest.AfterTest(t)()
defer log.Scope(t).Close(t)

params := base.TestClusterArgs{
ServerArgs: base.TestServerArgs{
Knobs: base.TestingKnobs{
SpanConfig: &spanconfig.TestingKnobs{
// We compare job progress before and after a restore. Disable
// the automatic jobs checkpointing which could possibly mutate
// the progress data during the backup/restore process.
JobDisablePersistingCheckpoints: true,
},
},
}}
const numAccounts = 10
tcBackup, sqlDB, tempDir, cleanupFn := BackupRestoreTestSetup(t, singleNode, numAccounts, InitManualReplication)
tcRestore, sqlDBRestore, cleanupEmptyCluster := backupRestoreTestSetupEmpty(t, singleNode, tempDir, InitManualReplication, base.TestClusterArgs{})
tcBackup, sqlDB, tempDir, cleanupFn := backupRestoreTestSetupWithParams(t, singleNode, numAccounts, InitManualReplication, params)
tcRestore, sqlDBRestore, cleanupEmptyCluster := backupRestoreTestSetupEmpty(t, singleNode, tempDir, InitManualReplication, params)
defer cleanupFn()
defer cleanupEmptyCluster()

Expand Down
8 changes: 7 additions & 1 deletion pkg/ccl/importccl/exportcsv_test.go
Original file line number Diff line number Diff line change
Expand Up @@ -50,7 +50,13 @@ func setupExportableBank(t *testing.T, nodes, rows int) (*sqlutils.SQLRunner, st
dir, cleanupDir := testutils.TempDir(t)

tc := testcluster.StartTestCluster(t, nodes,
base.TestClusterArgs{ServerArgs: base.TestServerArgs{ExternalIODir: dir, UseDatabase: "test"}},
base.TestClusterArgs{
ServerArgs: base.TestServerArgs{
ExternalIODir: dir,
UseDatabase: "test",
DisableSpanConfigs: true,
},
},
)
conn := tc.Conns[0]
db := sqlutils.MakeSQLRunner(conn)
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -36,7 +36,6 @@ func TestPreSeedSpanConfigsWrittenWhenActive(t *testing.T) {
ctx := context.Background()
tc := testcluster.StartTestCluster(t, 1, base.TestClusterArgs{
ServerArgs: base.TestServerArgs{
EnableSpanConfigs: true, // we use spanconfig.KVAccessor to check if its contents are as we'd expect
Knobs: base.TestingKnobs{
Server: &server.TestingKnobs{
DisableAutomaticVersionUpgrade: 1,
Expand Down Expand Up @@ -89,7 +88,6 @@ func TestSeedTenantSpanConfigs(t *testing.T) {
ctx := context.Background()
tc := testcluster.StartTestCluster(t, 1, base.TestClusterArgs{
ServerArgs: base.TestServerArgs{
EnableSpanConfigs: true, // we use spanconfig.KVAccessor to check if its contents are as we'd expect
Knobs: base.TestingKnobs{
Server: &server.TestingKnobs{
DisableAutomaticVersionUpgrade: 1,
Expand Down Expand Up @@ -157,7 +155,6 @@ func TestSeedTenantSpanConfigsWithExistingEntry(t *testing.T) {
ctx := context.Background()
tc := testcluster.StartTestCluster(t, 1, base.TestClusterArgs{
ServerArgs: base.TestServerArgs{
EnableSpanConfigs: true, // we use spanconfig.KVAccessor to check if its contents are as we'd expect
Knobs: base.TestingKnobs{
Server: &server.TestingKnobs{
DisableAutomaticVersionUpgrade: 1,
Expand Down
1 change: 1 addition & 0 deletions pkg/ccl/serverccl/statusccl/BUILD.bazel
Original file line number Diff line number Diff line change
Expand Up @@ -40,6 +40,7 @@ go_test(
"//pkg/security/securitytest",
"//pkg/server",
"//pkg/server/serverpb",
"//pkg/spanconfig",
"//pkg/sql/catalog/catconstants",
"//pkg/sql/catalog/descpb",
"//pkg/sql/sem/tree",
Expand Down
4 changes: 4 additions & 0 deletions pkg/ccl/serverccl/statusccl/tenant_status_test.go
Original file line number Diff line number Diff line change
Expand Up @@ -25,6 +25,7 @@ import (
"github.com/cockroachdb/cockroach/pkg/roachpb"
"github.com/cockroachdb/cockroach/pkg/security"
"github.com/cockroachdb/cockroach/pkg/server/serverpb"
"github.com/cockroachdb/cockroach/pkg/spanconfig"
"github.com/cockroachdb/cockroach/pkg/sql/catalog/catconstants"
"github.com/cockroachdb/cockroach/pkg/sql/catalog/descpb"
"github.com/cockroachdb/cockroach/pkg/sql/sem/tree"
Expand Down Expand Up @@ -91,6 +92,9 @@ func TestTenantCannotSeeNonTenantStats(t *testing.T) {
ctx := context.Background()

serverParams, _ := tests.CreateTestServerParams()
serverParams.Knobs.SpanConfig = &spanconfig.TestingKnobs{
ManagerDisableJobCreation: true, // TODO(irfansharif): #74919.
}
testCluster := serverutils.StartNewTestCluster(t, 3 /* numNodes */, base.TestClusterArgs{
ServerArgs: serverParams,
})
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -95,7 +95,6 @@ func TestDataDriven(t *testing.T) {
}
tc := testcluster.StartTestCluster(t, 1, base.TestClusterArgs{
ServerArgs: base.TestServerArgs{
EnableSpanConfigs: true,
Knobs: base.TestingKnobs{
JobsTestingKnobs: jobs.NewTestingKnobsWithShortIntervals(), // speeds up test
SpanConfig: scKnobs,
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -88,7 +88,6 @@ func TestDataDriven(t *testing.T) {
}
tc := testcluster.StartTestCluster(t, 1, base.TestClusterArgs{
ServerArgs: base.TestServerArgs{
EnableSpanConfigs: true,
Knobs: base.TestingKnobs{
JobsTestingKnobs: jobs.NewTestingKnobsWithShortIntervals(), // speeds up test
SpanConfig: scKnobs,
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -80,7 +80,6 @@ func TestDataDriven(t *testing.T) {
datadriven.Walk(t, testutils.TestDataPath(t), func(t *testing.T, path string) {
tc := testcluster.StartTestCluster(t, 1, base.TestClusterArgs{
ServerArgs: base.TestServerArgs{
EnableSpanConfigs: true,
Knobs: base.TestingKnobs{
SpanConfig: scKnobs,
},
Expand Down
2 changes: 1 addition & 1 deletion pkg/cli/testdata/doctor/test_examine_cluster
Original file line number Diff line number Diff line change
Expand Up @@ -3,5 +3,5 @@ debug doctor examine cluster
debug doctor examine cluster
Examining 43 descriptors and 42 namespace entries...
ParentID 50, ParentSchemaID 51: relation "foo" (55): expected matching namespace entry, found none
Examining 3 jobs...
Examining 4 jobs...
ERROR: validation failed
22 changes: 22 additions & 0 deletions pkg/clusterversion/cockroach_versions.go
Original file line number Diff line number Diff line change
Expand Up @@ -229,6 +229,16 @@ const (
// system.protected_ts_records table that describes what is protected by the
// record.
AlterSystemProtectedTimestampAddColumn
// EnsureSpanConfigReconciliation ensures that the host tenant has run its
// reconciliation process at least once.
EnsureSpanConfigReconciliation
// EnsureSpanConfigSubscription ensures that all KV nodes are subscribed to
// the global span configuration state, observing the entries installed as
// in EnsureSpanConfigReconciliation.
EnsureSpanConfigSubscription
// EnableSpanConfigStore enables the use of the span configs infrastructure
// in KV.
EnableSpanConfigStore

// *************************************************
// Step (1): Add new versions here.
Expand Down Expand Up @@ -337,6 +347,18 @@ var versionsSingleton = keyedVersions{
Key: AlterSystemProtectedTimestampAddColumn,
Version: roachpb.Version{Major: 21, Minor: 2, Internal: 36},
},
{
Key: EnsureSpanConfigReconciliation,
Version: roachpb.Version{Major: 21, Minor: 2, Internal: 38},
},
{
Key: EnsureSpanConfigSubscription,
Version: roachpb.Version{Major: 21, Minor: 2, Internal: 40},
},
{
Key: EnableSpanConfigStore,
Version: roachpb.Version{Major: 21, Minor: 2, Internal: 42},
},

// *************************************************
// Step (2): Add new versions here.
Expand Down
7 changes: 5 additions & 2 deletions pkg/clusterversion/key_string.go

Some generated files are not rendered by default. Learn more about how customized files appear on GitHub.

1 change: 1 addition & 0 deletions pkg/jobs/jobspb/jobs.proto
Original file line number Diff line number Diff line change
Expand Up @@ -533,6 +533,7 @@ message AutoSpanConfigReconciliationDetails {
// AutoSpanConfigReconciliationProgress is the persisted progress for the span
// config reconciliation job.
message AutoSpanConfigReconciliationProgress {
util.hlc.Timestamp checkpoint = 1 [(gogoproto.nullable) = false];
}

message ResumeSpanList {
Expand Down
1 change: 1 addition & 0 deletions pkg/keys/constants.go
Original file line number Diff line number Diff line change
Expand Up @@ -287,6 +287,7 @@ var (
// ScratchRangeMin is a key used in tests to write arbitrary data without
// overlapping with meta, system or tenant ranges.
ScratchRangeMin = TableDataMax
ScratchRangeMax = TenantPrefix
//
// SystemConfigSplitKey is the key to split at immediately prior to the
// system config span. NB: Split keys need to be valid column keys.
Expand Down
8 changes: 8 additions & 0 deletions pkg/kv/kvserver/client_raft_test.go
Original file line number Diff line number Diff line change
Expand Up @@ -38,6 +38,7 @@ import (
"github.com/cockroachdb/cockroach/pkg/server"
"github.com/cockroachdb/cockroach/pkg/server/serverpb"
"github.com/cockroachdb/cockroach/pkg/settings/cluster"
"github.com/cockroachdb/cockroach/pkg/spanconfig"
"github.com/cockroachdb/cockroach/pkg/storage"
"github.com/cockroachdb/cockroach/pkg/testutils"
"github.com/cockroachdb/cockroach/pkg/testutils/skip"
Expand Down Expand Up @@ -3134,6 +3135,13 @@ func TestDecommission(t *testing.T) {
ctx := context.Background()
tc := testcluster.StartTestCluster(t, 5, base.TestClusterArgs{
ReplicationMode: base.ReplicationAuto,
ServerArgs: base.TestServerArgs{
Knobs: base.TestingKnobs{
SpanConfig: &spanconfig.TestingKnobs{
ConfigureScratchRange: true,
},
},
},
})
defer tc.Stopper().Stop(ctx)

Expand Down
3 changes: 3 additions & 0 deletions pkg/kv/kvserver/client_replica_test.go
Original file line number Diff line number Diff line change
Expand Up @@ -2073,6 +2073,9 @@ func TestSystemZoneConfigs(t *testing.T) {
DisableLoadBasedSplitting: true,
},
},
// This test was written for the gossip-backed SystemConfigSpan
// infrastructure.
DisableSpanConfigs: true,
// Scan like a bat out of hell to ensure replication and replica GC
// happen in a timely manner.
ScanInterval: 50 * time.Millisecond,
Expand Down
1 change: 0 additions & 1 deletion pkg/kv/kvserver/client_spanconfigs_test.go
Original file line number Diff line number Diff line change
Expand Up @@ -43,7 +43,6 @@ func TestSpanConfigUpdateAppliedToReplica(t *testing.T) {
ctx := context.Background()

args := base.TestServerArgs{
EnableSpanConfigs: true,
Knobs: base.TestingKnobs{
Store: &kvserver.StoreTestingKnobs{
DisableMergeQueue: true,
Expand Down
15 changes: 12 additions & 3 deletions pkg/kv/kvserver/client_split_test.go
Original file line number Diff line number Diff line change
Expand Up @@ -1014,6 +1014,8 @@ func TestStoreZoneUpdateAndRangeSplit(t *testing.T) {
Knobs: base.TestingKnobs{
Store: &kvserver.StoreTestingKnobs{
DisableMergeQueue: true,
// This test was written with the SystemConfigSpan in mind.
UseSystemConfigSpanForQueues: true,
},
},
})
Expand Down Expand Up @@ -1083,6 +1085,8 @@ func TestStoreRangeSplitWithMaxBytesUpdate(t *testing.T) {
Knobs: base.TestingKnobs{
Store: &kvserver.StoreTestingKnobs{
DisableMergeQueue: true,
// This test was written with the system config span in mind.
UseSystemConfigSpanForQueues: true,
},
},
})
Expand Down Expand Up @@ -1308,7 +1312,10 @@ func TestStoreRangeSystemSplits(t *testing.T) {
// Intentionally leave the merge queue enabled. This indirectly tests that the
// merge queue respects these split points.
ctx := context.Background()
s, _, _ := serverutils.StartServer(t, base.TestServerArgs{})
s, _, _ := serverutils.StartServer(t, base.TestServerArgs{
// This test was written with the system config span in mind.
DisableSpanConfigs: true,
})
defer s.Stopper().Stop(ctx)

userTableMax := keys.TestingUserDescID(4)
Expand Down Expand Up @@ -3557,8 +3564,10 @@ func TestStoreRangeSplitAndMergeWithGlobalReads(t *testing.T) {
serv, _, _ := serverutils.StartServer(t, base.TestServerArgs{
Knobs: base.TestingKnobs{
Store: &kvserver.StoreTestingKnobs{
DisableMergeQueue: true,
TestingResponseFilter: respFilter,
DisableMergeQueue: true,
// This test was written with the system config span in mind.
UseSystemConfigSpanForQueues: true,
TestingResponseFilter: respFilter,
},
},
})
Expand Down
20 changes: 19 additions & 1 deletion pkg/kv/kvserver/replicate_queue_test.go
Original file line number Diff line number Diff line change
Expand Up @@ -33,6 +33,7 @@ import (
"github.com/cockroachdb/cockroach/pkg/kv/kvserver/liveness/livenesspb"
"github.com/cockroachdb/cockroach/pkg/roachpb"
"github.com/cockroachdb/cockroach/pkg/server"
"github.com/cockroachdb/cockroach/pkg/spanconfig"
"github.com/cockroachdb/cockroach/pkg/testutils"
"github.com/cockroachdb/cockroach/pkg/testutils/skip"
"github.com/cockroachdb/cockroach/pkg/testutils/sqlutils"
Expand Down Expand Up @@ -414,7 +415,16 @@ func TestReplicateQueueDecommissioningNonVoters(t *testing.T) {
// Setup a scratch range on a test cluster with 2 non-voters and 1 voter.
setupFn := func(t *testing.T) (*testcluster.TestCluster, roachpb.RangeDescriptor) {
tc := testcluster.StartTestCluster(t, 5,
base.TestClusterArgs{ReplicationMode: base.ReplicationAuto},
base.TestClusterArgs{
ReplicationMode: base.ReplicationAuto,
ServerArgs: base.TestServerArgs{
Knobs: base.TestingKnobs{
SpanConfig: &spanconfig.TestingKnobs{
ConfigureScratchRange: true,
},
},
},
},
)

scratchKey := tc.ScratchRange(t)
Expand Down Expand Up @@ -529,6 +539,9 @@ func TestReplicateQueueDeadNonVoters(t *testing.T) {
ReplicationMode: base.ReplicationAuto,
ServerArgs: base.TestServerArgs{
Knobs: base.TestingKnobs{
SpanConfig: &spanconfig.TestingKnobs{
ConfigureScratchRange: true,
},
NodeLiveness: kvserver.NodeLivenessTestingKnobs{
StorePoolNodeLivenessFn: func(
id roachpb.NodeID, now time.Time, duration time.Duration,
Expand Down Expand Up @@ -678,6 +691,11 @@ func TestReplicateQueueSwapVotersWithNonVoters(t *testing.T) {
},
},
},
Knobs: base.TestingKnobs{
SpanConfig: &spanconfig.TestingKnobs{
ConfigureScratchRange: true,
},
},
}
}
clusterArgs := base.TestClusterArgs{
Expand Down
Loading

0 comments on commit 365b4da

Please sign in to comment.