Skip to content

Commit

Permalink
schematelemetry: add job, schedule, controller and executor
Browse files Browse the repository at this point in the history
This commit adds all the boilerplate for automatically scheduling SQL
schema telemetry jobs, by default on a daily basis.

This commit also adds a couple of builtins to create the job schedule
and also to immediately create a schema telemetry logging job.

Fixes #84284.

Release note (sql change): this change ensures the existence of
a new scheduled job for collecting SQL schema telemetry. By default it
runs daily but this can be adjusted via the
sql.schema.telemetry.recurrence cluster setting. The schedule can also
be paused via PAUSE SCHEDULE followed by its ID, which can be retrieved
by SELECT * FROM [SHOW SCHEDULES] WHERE label = 'sql-schema-telemetry'.
  • Loading branch information
Marius Posta committed Jul 21, 2022
1 parent 0db7d8f commit dcf7d5f
Show file tree
Hide file tree
Showing 37 changed files with 1,066 additions and 10 deletions.
4 changes: 3 additions & 1 deletion docs/generated/settings/settings-for-tenants.txt
Original file line number Diff line number Diff line change
Expand Up @@ -248,6 +248,8 @@ sql.multiple_modifications_of_table.enabled boolean false if true, allow stateme
sql.multiregion.drop_primary_region.enabled boolean true allows dropping the PRIMARY REGION of a database if it is the last region
sql.notices.enabled boolean true enable notices in the server/client protocol being sent
sql.optimizer.uniqueness_checks_for_gen_random_uuid.enabled boolean false if enabled, uniqueness checks may be planned for mutations of UUID columns updated with gen_random_uuid(); otherwise, uniqueness is assumed due to near-zero collision probability
sql.schema.telemetry.recurrence string @daily cron-tab recurrence for SQL schema telemetry job
sql.schema_telemetry.job.enabled boolean true whether the schema telemetry job is enabled
sql.spatial.experimental_box2d_comparison_operators.enabled boolean false enables the use of certain experimental box2d comparison operators
sql.stats.automatic_collection.enabled boolean true automatic statistics collection mode
sql.stats.automatic_collection.fraction_stale_rows float 0.2 target fraction of stale rows per table that will trigger a statistics refresh
Expand Down Expand Up @@ -283,4 +285,4 @@ trace.jaeger.agent string the address of a Jaeger agent to receive traces using
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.span_registry.enabled boolean true if set, ongoing traces can be seen at https://<ui>/#/debug/tracez
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 22.1-28 set the active cluster version in the format '<major>.<minor>'
version version 22.1-30 set the active cluster version in the format '<major>.<minor>'
4 changes: 3 additions & 1 deletion docs/generated/settings/settings.html
Original file line number Diff line number Diff line change
Expand Up @@ -179,6 +179,8 @@
<tr><td><code>sql.multiregion.drop_primary_region.enabled</code></td><td>boolean</td><td><code>true</code></td><td>allows dropping the PRIMARY REGION of a database if it is the last region</td></tr>
<tr><td><code>sql.notices.enabled</code></td><td>boolean</td><td><code>true</code></td><td>enable notices in the server/client protocol being sent</td></tr>
<tr><td><code>sql.optimizer.uniqueness_checks_for_gen_random_uuid.enabled</code></td><td>boolean</td><td><code>false</code></td><td>if enabled, uniqueness checks may be planned for mutations of UUID columns updated with gen_random_uuid(); otherwise, uniqueness is assumed due to near-zero collision probability</td></tr>
<tr><td><code>sql.schema.telemetry.recurrence</code></td><td>string</td><td><code>@daily</code></td><td>cron-tab recurrence for SQL schema telemetry job</td></tr>
<tr><td><code>sql.schema_telemetry.job.enabled</code></td><td>boolean</td><td><code>true</code></td><td>whether the schema telemetry job is enabled</td></tr>
<tr><td><code>sql.spatial.experimental_box2d_comparison_operators.enabled</code></td><td>boolean</td><td><code>false</code></td><td>enables the use of certain experimental box2d comparison operators</td></tr>
<tr><td><code>sql.stats.automatic_collection.enabled</code></td><td>boolean</td><td><code>true</code></td><td>automatic statistics collection mode</td></tr>
<tr><td><code>sql.stats.automatic_collection.fraction_stale_rows</code></td><td>float</td><td><code>0.2</code></td><td>target fraction of stale rows per table that will trigger a statistics refresh</td></tr>
Expand Down Expand Up @@ -214,6 +216,6 @@
<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.span_registry.enabled</code></td><td>boolean</td><td><code>true</code></td><td>if set, ongoing traces can be seen at https://<ui>/#/debug/tracez</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>22.1-28</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>22.1-30</code></td><td>set the active cluster version in the format '<major>.<minor>'</td></tr>
</tbody>
</table>
4 changes: 4 additions & 0 deletions docs/generated/sql/functions.md
Original file line number Diff line number Diff line change
Expand Up @@ -3023,6 +3023,8 @@ SELECT * FROM crdb_internal.check_consistency(true, ‘\x02’, ‘\x04’)</p>
</span></td><td>Volatile</td></tr>
<tr><td><a name="crdb_internal.create_session_revival_token"></a><code>crdb_internal.create_session_revival_token() &rarr; <a href="bytes.html">bytes</a></code></td><td><span class="funcdesc"><p>Generate a token that can be used to create a new session for the current user.</p>
</span></td><td>Volatile</td></tr>
<tr><td><a name="crdb_internal.create_sql_schema_telemetry_job"></a><code>crdb_internal.create_sql_schema_telemetry_job() &rarr; <a href="int.html">int</a></code></td><td><span class="funcdesc"><p>This function is used to create a schema telemetry job instance.</p>
</span></td><td>Volatile</td></tr>
<tr><td><a name="crdb_internal.decode_cluster_setting"></a><code>crdb_internal.decode_cluster_setting(setting: <a href="string.html">string</a>, value: <a href="string.html">string</a>) &rarr; <a href="string.html">string</a></code></td><td><span class="funcdesc"><p>Decodes the given encoded value for a cluster setting.</p>
</span></td><td>Immutable</td></tr>
<tr><td><a name="crdb_internal.deserialize_session"></a><code>crdb_internal.deserialize_session(session: <a href="bytes.html">bytes</a>) &rarr; <a href="bool.html">bool</a></code></td><td><span class="funcdesc"><p>This function deserializes the serialized variables into the current session.</p>
Expand Down Expand Up @@ -3120,6 +3122,8 @@ table. Returns an error if validation fails.</p>
</span></td><td>Immutable</td></tr>
<tr><td><a name="crdb_internal.round_decimal_values"></a><code>crdb_internal.round_decimal_values(val: <a href="decimal.html">decimal</a>[], scale: <a href="int.html">int</a>) &rarr; <a href="decimal.html">decimal</a>[]</code></td><td><span class="funcdesc"><p>This function is used internally to round decimal array values during mutations.</p>
</span></td><td>Stable</td></tr>
<tr><td><a name="crdb_internal.schedule_sql_schema_telemetry"></a><code>crdb_internal.schedule_sql_schema_telemetry() &rarr; <a href="bool.html">bool</a></code></td><td><span class="funcdesc"><p>This function is used to create the schema telemetry job schedule.</p>
</span></td><td>Volatile</td></tr>
<tr><td><a name="crdb_internal.schedule_sql_stats_compaction"></a><code>crdb_internal.schedule_sql_stats_compaction() &rarr; <a href="bool.html">bool</a></code></td><td><span class="funcdesc"><p>This function is used to start a SQL stats compaction job.</p>
</span></td><td>Volatile</td></tr>
<tr><td><a name="crdb_internal.serialize_session"></a><code>crdb_internal.serialize_session() &rarr; <a href="bytes.html">bytes</a></code></td><td><span class="funcdesc"><p>This function serializes the variables in the current session.</p>
Expand Down
4 changes: 4 additions & 0 deletions pkg/BUILD.bazel
Original file line number Diff line number Diff line change
Expand Up @@ -255,6 +255,7 @@ ALL_TESTS = [
"//pkg/sql/catalog/resolver:resolver_test",
"//pkg/sql/catalog/schemadesc:schemadesc_test",
"//pkg/sql/catalog/schemaexpr:schemaexpr_test",
"//pkg/sql/catalog/schematelemetry:schematelemetry_test",
"//pkg/sql/catalog/seqexpr:seqexpr_disallowed_imports_test",
"//pkg/sql/catalog/seqexpr:seqexpr_test",
"//pkg/sql/catalog/systemschema_test:systemschema_test_test",
Expand Down Expand Up @@ -1246,7 +1247,9 @@ GO_TARGETS = [
"//pkg/sql/catalog/schemadesc:schemadesc_test",
"//pkg/sql/catalog/schemaexpr:schemaexpr",
"//pkg/sql/catalog/schemaexpr:schemaexpr_test",
"//pkg/sql/catalog/schematelemetry/schematelemetrycontroller:schematelemetrycontroller",
"//pkg/sql/catalog/schematelemetry:schematelemetry",
"//pkg/sql/catalog/schematelemetry:schematelemetry_test",
"//pkg/sql/catalog/seqexpr:seqexpr",
"//pkg/sql/catalog/seqexpr:seqexpr_test",
"//pkg/sql/catalog/systemschema:systemschema",
Expand Down Expand Up @@ -2351,6 +2354,7 @@ GET_X_DATA_TARGETS = [
"//pkg/sql/catalog/schemadesc:get_x_data",
"//pkg/sql/catalog/schemaexpr:get_x_data",
"//pkg/sql/catalog/schematelemetry:get_x_data",
"//pkg/sql/catalog/schematelemetry/schematelemetrycontroller:get_x_data",
"//pkg/sql/catalog/seqexpr:get_x_data",
"//pkg/sql/catalog/systemschema:get_x_data",
"//pkg/sql/catalog/systemschema_test:get_x_data",
Expand Down
1 change: 1 addition & 0 deletions pkg/base/testing_knobs.go
Original file line number Diff line number Diff line change
Expand Up @@ -38,6 +38,7 @@ type TestingKnobs struct {
JobsTestingKnobs ModuleTestingKnobs
BackupRestore ModuleTestingKnobs
TTL ModuleTestingKnobs
SchemaTelemetry ModuleTestingKnobs
Streaming ModuleTestingKnobs
UpgradeManager ModuleTestingKnobs
IndexUsageStatsKnobs ModuleTestingKnobs
Expand Down
7 changes: 7 additions & 0 deletions pkg/clusterversion/cockroach_versions.go
Original file line number Diff line number Diff line change
Expand Up @@ -327,6 +327,9 @@ const (
AlterSystemSQLInstancesAddLocality
// SystemExternalConnectionsTable adds system.external_connections table.
SystemExternalConnectionsTable
// SQLSchemaTelemetryScheduledJobs adds an automatic schedule for SQL schema
// telemetry logging jobs.
SQLSchemaTelemetryScheduledJobs

// *************************************************
// Step (1): Add new versions here.
Expand Down Expand Up @@ -564,6 +567,10 @@ var versionsSingleton = keyedVersions{
Key: SystemExternalConnectionsTable,
Version: roachpb.Version{Major: 22, Minor: 1, Internal: 28},
},
{
Key: SQLSchemaTelemetryScheduledJobs,
Version: roachpb.Version{Major: 22, Minor: 1, Internal: 30},
},

// *************************************************
// Step (2): Add new versions here.
Expand Down
5 changes: 3 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/gen/protobuf.bzl
Original file line number Diff line number Diff line change
Expand Up @@ -37,6 +37,7 @@ PROTOBUF_SRCS = [
"//pkg/settings:settings_go_proto",
"//pkg/sql/catalog/catpb:catpb_go_proto",
"//pkg/sql/catalog/descpb:descpb_go_proto",
"//pkg/sql/catalog/schematelemetry/schematelemetrycontroller:schematelemetrycontroller_go_proto",
"//pkg/sql/contentionpb:contentionpb_go_proto",
"//pkg/sql/execinfrapb:execinfrapb_go_proto",
"//pkg/sql/inverted:inverted_go_proto",
Expand Down
1 change: 1 addition & 0 deletions pkg/server/BUILD.bazel
Original file line number Diff line number Diff line change
Expand Up @@ -161,6 +161,7 @@ go_library(
"//pkg/sql/catalog/descs",
"//pkg/sql/catalog/hydratedtables",
"//pkg/sql/catalog/lease",
"//pkg/sql/catalog/schematelemetry",
"//pkg/sql/catalog/systemschema",
"//pkg/sql/clusterunique",
"//pkg/sql/colexec",
Expand Down
1 change: 1 addition & 0 deletions pkg/server/server.go
Original file line number Diff line number Diff line change
Expand Up @@ -60,6 +60,7 @@ import (
"github.com/cockroachdb/cockroach/pkg/spanconfig/spanconfigkvsubscriber"
"github.com/cockroachdb/cockroach/pkg/spanconfig/spanconfigptsreader"
"github.com/cockroachdb/cockroach/pkg/sql"
_ "github.com/cockroachdb/cockroach/pkg/sql/catalog/schematelemetry" // register schedules declared outside of pkg/sql
"github.com/cockroachdb/cockroach/pkg/sql/catalog/systemschema"
"github.com/cockroachdb/cockroach/pkg/sql/flowinfra"
_ "github.com/cockroachdb/cockroach/pkg/sql/gcjob" // register jobs declared outside of pkg/sql
Expand Down
4 changes: 4 additions & 0 deletions pkg/server/server_sql.go
Original file line number Diff line number Diff line change
Expand Up @@ -873,6 +873,9 @@ func newSQLServer(ctx context.Context, cfg sqlServerArgs) (*SQLServer, error) {
if ttlKnobs := cfg.TestingKnobs.TTL; ttlKnobs != nil {
execCfg.TTLTestingKnobs = ttlKnobs.(*sql.TTLTestingKnobs)
}
if schemaTelemetryKnobs := cfg.TestingKnobs.SchemaTelemetry; schemaTelemetryKnobs != nil {
execCfg.SchemaTelemetryTestingKnobs = schemaTelemetryKnobs.(*sql.SchemaTelemetryTestingKnobs)
}
if sqlStatsKnobs := cfg.TestingKnobs.SQLStatsKnobs; sqlStatsKnobs != nil {
execCfg.SQLStatsTestingKnobs = sqlStatsKnobs.(*sqlstats.TestingKnobs)
}
Expand Down Expand Up @@ -913,6 +916,7 @@ func newSQLServer(ctx context.Context, cfg sqlServerArgs) (*SQLServer, error) {
)

distSQLServer.ServerConfig.SQLStatsController = pgServer.SQLServer.GetSQLStatsController()
distSQLServer.ServerConfig.SchemaTelemetryController = pgServer.SQLServer.GetSchemaTelemetryController()
distSQLServer.ServerConfig.IndexUsageStatsController = pgServer.SQLServer.GetIndexUsageStatsController()

// We use one BytesMonitor for all InternalExecutor's created by the
Expand Down
1 change: 1 addition & 0 deletions pkg/sql/BUILD.bazel
Original file line number Diff line number Diff line change
Expand Up @@ -319,6 +319,7 @@ go_library(
"//pkg/sql/catalog/resolver",
"//pkg/sql/catalog/schemadesc",
"//pkg/sql/catalog/schemaexpr",
"//pkg/sql/catalog/schematelemetry/schematelemetrycontroller",
"//pkg/sql/catalog/seqexpr",
"//pkg/sql/catalog/systemschema",
"//pkg/sql/catalog/tabledesc",
Expand Down
48 changes: 46 additions & 2 deletions pkg/sql/catalog/schematelemetry/BUILD.bazel
Original file line number Diff line number Diff line change
@@ -1,26 +1,70 @@
load("//build/bazelutil/unused_checker:unused.bzl", "get_x_data")
load("@io_bazel_rules_go//go:def.bzl", "go_library")
load("@io_bazel_rules_go//go:def.bzl", "go_library", "go_test")

go_library(
name = "schematelemetry",
srcs = ["schema_telemetry_event.go"],
srcs = [
"scheduled_job_executor.go",
"schema_telemetry_event.go",
"schema_telemetry_job.go",
],
importpath = "github.com/cockroachdb/cockroach/pkg/sql/catalog/schematelemetry",
visibility = ["//visibility:public"],
deps = [
"//pkg/jobs",
"//pkg/jobs/jobspb",
"//pkg/kv",
"//pkg/scheduledjobs",
"//pkg/security/username",
"//pkg/server/telemetry",
"//pkg/settings",
"//pkg/settings/cluster",
"//pkg/sql",
"//pkg/sql/catalog",
"//pkg/sql/catalog/colinfo",
"//pkg/sql/catalog/descpb",
"//pkg/sql/catalog/descs",
"//pkg/sql/catalog/schematelemetry/schematelemetrycontroller",
"//pkg/sql/descmetadata",
"//pkg/sql/schemachanger/scdecomp",
"//pkg/sql/schemachanger/scpb",
"//pkg/sql/schemachanger/screl",
"//pkg/sql/sem/tree",
"//pkg/sql/sqltelemetry",
"//pkg/sql/sqlutil",
"//pkg/util/hlc",
"//pkg/util/log/eventpb",
"//pkg/util/metric",
"@com_github_cockroachdb_errors//:errors",
"@com_github_gogo_protobuf//types",
],
)

go_test(
name = "schematelemetry_test",
srcs = [
"main_test.go",
"schema_telemetry_test.go",
],
deps = [
"//pkg/base",
"//pkg/jobs",
"//pkg/jobs/jobstest",
"//pkg/security/securityassets",
"//pkg/security/securitytest",
"//pkg/server",
"//pkg/sql",
"//pkg/sql/catalog/schematelemetry/schematelemetrycontroller",
"//pkg/sql/sem/builtins/builtinconstants",
"//pkg/sql/sem/tree",
"//pkg/testutils/serverutils",
"//pkg/testutils/sqlutils",
"//pkg/testutils/testcluster",
"//pkg/util/leaktest",
"//pkg/util/log",
"//pkg/util/timeutil",
"@com_github_stretchr_testify//require",
],
)

get_x_data(name = "get_x_data")
31 changes: 31 additions & 0 deletions pkg/sql/catalog/schematelemetry/main_test.go
Original file line number Diff line number Diff line change
@@ -0,0 +1,31 @@
// Copyright 2022 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 schematelemetry_test

import (
"os"
"testing"

"github.com/cockroachdb/cockroach/pkg/security/securityassets"
"github.com/cockroachdb/cockroach/pkg/security/securitytest"
"github.com/cockroachdb/cockroach/pkg/server"
"github.com/cockroachdb/cockroach/pkg/testutils/serverutils"
"github.com/cockroachdb/cockroach/pkg/testutils/testcluster"
)

//go:generate ../../../util/leaktest/add-leaktest.sh *_test.go

func TestMain(m *testing.M) {
securityassets.SetLoader(securitytest.EmbeddedAssets)
serverutils.InitTestServerFactory(server.TestServerFactory)
serverutils.InitTestClusterFactory(testcluster.TestClusterFactory)
os.Exit(m.Run())
}
Loading

0 comments on commit dcf7d5f

Please sign in to comment.