diff --git a/pkg/base/test_server_args.go b/pkg/base/test_server_args.go index e72db2084e34..5406baf35c53 100644 --- a/pkg/base/test_server_args.go +++ b/pkg/base/test_server_args.go @@ -216,11 +216,10 @@ type TestTenantArgs struct { // cluster settings. AllowSettingClusterSettings bool - // TenantIDCodecOverride overrides the tenant ID used to construct the SQL - // server's codec, but nothing else (e.g. its certs). Used for testing. - TenantIDCodecOverride roachpb.TenantID - // Stopper, if not nil, is used to stop the tenant manually otherwise the // TestServer stopper will be used. Stopper *stop.Stopper + + // TestingKnobs for the test server. + TestingKnobs TestingKnobs } diff --git a/pkg/ccl/backupccl/backup_test.go b/pkg/ccl/backupccl/backup_test.go index 4320051824b1..a449d824c891 100644 --- a/pkg/ccl/backupccl/backup_test.go +++ b/pkg/ccl/backupccl/backup_test.go @@ -3465,7 +3465,7 @@ func TestBackupTenantsWithRevisionHistory(t *testing.T) { ctx, tc, sqlDB, _, cleanupFn := BackupRestoreTestSetup(t, singleNode, numAccounts, InitNone) defer cleanupFn() - _, _, err := tc.Servers[0].StartTenant(base.TestTenantArgs{TenantID: roachpb.MakeTenantID(10)}) + _, err := tc.Servers[0].StartTenant(base.TestTenantArgs{TenantID: roachpb.MakeTenantID(10)}) require.NoError(t, err) const msg = "can not backup tenants with revision history" @@ -5819,17 +5819,17 @@ func TestBackupRestoreTenant(t *testing.T) { _ = security.EmbeddedTenantIDs() // Setup a few tenants, each with a different table. - conn10 := serverutils.StartTenant(t, srv, base.TestTenantArgs{TenantID: roachpb.MakeTenantID(10)}) + _, conn10 := serverutils.StartTenant(t, srv, base.TestTenantArgs{TenantID: roachpb.MakeTenantID(10)}) defer conn10.Close() tenant10 := sqlutils.MakeSQLRunner(conn10) tenant10.Exec(t, `CREATE DATABASE foo; CREATE TABLE foo.bar(i int primary key); INSERT INTO foo.bar VALUES (110), (210)`) - conn11 := serverutils.StartTenant(t, srv, base.TestTenantArgs{TenantID: roachpb.MakeTenantID(11)}) + _, conn11 := serverutils.StartTenant(t, srv, base.TestTenantArgs{TenantID: roachpb.MakeTenantID(11)}) defer conn11.Close() tenant11 := sqlutils.MakeSQLRunner(conn11) tenant11.Exec(t, `CREATE DATABASE foo; CREATE TABLE foo.baz(i int primary key); INSERT INTO foo.baz VALUES (111), (211)`) - conn20 := serverutils.StartTenant(t, srv, base.TestTenantArgs{TenantID: roachpb.MakeTenantID(20)}) + _, conn20 := serverutils.StartTenant(t, srv, base.TestTenantArgs{TenantID: roachpb.MakeTenantID(20)}) defer conn20.Close() tenant20 := sqlutils.MakeSQLRunner(conn20) tenant20.Exec(t, `CREATE DATABASE foo; CREATE TABLE foo.qux(i int primary key); INSERT INTO foo.qux VALUES (120), (220)`) @@ -5881,7 +5881,7 @@ func TestBackupRestoreTenant(t *testing.T) { ) ten10Stopper := stop.NewStopper() - restoreConn10 := serverutils.StartTenant( + _, restoreConn10 := serverutils.StartTenant( t, restoreTC.Server(0), base.TestTenantArgs{ TenantID: roachpb.MakeTenantID(10), Existing: true, Stopper: ten10Stopper, }, @@ -5914,7 +5914,7 @@ func TestBackupRestoreTenant(t *testing.T) { [][]string{{`10`, `true`, `{"id": "10", "state": "ACTIVE"}`}}, ) - restoreConn10 = serverutils.StartTenant( + _, restoreConn10 = serverutils.StartTenant( t, restoreTC.Server(0), base.TestTenantArgs{TenantID: roachpb.MakeTenantID(10), Existing: true}, ) defer restoreConn10.Close() @@ -5938,7 +5938,7 @@ func TestBackupRestoreTenant(t *testing.T) { [][]string{{`10`, `true`, `{"id": "10", "state": "ACTIVE"}`}}, ) - restoreConn10 := serverutils.StartTenant( + _, restoreConn10 := serverutils.StartTenant( t, restoreTC.Server(0), base.TestTenantArgs{TenantID: roachpb.MakeTenantID(10), Existing: true}, ) defer restoreConn10.Close() @@ -5967,7 +5967,7 @@ func TestBackupRestoreTenant(t *testing.T) { }, ) - restoreConn10 := serverutils.StartTenant( + _, restoreConn10 := serverutils.StartTenant( t, restoreTC.Server(0), base.TestTenantArgs{TenantID: roachpb.MakeTenantID(10), Existing: true}, ) defer restoreConn10.Close() @@ -5976,7 +5976,7 @@ func TestBackupRestoreTenant(t *testing.T) { restoreTenant10.CheckQueryResults(t, `select * from foo.bar`, tenant10.QueryStr(t, `select * from foo.bar`)) restoreTenant10.CheckQueryResults(t, `select * from foo.bar2`, tenant10.QueryStr(t, `select * from foo.bar2`)) - restoreConn11 := serverutils.StartTenant( + _, restoreConn11 := serverutils.StartTenant( t, restoreTC.Server(0), base.TestTenantArgs{TenantID: roachpb.MakeTenantID(11), Existing: true}, ) defer restoreConn11.Close() @@ -5994,7 +5994,7 @@ func TestBackupRestoreTenant(t *testing.T) { restoreDB.Exec(t, `RESTORE TENANT 10 FROM 'nodelocal://1/t10' AS OF SYSTEM TIME `+ts1) - restoreConn10 := serverutils.StartTenant( + _, restoreConn10 := serverutils.StartTenant( t, restoreTC.Server(0), base.TestTenantArgs{TenantID: roachpb.MakeTenantID(10), Existing: true}, ) defer restoreConn10.Close() @@ -6012,7 +6012,7 @@ func TestBackupRestoreTenant(t *testing.T) { restoreDB.Exec(t, `RESTORE TENANT 20 FROM 'nodelocal://1/t20'`) - restoreConn20 := serverutils.StartTenant( + _, restoreConn20 := serverutils.StartTenant( t, restoreTC.Server(0), base.TestTenantArgs{TenantID: roachpb.MakeTenantID(20), Existing: true}, ) defer restoreConn20.Close() diff --git a/pkg/ccl/backupccl/show_test.go b/pkg/ccl/backupccl/show_test.go index f80b6d93f8e6..c3d15c077b1b 100644 --- a/pkg/ccl/backupccl/show_test.go +++ b/pkg/ccl/backupccl/show_test.go @@ -418,7 +418,7 @@ func TestShowBackupTenants(t *testing.T) { // NB: tenant certs for 10, 11, 20 are embedded. See: _ = security.EmbeddedTenantIDs() - conn10 := serverutils.StartTenant(t, srv, base.TestTenantArgs{TenantID: roachpb.MakeTenantID(10)}) + _, conn10 := serverutils.StartTenant(t, srv, base.TestTenantArgs{TenantID: roachpb.MakeTenantID(10)}) defer conn10.Close() tenant10 := sqlutils.MakeSQLRunner(conn10) tenant10.Exec(t, `CREATE DATABASE foo; CREATE TABLE foo.bar(i int primary key); INSERT INTO foo.bar VALUES (110), (210)`) diff --git a/pkg/ccl/serverccl/server_sql_test.go b/pkg/ccl/serverccl/server_sql_test.go index ca96a7199f8a..4fd914ea2906 100644 --- a/pkg/ccl/serverccl/server_sql_test.go +++ b/pkg/ccl/serverccl/server_sql_test.go @@ -18,6 +18,7 @@ import ( "github.com/cockroachdb/cockroach/pkg/base" "github.com/cockroachdb/cockroach/pkg/roachpb" "github.com/cockroachdb/cockroach/pkg/security" + "github.com/cockroachdb/cockroach/pkg/sql" "github.com/cockroachdb/cockroach/pkg/sql/pgwire/pgcode" "github.com/cockroachdb/cockroach/pkg/testutils/serverutils" "github.com/cockroachdb/cockroach/pkg/testutils/sqlutils" @@ -46,7 +47,7 @@ func TestSQLServer(t *testing.T) { tc := serverutils.StartNewTestCluster(t, 3, base.TestClusterArgs{}) defer tc.Stopper().Stop(ctx) - db := serverutils.StartTenant( + _, db := serverutils.StartTenant( t, tc.Server(0), base.TestTenantArgs{TenantID: roachpb.MakeTenantID(security.EmbeddedTenantIDs()[0])}, @@ -72,7 +73,7 @@ func TestTenantCannotSetClusterSetting(t *testing.T) { defer tc.Stopper().Stop(ctx) // StartTenant with the default permissions to - db := serverutils.StartTenant(t, tc.Server(0), base.TestTenantArgs{TenantID: roachpb.MakeTenantID(10), AllowSettingClusterSettings: false}) + _, db := serverutils.StartTenant(t, tc.Server(0), base.TestTenantArgs{TenantID: roachpb.MakeTenantID(10), AllowSettingClusterSettings: false}) defer db.Close() _, err := db.Exec(`SET CLUSTER SETTING sql.defaults.vectorize=off`) var pqErr *pq.Error @@ -89,10 +90,14 @@ func TestTenantUnauthenticatedAccess(t *testing.T) { tc := serverutils.StartNewTestCluster(t, 1, base.TestClusterArgs{}) defer tc.Stopper().Stop(ctx) - _, _, err := tc.Server(0).StartTenant(base.TestTenantArgs{ + _, err := tc.Server(0).StartTenant(base.TestTenantArgs{ TenantID: roachpb.MakeTenantID(security.EmbeddedTenantIDs()[0]), - // Configure the SQL server to access the wrong tenant keyspace. - TenantIDCodecOverride: roachpb.MakeTenantID(security.EmbeddedTenantIDs()[1]), + TestingKnobs: base.TestingKnobs{ + TenantTestingKnobs: &sql.TenantTestingKnobs{ + // Configure the SQL server to access the wrong tenant keyspace. + TenantIDCodecOverride: roachpb.MakeTenantID(security.EmbeddedTenantIDs()[1]), + }, + }, }) require.Error(t, err) require.Regexp(t, `Unauthenticated desc = requested key /Tenant/11/System/"system-version/" not fully contained in tenant keyspace /Tenant/1{0-1}`, err) @@ -107,12 +112,12 @@ func TestTenantHTTP(t *testing.T) { tc := serverutils.StartNewTestCluster(t, 1, base.TestClusterArgs{}) defer tc.Stopper().Stop(ctx) - _, httpAddr, err := tc.Server(0).StartTenant(base.TestTenantArgs{ + tenant, err := tc.Server(0).StartTenant(base.TestTenantArgs{ TenantID: roachpb.MakeTenantID(security.EmbeddedTenantIDs()[0]), }) require.NoError(t, err) t.Run("prometheus", func(t *testing.T) { - resp, err := httputil.Get(ctx, "http://"+httpAddr+"/_status/vars") + resp, err := httputil.Get(ctx, "http://"+tenant.HTTPAddr()+"/_status/vars") defer http.DefaultClient.CloseIdleConnections() require.NoError(t, err) defer resp.Body.Close() @@ -121,7 +126,7 @@ func TestTenantHTTP(t *testing.T) { require.Contains(t, string(body), "sql_ddl_started_count_internal") }) t.Run("pprof", func(t *testing.T) { - resp, err := httputil.Get(ctx, "http://"+httpAddr+"/debug/pprof/goroutine?debug=2") + resp, err := httputil.Get(ctx, "http://"+tenant.HTTPAddr()+"/debug/pprof/goroutine?debug=2") defer http.DefaultClient.CloseIdleConnections() require.NoError(t, err) defer resp.Body.Close() diff --git a/pkg/cli/mt_start_sql.go b/pkg/cli/mt_start_sql.go index d33bf22340e3..469005d8e439 100644 --- a/pkg/cli/mt_start_sql.go +++ b/pkg/cli/mt_start_sql.go @@ -18,6 +18,7 @@ import ( "github.com/cockroachdb/cockroach/pkg/base" "github.com/cockroachdb/cockroach/pkg/clusterversion" "github.com/cockroachdb/cockroach/pkg/server" + "github.com/cockroachdb/cockroach/pkg/settings/cluster" "github.com/cockroachdb/cockroach/pkg/util/log" "github.com/cockroachdb/errors" "github.com/spf13/cobra" @@ -97,7 +98,7 @@ func runStartSQL(cmd *cobra.Command, args []string) error { tempStorageMaxSizeBytes, ) - addr, httpAddr, err := server.StartTenant( + sqlServer, addr, httpAddr, err := server.StartTenant( ctx, stopper, clusterName, @@ -107,6 +108,14 @@ func runStartSQL(cmd *cobra.Command, args []string) error { if err != nil { return err } + + // Start up the diagnostics reporting loop. + // We don't do this in (*server.SQLServer).start() because we don't + // want this overhead and possible interference in tests. + if !cluster.TelemetryOptOut() { + sqlServer.StartDiagnostics(ctx) + } + log.Infof(ctx, "SQL server for tenant %s listening at %s, http at %s", serverCfg.SQLConfig.TenantID, addr, httpAddr) // TODO(tbg): make the other goodies in `./cockroach start` reusable, such as diff --git a/pkg/server/config.go b/pkg/server/config.go index e417e2af8cb1..532d0956730b 100644 --- a/pkg/server/config.go +++ b/pkg/server/config.go @@ -334,12 +334,6 @@ type SQLConfig struct { // // Only applies when the SQL server is deployed individually. TenantKVAddrs []string - - // TenantIDCodecOverride overrides the tenant ID used to construct the SQL - // server's codec, but nothing else (e.g. its certs). Used for testing. - // - // Only applies when the SQL server is deployed individually. - TenantIDCodecOverride roachpb.TenantID } // MakeSQLConfig returns a SQLConfig with default values. diff --git a/pkg/server/diagnostics/main_test.go b/pkg/server/diagnostics/main_test.go new file mode 100644 index 000000000000..48d8f7ce7cfe --- /dev/null +++ b/pkg/server/diagnostics/main_test.go @@ -0,0 +1,27 @@ +// Copyright 2015 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 diagnostics_test + +import ( + "os" + "testing" + + "github.com/cockroachdb/cockroach/pkg/security" + "github.com/cockroachdb/cockroach/pkg/security/securitytest" + "github.com/cockroachdb/cockroach/pkg/server" + "github.com/cockroachdb/cockroach/pkg/testutils/serverutils" +) + +func TestMain(m *testing.M) { + security.SetAssetLoader(securitytest.EmbeddedAssets) + serverutils.InitTestServerFactory(server.TestServerFactory) + os.Exit(m.Run()) +} diff --git a/pkg/server/diagnostics/reporter.go b/pkg/server/diagnostics/reporter.go new file mode 100644 index 000000000000..a3da44d365cd --- /dev/null +++ b/pkg/server/diagnostics/reporter.go @@ -0,0 +1,439 @@ +// Copyright 2016 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 diagnostics + +import ( + "bytes" + "context" + "io/ioutil" + "math/rand" + "net/http" + "reflect" + "runtime" + "time" + + "github.com/cockroachdb/cockroach/pkg/base" + "github.com/cockroachdb/cockroach/pkg/build" + "github.com/cockroachdb/cockroach/pkg/config/zonepb" + "github.com/cockroachdb/cockroach/pkg/keys" + "github.com/cockroachdb/cockroach/pkg/kv" + "github.com/cockroachdb/cockroach/pkg/roachpb" + "github.com/cockroachdb/cockroach/pkg/security" + "github.com/cockroachdb/cockroach/pkg/server/diagnosticspb" + "github.com/cockroachdb/cockroach/pkg/server/status" + "github.com/cockroachdb/cockroach/pkg/server/telemetry" + "github.com/cockroachdb/cockroach/pkg/settings" + "github.com/cockroachdb/cockroach/pkg/settings/cluster" + "github.com/cockroachdb/cockroach/pkg/sql" + "github.com/cockroachdb/cockroach/pkg/sql/catalog/descpb" + "github.com/cockroachdb/cockroach/pkg/sql/sem/tree" + "github.com/cockroachdb/cockroach/pkg/sql/sessiondata" + "github.com/cockroachdb/cockroach/pkg/util/cloudinfo" + "github.com/cockroachdb/cockroach/pkg/util/httputil" + "github.com/cockroachdb/cockroach/pkg/util/log" + "github.com/cockroachdb/cockroach/pkg/util/protoutil" + "github.com/cockroachdb/cockroach/pkg/util/stop" + "github.com/cockroachdb/cockroach/pkg/util/timeutil" + "github.com/cockroachdb/cockroach/pkg/util/uuid" + "github.com/cockroachdb/errors" + "github.com/gogo/protobuf/proto" + "github.com/mitchellh/reflectwalk" + "github.com/shirou/gopsutil/cpu" + "github.com/shirou/gopsutil/host" + "github.com/shirou/gopsutil/load" + "github.com/shirou/gopsutil/mem" +) + +// ReportFrequency is the interval at which diagnostics data should be reported. +var ReportFrequency = settings.RegisterPublicNonNegativeDurationSetting( + "diagnostics.reporting.interval", + "interval at which diagnostics data should be reported", + time.Hour, +) + +const updateCheckJitterSeconds = 120 + +// Reporter is a helper struct that phones home to report usage and diagnostics. +type Reporter struct { + StartTime time.Time + AmbientCtx *log.AmbientContext + Config *base.Config + Settings *cluster.Settings + + // ClusterID is not yet available at the time the reporter is created, so + // instead initialize with a function that gets it dynamically. + ClusterID func() uuid.UUID + TenantID roachpb.TenantID + + // SQLInstanceID is not yet available at the time the reporter is created, + // so instead initialize with a function that gets it dynamically. + SQLInstanceID func() base.SQLInstanceID + SQLServer *sql.Server + InternalExec *sql.InternalExecutor + DB *kv.DB + Recorder *status.MetricsRecorder + + // Locality is a description of the topography of the server. + Locality roachpb.Locality + + // TestingKnobs is used for internal test controls only. + TestingKnobs *diagnosticspb.TestingKnobs +} + +// PeriodicallyReportDiagnostics calls ReportDiagnostics on a regular schedule. +func (r *Reporter) PeriodicallyReportDiagnostics(ctx context.Context, stopper *stop.Stopper) { + stopper.RunWorker(ctx, func(ctx context.Context) { + defer log.RecoverAndReportNonfatalPanic(ctx, &r.Settings.SV) + nextReport := r.StartTime + + var timer timeutil.Timer + defer timer.Stop() + for { + // TODO(dt): we should allow tuning the reset and report intervals separately. + // Consider something like rand.Float() > resetFreq/reportFreq here to sample + // stat reset periods for reporting. + // Always report diagnostics + // 1. In 20.2, this Reporter code only runs for tenants. + // 2. The diagnostics reporting cluster setting is disabled in + // tenant clusters. + // 3. Tenant cluster settings cannot be changed in 20.2. + r.ReportDiagnostics(ctx) + + nextReport = nextReport.Add(ReportFrequency.Get(&r.Settings.SV)) + + timer.Reset(addJitter(nextReport.Sub(timeutil.Now()), updateCheckJitterSeconds)) + select { + case <-stopper.ShouldQuiesce(): + return + case <-timer.C: + timer.Read = true + } + } + }) +} + +// ReportDiagnostics phones home to report usage and diagnostics. +// +// NOTE: This can be slow because of cloud detection; use cloudinfo.Disable() in +// tests to avoid that. +func (r *Reporter) ReportDiagnostics(ctx context.Context) { + ctx, span := r.AmbientCtx.AnnotateCtxWithSpan(ctx, "usageReport") + defer span.Finish() + + report := r.getReportingInfo(ctx, telemetry.ResetCounts) + + clusterInfo := diagnosticspb.ClusterInfo{ + ClusterID: r.ClusterID(), + TenantID: r.TenantID, + IsInsecure: r.Config.Insecure, + IsInternal: sql.ClusterIsInternal(&r.Settings.SV), + } + reportingURL := diagnosticspb.BuildReportingURL(&clusterInfo, report, r.TestingKnobs) + if reportingURL == nil { + return + } + + b, err := protoutil.Marshal(report) + if err != nil { + log.Warningf(ctx, "%v", err) + return + } + + res, err := httputil.Post( + ctx, reportingURL.String(), "application/x-protobuf", bytes.NewReader(b), + ) + if err != nil { + if log.V(2) { + // This is probably going to be relatively common in production + // environments where network access is usually curtailed. + log.Warningf(ctx, "failed to report node usage metrics: %v", err) + } + return + } + defer res.Body.Close() + b, err = ioutil.ReadAll(res.Body) + if err != nil || res.StatusCode != http.StatusOK { + log.Warningf(ctx, "failed to report node usage metrics: status: %s, body: %s, "+ + "error: %v", res.Status, b, err) + return + } + r.SQLServer.ResetReportedStats(ctx) +} + +func (r *Reporter) getReportingInfo( + ctx context.Context, reset telemetry.ResetCounters, +) *diagnosticspb.DiagnosticReport { + info := diagnosticspb.DiagnosticReport{} + secret := sql.ClusterSecret.Get(&r.Settings.SV) + uptime := int64(timeutil.Now().Sub(r.StartTime).Seconds()) + + // Populate the hardware, OS, binary, and location of the CRDB node or SQL + // instance. + r.populateEnvironment(ctx, secret, &info.Env) + + // Always populate SQL info, since even full CRDB running KV will also be + // running SQL. + r.populateSQLInfo(uptime, &info.SQL) + + // Do not collect node or store information for tenant reports. + if r.TenantID == roachpb.SystemTenantID { + r.populateNodeInfo(ctx, uptime, &info) + } + + schema, err := r.collectSchemaInfo(ctx) + if err != nil { + log.Warningf(ctx, "error collecting schema info for diagnostic report: %+v", err) + schema = nil + } + info.Schema = schema + + info.FeatureUsage = telemetry.GetFeatureCounts(telemetry.Quantized, reset) + + // Read the system.settings table to determine the settings for which we have + // explicitly set values -- the in-memory SV has the set and default values + // flattened for quick reads, but we'd rather only report the non-defaults. + if datums, err := r.InternalExec.QueryEx( + ctx, "read-setting", nil, /* txn */ + sessiondata.InternalExecutorOverride{User: security.RootUser}, + "SELECT name FROM system.settings", + ); err != nil { + log.Warningf(ctx, "failed to read settings: %s", err) + } else { + info.AlteredSettings = make(map[string]string, len(datums)) + for _, row := range datums { + name := string(tree.MustBeDString(row[0])) + info.AlteredSettings[name] = settings.RedactedValue(name, &r.Settings.SV) + } + } + + if datums, err := r.InternalExec.QueryEx( + ctx, + "read-zone-configs", + nil, /* txn */ + sessiondata.InternalExecutorOverride{User: security.RootUser}, + "SELECT id, config FROM system.zones", + ); err != nil { + log.Warningf(ctx, "%v", err) + } else { + info.ZoneConfigs = make(map[int64]zonepb.ZoneConfig) + for _, row := range datums { + id := int64(tree.MustBeDInt(row[0])) + var zone zonepb.ZoneConfig + if bytes, ok := row[1].(*tree.DBytes); !ok { + continue + } else { + if err := protoutil.Unmarshal([]byte(*bytes), &zone); err != nil { + log.Warningf(ctx, "unable to parse zone config %d: %v", id, err) + continue + } + } + var anonymizedZone zonepb.ZoneConfig + anonymizeZoneConfig(&anonymizedZone, zone, secret) + info.ZoneConfigs[id] = anonymizedZone + } + } + + info.SqlStats = r.SQLServer.GetScrubbedReportingStats() + return &info +} + +// populateEnvironment fills fields in the given environment, such as the +// hardware, OS, binary, and location of the CRDB node or SQL instance. +func (r *Reporter) populateEnvironment( + ctx context.Context, secret string, env *diagnosticspb.Environment, +) { + env.Build = build.GetInfo() + env.LicenseType = getLicenseType(ctx, r.Settings) + populateHardwareInfo(ctx, env) + + // Add in the localities. + for _, tier := range r.Locality.Tiers { + env.Locality.Tiers = append(env.Locality.Tiers, roachpb.Tier{ + Key: sql.HashForReporting(secret, tier.Key), + Value: sql.HashForReporting(secret, tier.Value), + }) + } +} + +// populateNodeInfo populates the NodeInfo and StoreInfo fields of the +// diagnostics report. +func (r *Reporter) populateNodeInfo( + ctx context.Context, uptime int64, info *diagnosticspb.DiagnosticReport, +) { + n := r.Recorder.GenerateNodeStatus(ctx) + info.Node.NodeID = n.Desc.NodeID + info.Node.Uptime = uptime + + info.Stores = make([]diagnosticspb.StoreInfo, len(n.StoreStatuses)) + for i, r := range n.StoreStatuses { + info.Stores[i].NodeID = r.Desc.Node.NodeID + info.Stores[i].StoreID = r.Desc.StoreID + info.Stores[i].KeyCount = int64(r.Metrics["keycount"]) + info.Stores[i].Capacity = int64(r.Metrics["capacity"]) + info.Stores[i].Available = int64(r.Metrics["capacity.available"]) + info.Stores[i].Used = int64(r.Metrics["capacity.used"]) + info.Node.KeyCount += info.Stores[i].KeyCount + info.Stores[i].RangeCount = int64(r.Metrics["replicas"]) + info.Node.RangeCount += info.Stores[i].RangeCount + bytes := int64(r.Metrics["sysbytes"] + r.Metrics["intentbytes"] + r.Metrics["valbytes"] + r.Metrics["keybytes"]) + info.Stores[i].Bytes = bytes + info.Node.Bytes += bytes + info.Stores[i].EncryptionAlgorithm = int64(r.Metrics["rocksdb.encryption.algorithm"]) + } + + // Fill in all deprecated NodeInfo fields with information that is now in + // other parts of the diagnostics report. + // TODO(andyk): Remove this code once the registration server gets this + // information from the other fields. + info.Node.Locality = info.Env.Locality + info.Node.Hardware = info.Env.Hardware + info.Node.Os = info.Env.Os + info.Node.Build = info.Env.Build + info.Node.LicenseType = info.Env.LicenseType + info.Node.Topology = info.Env.Topology +} + +func (r *Reporter) populateSQLInfo(uptime int64, sql *diagnosticspb.SQLInstanceInfo) { + sql.SQLInstanceID = r.SQLInstanceID() + sql.Uptime = uptime +} + +func (r *Reporter) collectSchemaInfo(ctx context.Context) ([]descpb.TableDescriptor, error) { + startKey := keys.MakeSQLCodec(r.TenantID).TablePrefix(keys.DescriptorTableID) + endKey := startKey.PrefixEnd() + kvs, err := r.DB.Scan(ctx, startKey, endKey, 0) + if err != nil { + return nil, err + } + tables := make([]descpb.TableDescriptor, 0, len(kvs)) + redactor := stringRedactor{} + for _, kv := range kvs { + var desc descpb.Descriptor + if err := kv.ValueProto(&desc); err != nil { + return nil, errors.Wrapf(err, "%s: unable to unmarshal SQL descriptor", kv.Key) + } + if t := descpb.TableFromDescriptor(&desc, kv.Value.Timestamp); t != nil && t.ID > keys.MaxReservedDescID { + if err := reflectwalk.Walk(t, redactor); err != nil { + panic(err) // stringRedactor never returns a non-nil err + } + tables = append(tables, *t) + } + } + return tables, nil +} + +func getLicenseType(ctx context.Context, settings *cluster.Settings) string { + licenseType, err := base.LicenseType(settings) + if err != nil { + log.Errorf(ctx, "error retrieving license type: %s", err) + return "" + } + return licenseType +} + +// populateHardwareInfo populates OS, CPU, memory, etc. information about the +// environment in which CRDB is running. +func populateHardwareInfo(ctx context.Context, e *diagnosticspb.Environment) { + if platform, family, version, err := host.PlatformInformation(); err == nil { + e.Os.Family = family + e.Os.Platform = platform + e.Os.Version = version + } + + if virt, role, err := host.Virtualization(); err == nil && role == "guest" { + e.Hardware.Virtualization = virt + } + + if m, err := mem.VirtualMemory(); err == nil { + e.Hardware.Mem.Available = m.Available + e.Hardware.Mem.Total = m.Total + } + + e.Hardware.Cpu.Numcpu = int32(runtime.NumCPU()) + if cpus, err := cpu.InfoWithContext(ctx); err == nil && len(cpus) > 0 { + e.Hardware.Cpu.Sockets = int32(len(cpus)) + c := cpus[0] + e.Hardware.Cpu.Cores = c.Cores + e.Hardware.Cpu.Model = c.ModelName + e.Hardware.Cpu.Mhz = float32(c.Mhz) + e.Hardware.Cpu.Features = c.Flags + } + + if l, err := load.AvgWithContext(ctx); err == nil { + e.Hardware.Loadavg15 = float32(l.Load15) + } + + e.Hardware.Provider, e.Hardware.InstanceClass = cloudinfo.GetInstanceClass(ctx) + e.Topology.Provider, e.Topology.Region = cloudinfo.GetInstanceRegion(ctx) +} + +func anonymizeZoneConfig(dst *zonepb.ZoneConfig, src zonepb.ZoneConfig, secret string) { + if src.RangeMinBytes != nil { + dst.RangeMinBytes = proto.Int64(*src.RangeMinBytes) + } + if src.RangeMaxBytes != nil { + dst.RangeMaxBytes = proto.Int64(*src.RangeMaxBytes) + } + if src.GC != nil { + dst.GC = &zonepb.GCPolicy{TTLSeconds: src.GC.TTLSeconds} + } + if src.NumReplicas != nil { + dst.NumReplicas = proto.Int32(*src.NumReplicas) + } + dst.Constraints = make([]zonepb.ConstraintsConjunction, len(src.Constraints)) + for i := range src.Constraints { + dst.Constraints[i].NumReplicas = src.Constraints[i].NumReplicas + dst.Constraints[i].Constraints = make([]zonepb.Constraint, len(src.Constraints[i].Constraints)) + for j := range src.Constraints[i].Constraints { + dst.Constraints[i].Constraints[j].Type = src.Constraints[i].Constraints[j].Type + if key := src.Constraints[i].Constraints[j].Key; key != "" { + dst.Constraints[i].Constraints[j].Key = sql.HashForReporting(secret, key) + } + if val := src.Constraints[i].Constraints[j].Value; val != "" { + dst.Constraints[i].Constraints[j].Value = sql.HashForReporting(secret, val) + } + } + } + dst.LeasePreferences = make([]zonepb.LeasePreference, len(src.LeasePreferences)) + for i := range src.LeasePreferences { + dst.LeasePreferences[i].Constraints = make([]zonepb.Constraint, len(src.LeasePreferences[i].Constraints)) + for j := range src.LeasePreferences[i].Constraints { + dst.LeasePreferences[i].Constraints[j].Type = src.LeasePreferences[i].Constraints[j].Type + if key := src.LeasePreferences[i].Constraints[j].Key; key != "" { + dst.LeasePreferences[i].Constraints[j].Key = sql.HashForReporting(secret, key) + } + if val := src.LeasePreferences[i].Constraints[j].Value; val != "" { + dst.LeasePreferences[i].Constraints[j].Value = sql.HashForReporting(secret, val) + } + } + } + dst.Subzones = make([]zonepb.Subzone, len(src.Subzones)) + for i := range src.Subzones { + dst.Subzones[i].IndexID = src.Subzones[i].IndexID + dst.Subzones[i].PartitionName = sql.HashForReporting(secret, src.Subzones[i].PartitionName) + anonymizeZoneConfig(&dst.Subzones[i].Config, src.Subzones[i].Config, secret) + } +} + +type stringRedactor struct{} + +func (stringRedactor) Primitive(v reflect.Value) error { + if v.Kind() == reflect.String && v.String() != "" { + v.Set(reflect.ValueOf("_").Convert(v.Type())) + } + return nil +} + +// randomly shift `d` to be up to `jitterSec` shorter or longer. +func addJitter(d time.Duration, jitterSec int) time.Duration { + j := time.Duration(rand.Intn(jitterSec*2)-jitterSec) * time.Second + return d + j +} diff --git a/pkg/server/diagnostics/reporter_test.go b/pkg/server/diagnostics/reporter_test.go new file mode 100644 index 000000000000..16c67f43ea2c --- /dev/null +++ b/pkg/server/diagnostics/reporter_test.go @@ -0,0 +1,421 @@ +// Copyright 2016 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 diagnostics_test + +import ( + "context" + gosql "database/sql" + "fmt" + "runtime" + "testing" + + "github.com/cockroachdb/cockroach/pkg/base" + "github.com/cockroachdb/cockroach/pkg/ccl/kvccl/kvtenantccl" + "github.com/cockroachdb/cockroach/pkg/clusterversion" + "github.com/cockroachdb/cockroach/pkg/config/zonepb" + "github.com/cockroachdb/cockroach/pkg/keys" + "github.com/cockroachdb/cockroach/pkg/roachpb" + "github.com/cockroachdb/cockroach/pkg/security" + "github.com/cockroachdb/cockroach/pkg/server" + "github.com/cockroachdb/cockroach/pkg/server/diagnostics" + "github.com/cockroachdb/cockroach/pkg/server/diagnosticspb" + "github.com/cockroachdb/cockroach/pkg/server/status" + "github.com/cockroachdb/cockroach/pkg/server/telemetry" + "github.com/cockroachdb/cockroach/pkg/settings/cluster" + "github.com/cockroachdb/cockroach/pkg/sql" + "github.com/cockroachdb/cockroach/pkg/sql/catalog/lease" + "github.com/cockroachdb/cockroach/pkg/testutils" + "github.com/cockroachdb/cockroach/pkg/testutils/diagutils" + "github.com/cockroachdb/cockroach/pkg/testutils/serverutils" + "github.com/cockroachdb/cockroach/pkg/util/cloudinfo" + "github.com/cockroachdb/cockroach/pkg/util/leaktest" + "github.com/cockroachdb/cockroach/pkg/util/log" + "github.com/cockroachdb/errors" + "github.com/stretchr/testify/require" +) + +// Dummy import to pull in kvtenantccl. This allows us to start tenants. +var _ = kvtenantccl.Connector{} + +const elemName = "somestring" + +func TestTenantReport(t *testing.T) { + defer leaktest.AfterTest(t) + defer log.Scope(t).Close(t) + + rt := startReporterTest(t) + defer rt.Close() + + tenantArgs := base.TestTenantArgs{ + TenantID: roachpb.MakeTenantID(security.EmbeddedTenantIDs()[0]), + AllowSettingClusterSettings: true, + TestingKnobs: rt.testingKnobs, + } + tenant, tenantDB := serverutils.StartTenant(t, rt.server, tenantArgs) + reporter := tenant.DiagnosticsReporter().(*diagnostics.Reporter) + + ctx := context.Background() + setupCluster(t, tenantDB) + + // Clear the SQL stat pool before getting diagnostics. + rt.server.SQLServer().(*sql.Server).ResetSQLStats(ctx) + reporter.ReportDiagnostics(ctx) + + require.Equal(t, 1, rt.diagServer.NumRequests()) + + last := rt.diagServer.LastRequestData() + require.Equal(t, rt.server.ClusterID().String(), last.UUID) + require.Equal(t, tenantArgs.TenantID.String(), last.TenantID) + require.Equal(t, "", last.NodeID) + require.Equal(t, tenant.SQLInstanceID().String(), last.SQLInstanceID) + require.Equal(t, "true", last.Internal) + + // Verify environment. + verifyEnvironment(t, "", roachpb.Locality{}, &last.Env) + + // Verify SQL info. + require.Equal(t, tenant.SQLInstanceID(), last.SQL.SQLInstanceID) + + // Verify FeatureUsage. + require.NotZero(t, len(last.FeatureUsage)) + + // Call PeriodicallyReportDiagnostics and ensure it sends out a report. + reporter.PeriodicallyReportDiagnostics(ctx, rt.server.Stopper()) + testutils.SucceedsSoon(t, func() error { + if rt.diagServer.NumRequests() != 2 { + return errors.Errorf("did not receive a diagnostics report") + } + return nil + }) +} + +// TestServerReport checks nodes, stores, localities, and zone configs. +// Telemetry metrics are checked in datadriven tests (see sql.TestTelemetry). +func TestServerReport(t *testing.T) { + defer leaktest.AfterTest(t) + defer log.Scope(t).Close(t) + + rt := startReporterTest(t) + defer rt.Close() + + ctx := context.Background() + setupCluster(t, rt.serverDB) + + for _, cmd := range []struct { + resource string + config string + }{ + {"TABLE system.rangelog", fmt.Sprintf(`constraints: [+zone=%[1]s, +%[1]s]`, elemName)}, + {"TABLE system.rangelog", `{gc: {ttlseconds: 1}}`}, + {"DATABASE system", `num_replicas: 5`}, + {"DATABASE system", fmt.Sprintf(`constraints: {"+zone=%[1]s,+%[1]s": 2, +%[1]s: 1}`, elemName)}, + {"DATABASE system", fmt.Sprintf(`experimental_lease_preferences: [[+zone=%[1]s,+%[1]s], [+%[1]s]]`, elemName)}, + } { + testutils.SucceedsSoon(t, func() error { + if _, err := rt.serverDB.Exec( + fmt.Sprintf(`ALTER %s CONFIGURE ZONE = '%s'`, cmd.resource, cmd.config), + ); err != nil { + // Work around gossip asynchronicity. + return errors.Errorf("error applying zone config %q to %q: %v", cmd.config, cmd.resource, err) + } + return nil + }) + } + + expectedUsageReports := 0 + + clusterSecret := sql.ClusterSecret.Get(&rt.settings.SV) + testutils.SucceedsSoon(t, func() error { + expectedUsageReports++ + + node := rt.server.MetricsRecorder().(*status.MetricsRecorder).GenerateNodeStatus(ctx) + // Clear the SQL stat pool before getting diagnostics. + rt.server.SQLServer().(*sql.Server).ResetSQLStats(ctx) + rt.server.ReportDiagnostics(ctx) + + keyCounts := make(map[roachpb.StoreID]int64) + rangeCounts := make(map[roachpb.StoreID]int64) + totalKeys := int64(0) + totalRanges := int64(0) + + for _, store := range node.StoreStatuses { + keys, ok := store.Metrics["keycount"] + require.True(t, ok, "keycount not in metrics") + totalKeys += int64(keys) + keyCounts[store.Desc.StoreID] = int64(keys) + + replicas, ok := store.Metrics["replicas"] + require.True(t, ok, "replicas not in metrics") + totalRanges += int64(replicas) + rangeCounts[store.Desc.StoreID] = int64(replicas) + } + + require.Equal(t, expectedUsageReports, rt.diagServer.NumRequests()) + + last := rt.diagServer.LastRequestData() + if expected, actual := rt.server.ClusterID().String(), last.UUID; expected != actual { + return errors.Errorf("expected cluster id %v got %v", expected, actual) + } + if expected, actual := "system", last.TenantID; expected != actual { + return errors.Errorf("expected tenant id %v got %v", expected, actual) + } + if expected, actual := rt.server.NodeID().String(), last.NodeID; expected != actual { + return errors.Errorf("expected node id %v got %v", expected, actual) + } + if expected, actual := rt.server.NodeID().String(), last.SQLInstanceID; expected != actual { + return errors.Errorf("expected sql instance id %v got %v", expected, actual) + } + if expected, actual := rt.server.NodeID(), last.Node.NodeID; expected != actual { + return errors.Errorf("expected node id %v got %v", expected, actual) + } + + if last.Node.Hardware.Mem.Total == 0 { + return errors.Errorf("expected non-zero total mem") + } + if last.Node.Hardware.Mem.Available == 0 { + return errors.Errorf("expected non-zero available mem") + } + if actual, expected := last.Node.Hardware.Cpu.Numcpu, runtime.NumCPU(); int(actual) != expected { + return errors.Errorf("expected %d num cpu, got %d", expected, actual) + } + if last.Node.Hardware.Cpu.Sockets == 0 { + return errors.Errorf("expected non-zero sockets") + } + if last.Node.Hardware.Cpu.Mhz == 0.0 { + return errors.Errorf("expected non-zero speed") + } + if last.Node.Os.Platform == "" { + return errors.Errorf("expected non-empty OS") + } + + if minExpected, actual := totalKeys, last.Node.KeyCount; minExpected > actual { + return errors.Errorf("expected node keys at least %v got %v", minExpected, actual) + } + if minExpected, actual := totalRanges, last.Node.RangeCount; minExpected > actual { + return errors.Errorf("expected node ranges at least %v got %v", minExpected, actual) + } + if minExpected, actual := len(rt.serverArgs.StoreSpecs), len(last.Stores); minExpected > actual { + return errors.Errorf("expected at least %v stores got %v", minExpected, actual) + } + if expected, actual := "true", last.Internal; expected != actual { + return errors.Errorf("expected internal to be %v, got %v", expected, actual) + } + if expected, actual := len(rt.serverArgs.Locality.Tiers), len(last.Node.Locality.Tiers); expected != actual { + return errors.Errorf("expected locality to have %d tier, got %d", expected, actual) + } + for i := range rt.serverArgs.Locality.Tiers { + if expected, actual := sql.HashForReporting(clusterSecret, rt.serverArgs.Locality.Tiers[i].Key), + last.Node.Locality.Tiers[i].Key; expected != actual { + return errors.Errorf("expected locality tier %d key to be %s, got %s", i, expected, actual) + } + if expected, actual := sql.HashForReporting(clusterSecret, rt.serverArgs.Locality.Tiers[i].Value), + last.Node.Locality.Tiers[i].Value; expected != actual { + return errors.Errorf("expected locality tier %d value to be %s, got %s", i, expected, actual) + } + } + + for _, store := range last.Stores { + if minExpected, actual := keyCounts[store.StoreID], store.KeyCount; minExpected > actual { + return errors.Errorf("expected at least %v keys in store %v got %v", minExpected, store.StoreID, actual) + } + if minExpected, actual := rangeCounts[store.StoreID], store.RangeCount; minExpected > actual { + return errors.Errorf("expected at least %v ranges in store %v got %v", minExpected, store.StoreID, actual) + } + } + return nil + }) + + last := rt.diagServer.LastRequestData() + // This check isn't clean, since the body is a raw proto binary and thus could + // easily contain some encoded form of elemName, but *if* it ever does fail, + // that is probably very interesting. + require.NotContains(t, last.RawReportBody, elemName) + + // 3 + 3 = 6: set 3 initially and org is set mid-test for 3 altered settings, + // plus version, reporting and secret settings are set in startup + // migrations. + expected, actual := 6, len(last.AlteredSettings) + require.Equal(t, expected, actual, "expected %d changed settings, got %d: %v", expected, actual, last.AlteredSettings) + + for key, expected := range map[string]string{ + "cluster.organization": "", + "diagnostics.reporting.send_crash_reports": "false", + "server.time_until_store_dead": "1m30s", + "version": clusterversion.TestingBinaryVersion.String(), + "cluster.secret": "", + } { + got, ok := last.AlteredSettings[key] + require.True(t, ok, "expected report of altered setting %q", key) + require.Equal(t, expected, got, "expected reported value of setting %q to be %q not %q", key, expected, got) + } + + // Verify that we receive the four auto-populated zone configs plus the two + // modified above, and that their values are as expected. + for _, expectedID := range []int64{ + keys.RootNamespaceID, + keys.LivenessRangesID, + keys.MetaRangesID, + keys.RangeEventTableID, + keys.SystemDatabaseID, + } { + _, ok := last.ZoneConfigs[expectedID] + require.True(t, ok, "didn't find expected ID %d in reported ZoneConfigs: %+v", + expectedID, last.ZoneConfigs) + } + hashedElemName := sql.HashForReporting(clusterSecret, elemName) + hashedZone := sql.HashForReporting(clusterSecret, "zone") + for id, zone := range last.ZoneConfigs { + if id == keys.RootNamespaceID { + require.Equal(t, zone, *rt.server.ExecutorConfig().(sql.ExecutorConfig).DefaultZoneConfig) + } + if id == keys.RangeEventTableID { + require.Equal(t, int32(1), zone.GC.TTLSeconds) + constraints := []zonepb.ConstraintsConjunction{ + { + Constraints: []zonepb.Constraint{ + {Key: hashedZone, Value: hashedElemName, Type: zonepb.Constraint_REQUIRED}, + {Value: hashedElemName, Type: zonepb.Constraint_REQUIRED}, + }, + }, + } + require.Equal(t, zone.Constraints, constraints) + } + if id == keys.SystemDatabaseID { + constraints := []zonepb.ConstraintsConjunction{ + { + NumReplicas: 1, + Constraints: []zonepb.Constraint{{Value: hashedElemName, Type: zonepb.Constraint_REQUIRED}}, + }, + { + NumReplicas: 2, + Constraints: []zonepb.Constraint{ + {Key: hashedZone, Value: hashedElemName, Type: zonepb.Constraint_REQUIRED}, + {Value: hashedElemName, Type: zonepb.Constraint_REQUIRED}, + }, + }, + } + require.Equal(t, constraints, zone.Constraints) + prefs := []zonepb.LeasePreference{ + { + Constraints: []zonepb.Constraint{ + {Key: hashedZone, Value: hashedElemName, Type: zonepb.Constraint_REQUIRED}, + {Value: hashedElemName, Type: zonepb.Constraint_REQUIRED}, + }, + }, + { + Constraints: []zonepb.Constraint{{Value: hashedElemName, Type: zonepb.Constraint_REQUIRED}}, + }, + } + require.Equal(t, prefs, zone.LeasePreferences) + } + } +} + +type reporterTest struct { + cloudEnable func() + settings *cluster.Settings + diagServer *diagutils.Server + testingKnobs base.TestingKnobs + serverArgs base.TestServerArgs + server serverutils.TestServerInterface + serverDB *gosql.DB +} + +func (t *reporterTest) Close() { + t.cloudEnable() + t.diagServer.Close() + // stopper will wait for the update/report loop to finish too. + t.server.Stopper().Stop(context.Background()) +} + +func startReporterTest(t *testing.T) *reporterTest { + // Disable cloud info reporting, since it slows down tests. + rt := &reporterTest{ + cloudEnable: cloudinfo.Disable(), + settings: cluster.MakeTestingClusterSettings(), + diagServer: diagutils.NewServer(), + } + + url := rt.diagServer.URL() + rt.testingKnobs = base.TestingKnobs{ + SQLLeaseManager: &lease.ManagerTestingKnobs{ + // Disable SELECT called for delete orphaned leases to keep + // query stats stable. + DisableDeleteOrphanedLeases: true, + }, + Server: &server.TestingKnobs{ + DiagnosticsTestingKnobs: diagnosticspb.TestingKnobs{ + OverrideReportingURL: &url, + }, + }, + } + + storeSpec := base.DefaultTestStoreSpec + storeSpec.Attributes = roachpb.Attributes{Attrs: []string{elemName}} + rt.serverArgs = base.TestServerArgs{ + StoreSpecs: []base.StoreSpec{ + storeSpec, + base.DefaultTestStoreSpec, + }, + Settings: rt.settings, + Locality: roachpb.Locality{ + Tiers: []roachpb.Tier{ + {Key: "region", Value: "east"}, + {Key: "zone", Value: elemName}, + {Key: "state", Value: "ny"}, + {Key: "city", Value: "nyc"}, + }, + }, + Knobs: rt.testingKnobs, + } + rt.server, rt.serverDB, _ = serverutils.StartServer(t, rt.serverArgs) + + // Make sure the test's generated activity is the only activity we measure. + telemetry.GetFeatureCounts(telemetry.Raw, telemetry.ResetCounts) + + return rt +} + +func setupCluster(t *testing.T, db *gosql.DB) { + _, err := db.Exec(`SET CLUSTER SETTING server.time_until_store_dead = '90s'`) + require.NoError(t, err) + + _, err = db.Exec(`SET CLUSTER SETTING diagnostics.reporting.send_crash_reports = false`) + require.NoError(t, err) + + _, err = db.Exec(fmt.Sprintf(`CREATE DATABASE %s`, elemName)) + require.NoError(t, err) + + // Set cluster to an internal testing cluster + q := `SET CLUSTER SETTING cluster.organization = 'Cockroach Labs - Production Testing'` + _, err = db.Exec(q) + require.NoError(t, err) +} + +func verifyEnvironment( + t *testing.T, secret string, locality roachpb.Locality, env *diagnosticspb.Environment, +) { + require.NotEqual(t, 0, env.Hardware.Mem.Total) + require.NotEqual(t, 0, env.Hardware.Mem.Available) + require.Equal(t, int32(runtime.NumCPU()), env.Hardware.Cpu.Numcpu) + require.NotEqual(t, 0, env.Hardware.Cpu.Sockets) + require.NotEqual(t, 0.0, env.Hardware.Cpu.Mhz) + require.NotEqual(t, 0.0, env.Os.Platform) + require.NotEmpty(t, env.Build.Tag) + require.NotEmpty(t, env.Build.Distribution) + require.NotEmpty(t, env.LicenseType) + + require.Equal(t, len(locality.Tiers), len(env.Locality.Tiers)) + for i := range locality.Tiers { + require.Equal(t, sql.HashForReporting(secret, locality.Tiers[i].Key), env.Locality.Tiers[i].Key) + require.Equal(t, sql.HashForReporting(secret, locality.Tiers[i].Value), env.Locality.Tiers[i].Value) + } +} diff --git a/pkg/server/diagnosticspb/diagnostics.go b/pkg/server/diagnosticspb/diagnostics.go index 34a3b5aefe16..37513d205e89 100644 --- a/pkg/server/diagnosticspb/diagnostics.go +++ b/pkg/server/diagnosticspb/diagnostics.go @@ -14,6 +14,8 @@ import ( "net/url" "strconv" + "github.com/cockroachdb/cockroach/pkg/build" + "github.com/cockroachdb/cockroach/pkg/roachpb" "github.com/cockroachdb/cockroach/pkg/util/envutil" "github.com/cockroachdb/cockroach/pkg/util/uuid" ) @@ -60,6 +62,7 @@ type TestingKnobs struct { // ClusterInfo contains cluster information that will become part of URLs. type ClusterInfo struct { ClusterID uuid.UUID + TenantID roachpb.TenantID IsInsecure bool IsInternal bool } @@ -71,36 +74,59 @@ func BuildUpdatesURL(clusterInfo *ClusterInfo, nodeInfo *NodeInfo, knobs *Testin if knobs != nil && knobs.OverrideUpdatesURL != nil { url = *knobs.OverrideUpdatesURL } - return addInfoToURL(url, clusterInfo, nodeInfo) + report := &DiagnosticReport{Node: *nodeInfo} + return addInfoToURL(url, clusterInfo, report) } -// BuildReportingURL creates a URL to report diagnostics. +// BuildReportingURL creates a URL to report diagnostics. If this is a CRDB +// node, then nodeInfo is filled (and nodeInfo.NodeID is non-zero). Otherwise, +// this is a SQL-only tenant and sqlInfo is filled. +// // If an empty updates URL is set (via empty environment variable), returns nil. -func BuildReportingURL(clusterInfo *ClusterInfo, nodeInfo *NodeInfo, knobs *TestingKnobs) *url.URL { +func BuildReportingURL( + clusterInfo *ClusterInfo, report *DiagnosticReport, knobs *TestingKnobs, +) *url.URL { url := reportingURL if knobs != nil && knobs.OverrideReportingURL != nil { url = *knobs.OverrideReportingURL } - return addInfoToURL(url, clusterInfo, nodeInfo) + return addInfoToURL(url, clusterInfo, report) } -func addInfoToURL(url *url.URL, clusterInfo *ClusterInfo, nodeInfo *NodeInfo) *url.URL { +func addInfoToURL(url *url.URL, clusterInfo *ClusterInfo, report *DiagnosticReport) *url.URL { if url == nil { return nil } result := *url q := result.Query() - b := &nodeInfo.Build + + // If NodeID is non-zero, then maintain backwards-compatibility by using the + // NodeInfo fields. + // TODO(andyk): Update this to always use other report fields, once they're + // guaranteed to be populated by all callers. + var b build.Info + if report.Node.NodeID != 0 { + // SQLInstanceID is always set to the NodeID for CRDB nodes. + b = report.Node.Build + q.Set("nodeid", strconv.Itoa(int(report.Node.NodeID))) + q.Set("sqlid", strconv.Itoa(int(report.Node.NodeID))) + q.Set("uptime", strconv.Itoa(int(report.Node.Uptime))) + q.Set("licensetype", report.Node.LicenseType) + } else { + b = report.Env.Build + q.Set("sqlid", strconv.Itoa(int(report.SQL.SQLInstanceID))) + q.Set("uptime", strconv.Itoa(int(report.SQL.Uptime))) + q.Set("licensetype", report.Env.LicenseType) + } + q.Set("version", b.Tag) q.Set("platform", b.Platform) q.Set("uuid", clusterInfo.ClusterID.String()) - q.Set("nodeid", strconv.Itoa(int(nodeInfo.NodeID))) - q.Set("uptime", strconv.Itoa(int(nodeInfo.Uptime))) + q.Set("tenantid", clusterInfo.TenantID.String()) q.Set("insecure", strconv.FormatBool(clusterInfo.IsInsecure)) q.Set("internal", strconv.FormatBool(clusterInfo.IsInternal)) q.Set("buildchannel", b.Channel) q.Set("envchannel", b.EnvChannel) - q.Set("licensetype", nodeInfo.LicenseType) result.RawQuery = q.Encode() return &result } diff --git a/pkg/server/diagnosticspb/diagnostics.pb.go b/pkg/server/diagnosticspb/diagnostics.pb.go index c50c3b4b82cf..f9f1383d113a 100644 --- a/pkg/server/diagnosticspb/diagnostics.pb.go +++ b/pkg/server/diagnosticspb/diagnostics.pb.go @@ -12,6 +12,7 @@ import roachpb "github.com/cockroachdb/cockroach/pkg/roachpb" import descpb "github.com/cockroachdb/cockroach/pkg/sql/catalog/descpb" import github_com_cockroachdb_cockroach_pkg_roachpb "github.com/cockroachdb/cockroach/pkg/roachpb" +import github_com_cockroachdb_cockroach_pkg_base "github.com/cockroachdb/cockroach/pkg/base" import github_com_gogo_protobuf_sortkeys "github.com/gogo/protobuf/sortkeys" import encoding_binary "encoding/binary" @@ -37,6 +38,8 @@ type DiagnosticReport struct { AlteredSettings map[string]string `protobuf:"bytes,6,rep,name=altered_settings,json=alteredSettings,proto3" json:"altered_settings,omitempty" protobuf_key:"bytes,1,opt,name=key,proto3" protobuf_val:"bytes,2,opt,name=value,proto3"` ZoneConfigs map[int64]zonepb.ZoneConfig `protobuf:"bytes,8,rep,name=zone_configs,json=zoneConfigs,proto3" json:"zone_configs" protobuf_key:"varint,1,opt,name=key,proto3" protobuf_val:"bytes,2,opt,name=value,proto3"` FeatureUsage map[string]int32 `protobuf:"bytes,9,rep,name=feature_usage,json=featureUsage,proto3" json:"feature_usage" protobuf_key:"bytes,1,opt,name=key,proto3" protobuf_val:"varint,2,opt,name=value,proto3"` + SQL SQLInstanceInfo `protobuf:"bytes,10,opt,name=sql,proto3" json:"sql"` + Env Environment `protobuf:"bytes,11,opt,name=env,proto3" json:"env"` LegacyUnimplementedErrors map[string]int64 `protobuf:"bytes,5,rep,name=legacy_unimplemented_errors,json=legacyUnimplementedErrors,proto3" json:"legacy_unimplemented_errors,omitempty" protobuf_key:"bytes,1,opt,name=key,proto3" protobuf_val:"varint,2,opt,name=value,proto3"` LegacyErrorCounts map[string]int64 `protobuf:"bytes,7,rep,name=legacy_error_counts,json=legacyErrorCounts,proto3" json:"legacy_error_counts,omitempty" protobuf_key:"bytes,1,opt,name=key,proto3" protobuf_val:"varint,2,opt,name=value,proto3"` } @@ -45,7 +48,7 @@ func (m *DiagnosticReport) Reset() { *m = DiagnosticReport{} } func (m *DiagnosticReport) String() string { return proto.CompactTextString(m) } func (*DiagnosticReport) ProtoMessage() {} func (*DiagnosticReport) Descriptor() ([]byte, []int) { - return fileDescriptor_diagnostics_5703a37aaee3f284, []int{0} + return fileDescriptor_diagnostics_06065893ddb3d5bb, []int{0} } func (m *DiagnosticReport) XXX_Unmarshal(b []byte) error { return m.Unmarshal(b) @@ -70,25 +73,29 @@ func (m *DiagnosticReport) XXX_DiscardUnknown() { var xxx_messageInfo_DiagnosticReport proto.InternalMessageInfo +// NodeInfo describes the CRDB node which is reporting diagnostics. NodeInfo +// will be set to its empty value (e.g. NodeID = 0) when the report is generated +// by a SQL-only instance. type NodeInfo struct { - NodeID github_com_cockroachdb_cockroach_pkg_roachpb.NodeID `protobuf:"varint,1,opt,name=node_id,json=nodeId,proto3,casttype=github.com/cockroachdb/cockroach/pkg/roachpb.NodeID" json:"node_id,omitempty"` - Bytes int64 `protobuf:"varint,2,opt,name=bytes,proto3" json:"bytes,omitempty"` - KeyCount int64 `protobuf:"varint,3,opt,name=key_count,json=keyCount,proto3" json:"key_count,omitempty"` - RangeCount int64 `protobuf:"varint,4,opt,name=range_count,json=rangeCount,proto3" json:"range_count,omitempty"` - Locality roachpb.Locality `protobuf:"bytes,5,opt,name=locality,proto3" json:"locality"` - Hardware HardwareInfo `protobuf:"bytes,6,opt,name=hardware,proto3" json:"hardware"` - Os OSInfo `protobuf:"bytes,7,opt,name=os,proto3" json:"os"` - Build build.Info `protobuf:"bytes,8,opt,name=build,proto3" json:"build"` - Uptime int64 `protobuf:"varint,9,opt,name=uptime,proto3" json:"uptime,omitempty"` - LicenseType string `protobuf:"bytes,10,opt,name=license_type,json=licenseType,proto3" json:"license_type,omitempty"` - Topology TopologyInfo `protobuf:"bytes,11,opt,name=topology,proto3" json:"topology"` + NodeID github_com_cockroachdb_cockroach_pkg_roachpb.NodeID `protobuf:"varint,1,opt,name=node_id,json=nodeId,proto3,casttype=github.com/cockroachdb/cockroach/pkg/roachpb.NodeID" json:"node_id,omitempty"` + Bytes int64 `protobuf:"varint,2,opt,name=bytes,proto3" json:"bytes,omitempty"` + KeyCount int64 `protobuf:"varint,3,opt,name=key_count,json=keyCount,proto3" json:"key_count,omitempty"` + RangeCount int64 `protobuf:"varint,4,opt,name=range_count,json=rangeCount,proto3" json:"range_count,omitempty"` + // Uptime is the number of seconds since the CRDB node was started. + Uptime int64 `protobuf:"varint,9,opt,name=uptime,proto3" json:"uptime,omitempty"` + Locality roachpb.Locality `protobuf:"bytes,5,opt,name=locality,proto3" json:"locality"` + Hardware HardwareInfo `protobuf:"bytes,6,opt,name=hardware,proto3" json:"hardware"` + Os OSInfo `protobuf:"bytes,7,opt,name=os,proto3" json:"os"` + Build build.Info `protobuf:"bytes,8,opt,name=build,proto3" json:"build"` + LicenseType string `protobuf:"bytes,10,opt,name=license_type,json=licenseType,proto3" json:"license_type,omitempty"` + Topology TopologyInfo `protobuf:"bytes,11,opt,name=topology,proto3" json:"topology"` } func (m *NodeInfo) Reset() { *m = NodeInfo{} } func (m *NodeInfo) String() string { return proto.CompactTextString(m) } func (*NodeInfo) ProtoMessage() {} func (*NodeInfo) Descriptor() ([]byte, []int) { - return fileDescriptor_diagnostics_5703a37aaee3f284, []int{1} + return fileDescriptor_diagnostics_06065893ddb3d5bb, []int{1} } func (m *NodeInfo) XXX_Unmarshal(b []byte) error { return m.Unmarshal(b) @@ -129,7 +136,7 @@ func (m *StoreInfo) Reset() { *m = StoreInfo{} } func (m *StoreInfo) String() string { return proto.CompactTextString(m) } func (*StoreInfo) ProtoMessage() {} func (*StoreInfo) Descriptor() ([]byte, []int) { - return fileDescriptor_diagnostics_5703a37aaee3f284, []int{2} + return fileDescriptor_diagnostics_06065893ddb3d5bb, []int{2} } func (m *StoreInfo) XXX_Unmarshal(b []byte) error { return m.Unmarshal(b) @@ -154,6 +161,94 @@ func (m *StoreInfo) XXX_DiscardUnknown() { var xxx_messageInfo_StoreInfo proto.InternalMessageInfo +// SQLInstanceInfo describes the SQL instance which is reporting diagnostics. +// This is always set, even for full CRDB nodes running both KV and SQL. +type SQLInstanceInfo struct { + // SQLInstanceID is the ephemeral identifier for the SQL instance which is + // reporting diagnostics. This is unique across all running SQL instances in + // the cluster (physical or tenant). + SQLInstanceID github_com_cockroachdb_cockroach_pkg_base.SQLInstanceID `protobuf:"varint,1,opt,name=sql_instance_id,json=sqlInstanceId,proto3,casttype=github.com/cockroachdb/cockroach/pkg/base.SQLInstanceID" json:"sql_instance_id,omitempty"` + // Uptime is the number of seconds since the SQL instance was started. + Uptime int64 `protobuf:"varint,2,opt,name=uptime,proto3" json:"uptime,omitempty"` +} + +func (m *SQLInstanceInfo) Reset() { *m = SQLInstanceInfo{} } +func (m *SQLInstanceInfo) String() string { return proto.CompactTextString(m) } +func (*SQLInstanceInfo) ProtoMessage() {} +func (*SQLInstanceInfo) Descriptor() ([]byte, []int) { + return fileDescriptor_diagnostics_06065893ddb3d5bb, []int{3} +} +func (m *SQLInstanceInfo) XXX_Unmarshal(b []byte) error { + return m.Unmarshal(b) +} +func (m *SQLInstanceInfo) XXX_Marshal(b []byte, deterministic bool) ([]byte, error) { + b = b[:cap(b)] + n, err := m.MarshalTo(b) + if err != nil { + return nil, err + } + return b[:n], nil +} +func (dst *SQLInstanceInfo) XXX_Merge(src proto.Message) { + xxx_messageInfo_SQLInstanceInfo.Merge(dst, src) +} +func (m *SQLInstanceInfo) XXX_Size() int { + return m.Size() +} +func (m *SQLInstanceInfo) XXX_DiscardUnknown() { + xxx_messageInfo_SQLInstanceInfo.DiscardUnknown(m) +} + +var xxx_messageInfo_SQLInstanceInfo proto.InternalMessageInfo + +// Environment describes the context in which the CRDB node or SQL instance is +// running. +type Environment struct { + // Locality is an ordered set of key/value tiers that describe the location of + // this CRDB node or SQL instance. + Locality roachpb.Locality `protobuf:"bytes,1,opt,name=locality,proto3" json:"locality"` + // Hardware describes the physical machine, virtual machine, or Linux + // container on which CRDB is running. + Hardware HardwareInfo `protobuf:"bytes,2,opt,name=hardware,proto3" json:"hardware"` + // OS describes the operating system software that hosts CRDB. + Os OSInfo `protobuf:"bytes,3,opt,name=os,proto3" json:"os"` + // Build describes the running CockroachDB binary. + Build build.Info `protobuf:"bytes,4,opt,name=build,proto3" json:"build"` + // LicenseType describes the license that is in effect (OSS, Enterprise, etc). + LicenseType string `protobuf:"bytes,5,opt,name=license_type,json=licenseType,proto3" json:"license_type,omitempty"` + // Topology provides the cloud provider and region name that is hosting CRDB. + Topology TopologyInfo `protobuf:"bytes,6,opt,name=topology,proto3" json:"topology"` +} + +func (m *Environment) Reset() { *m = Environment{} } +func (m *Environment) String() string { return proto.CompactTextString(m) } +func (*Environment) ProtoMessage() {} +func (*Environment) Descriptor() ([]byte, []int) { + return fileDescriptor_diagnostics_06065893ddb3d5bb, []int{4} +} +func (m *Environment) XXX_Unmarshal(b []byte) error { + return m.Unmarshal(b) +} +func (m *Environment) XXX_Marshal(b []byte, deterministic bool) ([]byte, error) { + b = b[:cap(b)] + n, err := m.MarshalTo(b) + if err != nil { + return nil, err + } + return b[:n], nil +} +func (dst *Environment) XXX_Merge(src proto.Message) { + xxx_messageInfo_Environment.Merge(dst, src) +} +func (m *Environment) XXX_Size() int { + return m.Size() +} +func (m *Environment) XXX_DiscardUnknown() { + xxx_messageInfo_Environment.DiscardUnknown(m) +} + +var xxx_messageInfo_Environment proto.InternalMessageInfo + type CPUInfo struct { Numcpu int32 `protobuf:"varint,1,opt,name=numcpu,proto3" json:"numcpu,omitempty"` Sockets int32 `protobuf:"varint,2,opt,name=sockets,proto3" json:"sockets,omitempty"` @@ -167,7 +262,7 @@ func (m *CPUInfo) Reset() { *m = CPUInfo{} } func (m *CPUInfo) String() string { return proto.CompactTextString(m) } func (*CPUInfo) ProtoMessage() {} func (*CPUInfo) Descriptor() ([]byte, []int) { - return fileDescriptor_diagnostics_5703a37aaee3f284, []int{3} + return fileDescriptor_diagnostics_06065893ddb3d5bb, []int{5} } func (m *CPUInfo) XXX_Unmarshal(b []byte) error { return m.Unmarshal(b) @@ -205,7 +300,7 @@ func (m *HardwareInfo) Reset() { *m = HardwareInfo{} } func (m *HardwareInfo) String() string { return proto.CompactTextString(m) } func (*HardwareInfo) ProtoMessage() {} func (*HardwareInfo) Descriptor() ([]byte, []int) { - return fileDescriptor_diagnostics_5703a37aaee3f284, []int{4} + return fileDescriptor_diagnostics_06065893ddb3d5bb, []int{6} } func (m *HardwareInfo) XXX_Unmarshal(b []byte) error { return m.Unmarshal(b) @@ -240,7 +335,7 @@ func (m *OSInfo) Reset() { *m = OSInfo{} } func (m *OSInfo) String() string { return proto.CompactTextString(m) } func (*OSInfo) ProtoMessage() {} func (*OSInfo) Descriptor() ([]byte, []int) { - return fileDescriptor_diagnostics_5703a37aaee3f284, []int{5} + return fileDescriptor_diagnostics_06065893ddb3d5bb, []int{7} } func (m *OSInfo) XXX_Unmarshal(b []byte) error { return m.Unmarshal(b) @@ -274,7 +369,7 @@ func (m *MemInfo) Reset() { *m = MemInfo{} } func (m *MemInfo) String() string { return proto.CompactTextString(m) } func (*MemInfo) ProtoMessage() {} func (*MemInfo) Descriptor() ([]byte, []int) { - return fileDescriptor_diagnostics_5703a37aaee3f284, []int{6} + return fileDescriptor_diagnostics_06065893ddb3d5bb, []int{8} } func (m *MemInfo) XXX_Unmarshal(b []byte) error { return m.Unmarshal(b) @@ -308,7 +403,7 @@ func (m *TopologyInfo) Reset() { *m = TopologyInfo{} } func (m *TopologyInfo) String() string { return proto.CompactTextString(m) } func (*TopologyInfo) ProtoMessage() {} func (*TopologyInfo) Descriptor() ([]byte, []int) { - return fileDescriptor_diagnostics_5703a37aaee3f284, []int{7} + return fileDescriptor_diagnostics_06065893ddb3d5bb, []int{9} } func (m *TopologyInfo) XXX_Unmarshal(b []byte) error { return m.Unmarshal(b) @@ -342,6 +437,8 @@ func init() { proto.RegisterMapType((map[int64]zonepb.ZoneConfig)(nil), "cockroach.server.diagnosticspb.DiagnosticReport.ZoneConfigsEntry") proto.RegisterType((*NodeInfo)(nil), "cockroach.server.diagnosticspb.NodeInfo") proto.RegisterType((*StoreInfo)(nil), "cockroach.server.diagnosticspb.StoreInfo") + proto.RegisterType((*SQLInstanceInfo)(nil), "cockroach.server.diagnosticspb.SQLInstanceInfo") + proto.RegisterType((*Environment)(nil), "cockroach.server.diagnosticspb.Environment") proto.RegisterType((*CPUInfo)(nil), "cockroach.server.diagnosticspb.CPUInfo") proto.RegisterType((*HardwareInfo)(nil), "cockroach.server.diagnosticspb.HardwareInfo") proto.RegisterType((*OSInfo)(nil), "cockroach.server.diagnosticspb.OSInfo") @@ -522,6 +619,22 @@ func (m *DiagnosticReport) MarshalTo(dAtA []byte) (int, error) { i = encodeVarintDiagnostics(dAtA, i, uint64(v)) } } + dAtA[i] = 0x52 + i++ + i = encodeVarintDiagnostics(dAtA, i, uint64(m.SQL.Size())) + n3, err := m.SQL.MarshalTo(dAtA[i:]) + if err != nil { + return 0, err + } + i += n3 + dAtA[i] = 0x5a + i++ + i = encodeVarintDiagnostics(dAtA, i, uint64(m.Env.Size())) + n4, err := m.Env.MarshalTo(dAtA[i:]) + if err != nil { + return 0, err + } + i += n4 return i, nil } @@ -563,35 +676,35 @@ func (m *NodeInfo) MarshalTo(dAtA []byte) (int, error) { dAtA[i] = 0x2a i++ i = encodeVarintDiagnostics(dAtA, i, uint64(m.Locality.Size())) - n3, err := m.Locality.MarshalTo(dAtA[i:]) + n5, err := m.Locality.MarshalTo(dAtA[i:]) if err != nil { return 0, err } - i += n3 + i += n5 dAtA[i] = 0x32 i++ i = encodeVarintDiagnostics(dAtA, i, uint64(m.Hardware.Size())) - n4, err := m.Hardware.MarshalTo(dAtA[i:]) + n6, err := m.Hardware.MarshalTo(dAtA[i:]) if err != nil { return 0, err } - i += n4 + i += n6 dAtA[i] = 0x3a i++ i = encodeVarintDiagnostics(dAtA, i, uint64(m.Os.Size())) - n5, err := m.Os.MarshalTo(dAtA[i:]) + n7, err := m.Os.MarshalTo(dAtA[i:]) if err != nil { return 0, err } - i += n5 + i += n7 dAtA[i] = 0x42 i++ i = encodeVarintDiagnostics(dAtA, i, uint64(m.Build.Size())) - n6, err := m.Build.MarshalTo(dAtA[i:]) + n8, err := m.Build.MarshalTo(dAtA[i:]) if err != nil { return 0, err } - i += n6 + i += n8 if m.Uptime != 0 { dAtA[i] = 0x48 i++ @@ -606,11 +719,11 @@ func (m *NodeInfo) MarshalTo(dAtA []byte) (int, error) { dAtA[i] = 0x5a i++ i = encodeVarintDiagnostics(dAtA, i, uint64(m.Topology.Size())) - n7, err := m.Topology.MarshalTo(dAtA[i:]) + n9, err := m.Topology.MarshalTo(dAtA[i:]) if err != nil { return 0, err } - i += n7 + i += n9 return i, nil } @@ -677,6 +790,98 @@ func (m *StoreInfo) MarshalTo(dAtA []byte) (int, error) { return i, nil } +func (m *SQLInstanceInfo) Marshal() (dAtA []byte, err error) { + size := m.Size() + dAtA = make([]byte, size) + n, err := m.MarshalTo(dAtA) + if err != nil { + return nil, err + } + return dAtA[:n], nil +} + +func (m *SQLInstanceInfo) MarshalTo(dAtA []byte) (int, error) { + var i int + _ = i + var l int + _ = l + if m.SQLInstanceID != 0 { + dAtA[i] = 0x8 + i++ + i = encodeVarintDiagnostics(dAtA, i, uint64(m.SQLInstanceID)) + } + if m.Uptime != 0 { + dAtA[i] = 0x10 + i++ + i = encodeVarintDiagnostics(dAtA, i, uint64(m.Uptime)) + } + return i, nil +} + +func (m *Environment) Marshal() (dAtA []byte, err error) { + size := m.Size() + dAtA = make([]byte, size) + n, err := m.MarshalTo(dAtA) + if err != nil { + return nil, err + } + return dAtA[:n], nil +} + +func (m *Environment) MarshalTo(dAtA []byte) (int, error) { + var i int + _ = i + var l int + _ = l + dAtA[i] = 0xa + i++ + i = encodeVarintDiagnostics(dAtA, i, uint64(m.Locality.Size())) + n10, err := m.Locality.MarshalTo(dAtA[i:]) + if err != nil { + return 0, err + } + i += n10 + dAtA[i] = 0x12 + i++ + i = encodeVarintDiagnostics(dAtA, i, uint64(m.Hardware.Size())) + n11, err := m.Hardware.MarshalTo(dAtA[i:]) + if err != nil { + return 0, err + } + i += n11 + dAtA[i] = 0x1a + i++ + i = encodeVarintDiagnostics(dAtA, i, uint64(m.Os.Size())) + n12, err := m.Os.MarshalTo(dAtA[i:]) + if err != nil { + return 0, err + } + i += n12 + dAtA[i] = 0x22 + i++ + i = encodeVarintDiagnostics(dAtA, i, uint64(m.Build.Size())) + n13, err := m.Build.MarshalTo(dAtA[i:]) + if err != nil { + return 0, err + } + i += n13 + if len(m.LicenseType) > 0 { + dAtA[i] = 0x2a + i++ + i = encodeVarintDiagnostics(dAtA, i, uint64(len(m.LicenseType))) + i += copy(dAtA[i:], m.LicenseType) + } + dAtA[i] = 0x32 + i++ + i = encodeVarintDiagnostics(dAtA, i, uint64(m.Topology.Size())) + n14, err := m.Topology.MarshalTo(dAtA[i:]) + if err != nil { + return 0, err + } + i += n14 + return i, nil +} + func (m *CPUInfo) Marshal() (dAtA []byte, err error) { size := m.Size() dAtA = make([]byte, size) @@ -761,19 +966,19 @@ func (m *HardwareInfo) MarshalTo(dAtA []byte) (int, error) { dAtA[i] = 0x12 i++ i = encodeVarintDiagnostics(dAtA, i, uint64(m.Cpu.Size())) - n8, err := m.Cpu.MarshalTo(dAtA[i:]) + n15, err := m.Cpu.MarshalTo(dAtA[i:]) if err != nil { return 0, err } - i += n8 + i += n15 dAtA[i] = 0x1a i++ i = encodeVarintDiagnostics(dAtA, i, uint64(m.Mem.Size())) - n9, err := m.Mem.MarshalTo(dAtA[i:]) + n16, err := m.Mem.MarshalTo(dAtA[i:]) if err != nil { return 0, err } - i += n9 + i += n16 if m.Loadavg15 != 0 { dAtA[i] = 0x25 i++ @@ -965,6 +1170,10 @@ func (m *DiagnosticReport) Size() (n int) { n += mapEntrySize + 1 + sovDiagnostics(uint64(mapEntrySize)) } } + l = m.SQL.Size() + n += 1 + l + sovDiagnostics(uint64(l)) + l = m.Env.Size() + n += 1 + l + sovDiagnostics(uint64(l)) return n } @@ -1042,6 +1251,44 @@ func (m *StoreInfo) Size() (n int) { return n } +func (m *SQLInstanceInfo) Size() (n int) { + if m == nil { + return 0 + } + var l int + _ = l + if m.SQLInstanceID != 0 { + n += 1 + sovDiagnostics(uint64(m.SQLInstanceID)) + } + if m.Uptime != 0 { + n += 1 + sovDiagnostics(uint64(m.Uptime)) + } + return n +} + +func (m *Environment) Size() (n int) { + if m == nil { + return 0 + } + var l int + _ = l + l = m.Locality.Size() + n += 1 + l + sovDiagnostics(uint64(l)) + l = m.Hardware.Size() + n += 1 + l + sovDiagnostics(uint64(l)) + l = m.Os.Size() + n += 1 + l + sovDiagnostics(uint64(l)) + l = m.Build.Size() + n += 1 + l + sovDiagnostics(uint64(l)) + l = len(m.LicenseType) + if l > 0 { + n += 1 + l + sovDiagnostics(uint64(l)) + } + l = m.Topology.Size() + n += 1 + l + sovDiagnostics(uint64(l)) + return n +} + func (m *CPUInfo) Size() (n int) { if m == nil { return 0 @@ -1870,6 +2117,66 @@ func (m *DiagnosticReport) Unmarshal(dAtA []byte) error { } m.FeatureUsage[mapkey] = mapvalue iNdEx = postIndex + case 10: + if wireType != 2 { + return fmt.Errorf("proto: wrong wireType = %d for field SQL", wireType) + } + var msglen int + for shift := uint(0); ; shift += 7 { + if shift >= 64 { + return ErrIntOverflowDiagnostics + } + if iNdEx >= l { + return io.ErrUnexpectedEOF + } + b := dAtA[iNdEx] + iNdEx++ + msglen |= (int(b) & 0x7F) << shift + if b < 0x80 { + break + } + } + if msglen < 0 { + return ErrInvalidLengthDiagnostics + } + postIndex := iNdEx + msglen + if postIndex > l { + return io.ErrUnexpectedEOF + } + if err := m.SQL.Unmarshal(dAtA[iNdEx:postIndex]); err != nil { + return err + } + iNdEx = postIndex + case 11: + if wireType != 2 { + return fmt.Errorf("proto: wrong wireType = %d for field Env", wireType) + } + var msglen int + for shift := uint(0); ; shift += 7 { + if shift >= 64 { + return ErrIntOverflowDiagnostics + } + if iNdEx >= l { + return io.ErrUnexpectedEOF + } + b := dAtA[iNdEx] + iNdEx++ + msglen |= (int(b) & 0x7F) << shift + if b < 0x80 { + break + } + } + if msglen < 0 { + return ErrInvalidLengthDiagnostics + } + postIndex := iNdEx + msglen + if postIndex > l { + return io.ErrUnexpectedEOF + } + if err := m.Env.Unmarshal(dAtA[iNdEx:postIndex]); err != nil { + return err + } + iNdEx = postIndex default: iNdEx = preIndex skippy, err := skipDiagnostics(dAtA[iNdEx:]) @@ -2436,6 +2743,323 @@ func (m *StoreInfo) Unmarshal(dAtA []byte) error { } return nil } +func (m *SQLInstanceInfo) Unmarshal(dAtA []byte) error { + l := len(dAtA) + iNdEx := 0 + for iNdEx < l { + preIndex := iNdEx + var wire uint64 + for shift := uint(0); ; shift += 7 { + if shift >= 64 { + return ErrIntOverflowDiagnostics + } + if iNdEx >= l { + return io.ErrUnexpectedEOF + } + b := dAtA[iNdEx] + iNdEx++ + wire |= (uint64(b) & 0x7F) << shift + if b < 0x80 { + break + } + } + fieldNum := int32(wire >> 3) + wireType := int(wire & 0x7) + if wireType == 4 { + return fmt.Errorf("proto: SQLInstanceInfo: wiretype end group for non-group") + } + if fieldNum <= 0 { + return fmt.Errorf("proto: SQLInstanceInfo: illegal tag %d (wire type %d)", fieldNum, wire) + } + switch fieldNum { + case 1: + if wireType != 0 { + return fmt.Errorf("proto: wrong wireType = %d for field SQLInstanceID", wireType) + } + m.SQLInstanceID = 0 + for shift := uint(0); ; shift += 7 { + if shift >= 64 { + return ErrIntOverflowDiagnostics + } + if iNdEx >= l { + return io.ErrUnexpectedEOF + } + b := dAtA[iNdEx] + iNdEx++ + m.SQLInstanceID |= (github_com_cockroachdb_cockroach_pkg_base.SQLInstanceID(b) & 0x7F) << shift + if b < 0x80 { + break + } + } + case 2: + if wireType != 0 { + return fmt.Errorf("proto: wrong wireType = %d for field Uptime", wireType) + } + m.Uptime = 0 + for shift := uint(0); ; shift += 7 { + if shift >= 64 { + return ErrIntOverflowDiagnostics + } + if iNdEx >= l { + return io.ErrUnexpectedEOF + } + b := dAtA[iNdEx] + iNdEx++ + m.Uptime |= (int64(b) & 0x7F) << shift + if b < 0x80 { + break + } + } + default: + iNdEx = preIndex + skippy, err := skipDiagnostics(dAtA[iNdEx:]) + if err != nil { + return err + } + if skippy < 0 { + return ErrInvalidLengthDiagnostics + } + if (iNdEx + skippy) > l { + return io.ErrUnexpectedEOF + } + iNdEx += skippy + } + } + + if iNdEx > l { + return io.ErrUnexpectedEOF + } + return nil +} +func (m *Environment) Unmarshal(dAtA []byte) error { + l := len(dAtA) + iNdEx := 0 + for iNdEx < l { + preIndex := iNdEx + var wire uint64 + for shift := uint(0); ; shift += 7 { + if shift >= 64 { + return ErrIntOverflowDiagnostics + } + if iNdEx >= l { + return io.ErrUnexpectedEOF + } + b := dAtA[iNdEx] + iNdEx++ + wire |= (uint64(b) & 0x7F) << shift + if b < 0x80 { + break + } + } + fieldNum := int32(wire >> 3) + wireType := int(wire & 0x7) + if wireType == 4 { + return fmt.Errorf("proto: Environment: wiretype end group for non-group") + } + if fieldNum <= 0 { + return fmt.Errorf("proto: Environment: illegal tag %d (wire type %d)", fieldNum, wire) + } + switch fieldNum { + case 1: + if wireType != 2 { + return fmt.Errorf("proto: wrong wireType = %d for field Locality", wireType) + } + var msglen int + for shift := uint(0); ; shift += 7 { + if shift >= 64 { + return ErrIntOverflowDiagnostics + } + if iNdEx >= l { + return io.ErrUnexpectedEOF + } + b := dAtA[iNdEx] + iNdEx++ + msglen |= (int(b) & 0x7F) << shift + if b < 0x80 { + break + } + } + if msglen < 0 { + return ErrInvalidLengthDiagnostics + } + postIndex := iNdEx + msglen + if postIndex > l { + return io.ErrUnexpectedEOF + } + if err := m.Locality.Unmarshal(dAtA[iNdEx:postIndex]); err != nil { + return err + } + iNdEx = postIndex + case 2: + if wireType != 2 { + return fmt.Errorf("proto: wrong wireType = %d for field Hardware", wireType) + } + var msglen int + for shift := uint(0); ; shift += 7 { + if shift >= 64 { + return ErrIntOverflowDiagnostics + } + if iNdEx >= l { + return io.ErrUnexpectedEOF + } + b := dAtA[iNdEx] + iNdEx++ + msglen |= (int(b) & 0x7F) << shift + if b < 0x80 { + break + } + } + if msglen < 0 { + return ErrInvalidLengthDiagnostics + } + postIndex := iNdEx + msglen + if postIndex > l { + return io.ErrUnexpectedEOF + } + if err := m.Hardware.Unmarshal(dAtA[iNdEx:postIndex]); err != nil { + return err + } + iNdEx = postIndex + case 3: + if wireType != 2 { + return fmt.Errorf("proto: wrong wireType = %d for field Os", wireType) + } + var msglen int + for shift := uint(0); ; shift += 7 { + if shift >= 64 { + return ErrIntOverflowDiagnostics + } + if iNdEx >= l { + return io.ErrUnexpectedEOF + } + b := dAtA[iNdEx] + iNdEx++ + msglen |= (int(b) & 0x7F) << shift + if b < 0x80 { + break + } + } + if msglen < 0 { + return ErrInvalidLengthDiagnostics + } + postIndex := iNdEx + msglen + if postIndex > l { + return io.ErrUnexpectedEOF + } + if err := m.Os.Unmarshal(dAtA[iNdEx:postIndex]); err != nil { + return err + } + iNdEx = postIndex + case 4: + if wireType != 2 { + return fmt.Errorf("proto: wrong wireType = %d for field Build", wireType) + } + var msglen int + for shift := uint(0); ; shift += 7 { + if shift >= 64 { + return ErrIntOverflowDiagnostics + } + if iNdEx >= l { + return io.ErrUnexpectedEOF + } + b := dAtA[iNdEx] + iNdEx++ + msglen |= (int(b) & 0x7F) << shift + if b < 0x80 { + break + } + } + if msglen < 0 { + return ErrInvalidLengthDiagnostics + } + postIndex := iNdEx + msglen + if postIndex > l { + return io.ErrUnexpectedEOF + } + if err := m.Build.Unmarshal(dAtA[iNdEx:postIndex]); err != nil { + return err + } + iNdEx = postIndex + case 5: + if wireType != 2 { + return fmt.Errorf("proto: wrong wireType = %d for field LicenseType", wireType) + } + var stringLen uint64 + for shift := uint(0); ; shift += 7 { + if shift >= 64 { + return ErrIntOverflowDiagnostics + } + if iNdEx >= l { + return io.ErrUnexpectedEOF + } + b := dAtA[iNdEx] + iNdEx++ + stringLen |= (uint64(b) & 0x7F) << shift + if b < 0x80 { + break + } + } + intStringLen := int(stringLen) + if intStringLen < 0 { + return ErrInvalidLengthDiagnostics + } + postIndex := iNdEx + intStringLen + if postIndex > l { + return io.ErrUnexpectedEOF + } + m.LicenseType = string(dAtA[iNdEx:postIndex]) + iNdEx = postIndex + case 6: + if wireType != 2 { + return fmt.Errorf("proto: wrong wireType = %d for field Topology", wireType) + } + var msglen int + for shift := uint(0); ; shift += 7 { + if shift >= 64 { + return ErrIntOverflowDiagnostics + } + if iNdEx >= l { + return io.ErrUnexpectedEOF + } + b := dAtA[iNdEx] + iNdEx++ + msglen |= (int(b) & 0x7F) << shift + if b < 0x80 { + break + } + } + if msglen < 0 { + return ErrInvalidLengthDiagnostics + } + postIndex := iNdEx + msglen + if postIndex > l { + return io.ErrUnexpectedEOF + } + if err := m.Topology.Unmarshal(dAtA[iNdEx:postIndex]); err != nil { + return err + } + iNdEx = postIndex + default: + iNdEx = preIndex + skippy, err := skipDiagnostics(dAtA[iNdEx:]) + if err != nil { + return err + } + if skippy < 0 { + return ErrInvalidLengthDiagnostics + } + if (iNdEx + skippy) > l { + return io.ErrUnexpectedEOF + } + iNdEx += skippy + } + } + + if iNdEx > l { + return io.ErrUnexpectedEOF + } + return nil +} func (m *CPUInfo) Unmarshal(dAtA []byte) error { l := len(dAtA) iNdEx := 0 @@ -3259,89 +3883,97 @@ var ( ) func init() { - proto.RegisterFile("server/diagnosticspb/diagnostics.proto", fileDescriptor_diagnostics_5703a37aaee3f284) -} - -var fileDescriptor_diagnostics_5703a37aaee3f284 = []byte{ - // 1271 bytes of a gzipped FileDescriptorProto - 0x1f, 0x8b, 0x08, 0x00, 0x00, 0x00, 0x00, 0x00, 0x02, 0xff, 0xb4, 0x56, 0xcf, 0x6e, 0x1b, 0x37, - 0x13, 0xb7, 0xb4, 0xfa, 0x4b, 0x39, 0xf9, 0x1c, 0xc6, 0x9f, 0xbb, 0x75, 0x0a, 0x39, 0x55, 0xd0, - 0x34, 0x45, 0x0b, 0x09, 0x49, 0x5a, 0xa0, 0x2d, 0x1a, 0x04, 0x96, 0x9d, 0xa6, 0x01, 0xd2, 0xa4, - 0x58, 0x27, 0x3d, 0xe4, 0x50, 0x81, 0xe2, 0x52, 0xeb, 0xad, 0xb9, 0xcb, 0x35, 0xc9, 0x55, 0x20, - 0xbf, 0x42, 0x0f, 0xed, 0xa5, 0x87, 0x02, 0x7d, 0x9d, 0x02, 0x39, 0xe6, 0x98, 0x93, 0xd1, 0x3a, - 0x6f, 0xd1, 0x53, 0xc1, 0x21, 0x25, 0xaf, 0x95, 0xc4, 0x8a, 0x0b, 0xf4, 0x20, 0x88, 0x33, 0xc3, - 0xf9, 0xcd, 0xfc, 0x86, 0xc3, 0xe1, 0xa2, 0xab, 0x8a, 0xc9, 0x31, 0x93, 0xbd, 0x30, 0x26, 0x51, - 0x2a, 0x94, 0x8e, 0xa9, 0xca, 0x86, 0x45, 0xa9, 0x9b, 0x49, 0xa1, 0x05, 0x6e, 0x53, 0x41, 0xf7, - 0xa4, 0x20, 0x74, 0xb7, 0x6b, 0x3d, 0xba, 0x27, 0x3c, 0xd6, 0x57, 0x86, 0x79, 0xcc, 0xc3, 0x5e, - 0x9c, 0x8e, 0x84, 0xf5, 0x58, 0xf7, 0xa9, 0x48, 0x47, 0x71, 0xd4, 0x3b, 0x10, 0x29, 0xcb, 0x86, - 0xf0, 0xe7, 0x2c, 0xab, 0x91, 0x88, 0x04, 0x2c, 0x7b, 0x66, 0xe5, 0xb4, 0xef, 0x00, 0x7a, 0x36, - 0xec, 0x91, 0x2c, 0x1b, 0x28, 0x4d, 0xb4, 0x0b, 0xbd, 0x7e, 0x45, 0xed, 0xf3, 0x1e, 0x25, 0x9a, - 0x70, 0x11, 0xf5, 0x42, 0xa6, 0x68, 0x36, 0xec, 0x29, 0x2d, 0x73, 0xaa, 0x73, 0xc9, 0x42, 0xb7, - 0x69, 0x6d, 0xea, 0x9d, 0x30, 0x4d, 0x42, 0xa2, 0x89, 0xd5, 0x77, 0x7e, 0x42, 0x68, 0x65, 0x7b, - 0x96, 0x69, 0xc0, 0x32, 0x21, 0x35, 0xee, 0xa3, 0x4a, 0x2a, 0x42, 0xe6, 0x97, 0x2e, 0x97, 0xae, - 0xb5, 0x6e, 0x5c, 0xeb, 0x9e, 0xce, 0xad, 0xfb, 0x40, 0x84, 0xec, 0x5e, 0x3a, 0x12, 0xfd, 0xca, - 0xb3, 0xc3, 0x8d, 0xa5, 0x00, 0x7c, 0xf1, 0x5d, 0x54, 0x53, 0x5a, 0x48, 0xa6, 0xfc, 0xf2, 0x65, - 0xef, 0x5a, 0xeb, 0xc6, 0x47, 0x8b, 0x50, 0x76, 0xcc, 0xee, 0x02, 0x8c, 0x73, 0xc7, 0xdb, 0xa8, - 0xa6, 0xe8, 0x2e, 0x4b, 0x88, 0xef, 0x01, 0xd0, 0xd5, 0x22, 0xd0, 0x3e, 0x37, 0xbf, 0x21, 0x51, - 0xac, 0xfb, 0x88, 0x0c, 0x39, 0xdb, 0x66, 0x8a, 0xca, 0x38, 0xd3, 0x42, 0xce, 0x50, 0xc0, 0x17, - 0x3f, 0x40, 0x4d, 0xb5, 0xcf, 0x6d, 0xdd, 0xfc, 0x0a, 0x00, 0x7d, 0x3c, 0x07, 0xb4, 0x25, 0x38, - 0x67, 0x54, 0xb3, 0x70, 0x47, 0x13, 0xcd, 0x12, 0x96, 0x6a, 0xb3, 0x88, 0x21, 0x3f, 0x87, 0xd6, - 0x50, 0xfb, 0xdc, 0x28, 0x15, 0xfe, 0xb9, 0x84, 0x2e, 0x71, 0x16, 0x11, 0x3a, 0x19, 0xe4, 0x69, - 0x9c, 0x64, 0x1c, 0x1c, 0x58, 0x38, 0x60, 0x52, 0x0a, 0xa9, 0xfc, 0x2a, 0x84, 0x78, 0xb8, 0x88, - 0xf4, 0x7c, 0xe9, 0xbb, 0xf7, 0x01, 0xf3, 0x71, 0x11, 0xf2, 0x0e, 0x20, 0xde, 0x49, 0xb5, 0x9c, - 0x04, 0xef, 0xf2, 0x37, 0xd9, 0x71, 0x86, 0x56, 0x08, 0xd7, 0x4c, 0xb2, 0x70, 0xa0, 0x98, 0xd6, - 0x71, 0x1a, 0x29, 0xbf, 0x06, 0x59, 0xdc, 0x39, 0x73, 0x16, 0x9b, 0x16, 0x68, 0xc7, 0xe1, 0xd8, - 0xd8, 0xff, 0x23, 0x27, 0xb5, 0xf8, 0x29, 0xba, 0xe8, 0x4a, 0x00, 0xa4, 0x07, 0x54, 0xe4, 0xa9, - 0x56, 0x7e, 0x1d, 0x82, 0xde, 0xfd, 0x97, 0xd4, 0x81, 0xcd, 0x16, 0x20, 0xd9, 0xb0, 0x17, 0xf8, - 0xbc, 0x1e, 0xff, 0x88, 0x96, 0xcd, 0x75, 0x19, 0xd8, 0x1b, 0xa4, 0xfc, 0x06, 0x44, 0xdc, 0x3c, - 0x73, 0xc4, 0x27, 0x22, 0x65, 0x5b, 0x16, 0x03, 0x62, 0xb9, 0x53, 0x6e, 0x1d, 0x1c, 0xeb, 0x71, - 0x82, 0xce, 0x8d, 0x18, 0x31, 0x57, 0x69, 0x90, 0x2b, 0x12, 0x31, 0xbf, 0x09, 0xc1, 0xfa, 0x67, - 0x0e, 0xf6, 0xb5, 0x45, 0x79, 0x6c, 0x40, 0x8a, 0xd1, 0x96, 0x47, 0x05, 0xc3, 0x7a, 0x1f, 0xad, - 0xbe, 0xae, 0xf8, 0x78, 0x05, 0x79, 0x7b, 0x6c, 0x02, 0x37, 0xb2, 0x19, 0x98, 0x25, 0x5e, 0x45, - 0xd5, 0x31, 0xe1, 0x39, 0xf3, 0xcb, 0xa0, 0xb3, 0xc2, 0x97, 0xe5, 0xcf, 0x4b, 0xeb, 0x14, 0xad, - 0xcc, 0x33, 0x2b, 0xfa, 0x7b, 0xd6, 0xff, 0x8b, 0xa2, 0x7f, 0xeb, 0xc6, 0x95, 0x02, 0x21, 0x5b, - 0xd7, 0xae, 0x9d, 0x4c, 0x85, 0x2a, 0x15, 0x83, 0xdc, 0x46, 0x17, 0x5e, 0x61, 0xb4, 0x28, 0xcb, - 0x6a, 0x11, 0xe0, 0x3e, 0x6a, 0x9f, 0xde, 0xec, 0x8b, 0xd0, 0xbc, 0x22, 0xda, 0x36, 0x5a, 0x7b, - 0x7d, 0xff, 0x9c, 0x05, 0xa5, 0xf3, 0x47, 0x05, 0x35, 0xa6, 0xd3, 0x0c, 0x3f, 0x41, 0x75, 0x33, - 0xc9, 0x06, 0x71, 0x08, 0xce, 0xd5, 0xfe, 0xe6, 0xd1, 0xe1, 0x46, 0x0d, 0xcc, 0xdb, 0x7f, 0x1f, - 0x6e, 0xdc, 0x8c, 0x62, 0xbd, 0x9b, 0x0f, 0xbb, 0x54, 0x24, 0xbd, 0x59, 0xe9, 0xc2, 0xe1, 0xf1, - 0xba, 0x97, 0xed, 0x45, 0x3d, 0x37, 0x76, 0xed, 0x8c, 0xdc, 0x0e, 0x6a, 0x06, 0xf1, 0x5e, 0x68, - 0x52, 0x18, 0x4e, 0x34, 0x0c, 0x47, 0x48, 0x01, 0x04, 0x7c, 0x09, 0x35, 0xf7, 0xd8, 0xc4, 0xde, - 0x23, 0xdf, 0x03, 0x4b, 0x63, 0x8f, 0x4d, 0x80, 0x0d, 0xde, 0x40, 0x2d, 0x49, 0xd2, 0x88, 0x39, - 0x73, 0x05, 0xcc, 0x08, 0x54, 0x76, 0xc3, 0x2d, 0xd4, 0xe0, 0x82, 0x12, 0x1e, 0xeb, 0x89, 0x5f, - 0x85, 0x33, 0xbd, 0x54, 0x38, 0xd3, 0x69, 0x22, 0xf7, 0xdd, 0x96, 0xe9, 0x44, 0x9b, 0xba, 0xe0, - 0x07, 0xa8, 0xb1, 0x4b, 0x64, 0xf8, 0x94, 0x48, 0xe6, 0xd7, 0xc0, 0xfd, 0x93, 0x45, 0x3d, 0xfe, - 0x8d, 0xdb, 0x5f, 0x98, 0xda, 0x33, 0x0c, 0xfc, 0x15, 0x2a, 0x0b, 0x33, 0x0c, 0x4a, 0xf3, 0x33, - 0xfb, 0x75, 0x48, 0x0f, 0x77, 0x0a, 0x18, 0x65, 0xa1, 0xf0, 0x75, 0x54, 0x85, 0x17, 0xd3, 0x6f, - 0x00, 0xc0, 0xff, 0x0b, 0x00, 0xa0, 0xef, 0x16, 0xf6, 0xdb, 0x9d, 0x78, 0x0d, 0xd5, 0xf2, 0x4c, - 0xc7, 0x89, 0xb9, 0xa2, 0xa6, 0x36, 0x4e, 0xc2, 0xef, 0xa3, 0x65, 0x1e, 0x53, 0x96, 0x2a, 0x36, - 0xd0, 0x93, 0x8c, 0xf9, 0x08, 0x3a, 0xa1, 0xe5, 0x74, 0x8f, 0x26, 0x19, 0x33, 0xdc, 0xb5, 0xc8, - 0x04, 0x17, 0xd1, 0xc4, 0x6f, 0xbd, 0x1d, 0xf7, 0x47, 0x6e, 0x7f, 0x91, 0xfb, 0x14, 0xa3, 0xf3, - 0xbb, 0x87, 0x9a, 0xb3, 0xf7, 0xec, 0x3f, 0x6d, 0xa4, 0x1f, 0x50, 0x03, 0xde, 0x49, 0x03, 0x0e, - 0x57, 0xac, 0xbf, 0x75, 0x74, 0xb8, 0x51, 0xb7, 0xc1, 0x0d, 0xfa, 0xa7, 0x67, 0x42, 0x77, 0x7e, - 0x41, 0x1d, 0x40, 0x8b, 0x8d, 0xea, 0xbd, 0xb1, 0x51, 0x2b, 0xa7, 0x37, 0x6a, 0xf5, 0x95, 0x46, - 0x5d, 0x47, 0x0d, 0x4a, 0x32, 0x42, 0x4d, 0xa3, 0xd6, 0xac, 0xf3, 0x54, 0xc6, 0xef, 0xa1, 0x26, - 0x19, 0x93, 0x98, 0x9b, 0xc7, 0x1c, 0x9a, 0xc7, 0x0b, 0x8e, 0x15, 0x18, 0xa3, 0x4a, 0xae, 0x98, - 0x6d, 0x0a, 0x2f, 0x80, 0x35, 0xbe, 0x8e, 0x56, 0x59, 0x4a, 0xe5, 0x24, 0xd3, 0xb1, 0x48, 0x07, - 0x84, 0x47, 0x42, 0xc6, 0x7a, 0x37, 0x71, 0x4d, 0x70, 0xf1, 0xd8, 0xb6, 0x39, 0x35, 0x75, 0x7e, - 0x2d, 0xa1, 0xfa, 0xd6, 0x77, 0x8f, 0xe1, 0x70, 0xd6, 0x50, 0x2d, 0xcd, 0x13, 0x9a, 0xe5, 0xf6, - 0x6c, 0x02, 0x27, 0x61, 0x1f, 0xd5, 0x95, 0xa0, 0x7b, 0x4c, 0x2b, 0x37, 0xba, 0xa6, 0xa2, 0x29, - 0x09, 0x85, 0x0f, 0x1b, 0xcf, 0x8e, 0x34, 0x10, 0x8c, 0x36, 0x11, 0x21, 0xe3, 0x50, 0x8e, 0x66, - 0x60, 0x05, 0x33, 0x7c, 0x92, 0xdd, 0x03, 0xa8, 0x41, 0x39, 0x30, 0x4b, 0x43, 0xde, 0x0d, 0x7c, - 0xfb, 0x3c, 0x37, 0x83, 0x99, 0xdc, 0xf9, 0xad, 0x8c, 0x96, 0x8b, 0x77, 0x0a, 0x5f, 0x45, 0xe7, - 0xc7, 0xb1, 0xd4, 0x39, 0xe1, 0xf1, 0x01, 0x31, 0x1c, 0xdc, 0x18, 0x9b, 0xd3, 0xe2, 0xdb, 0xc8, - 0x33, 0x0c, 0xec, 0x24, 0xff, 0x70, 0x51, 0xeb, 0x3a, 0xea, 0xae, 0x6b, 0x8d, 0xa7, 0x01, 0x48, - 0x58, 0x02, 0x8c, 0xde, 0x02, 0xe0, 0x5b, 0x96, 0x14, 0x01, 0x12, 0x96, 0x98, 0x73, 0xe3, 0x82, - 0x84, 0x64, 0x1c, 0x5d, 0xff, 0x0c, 0x4a, 0x50, 0x0e, 0x8e, 0x15, 0x86, 0x74, 0x26, 0xc5, 0x38, - 0x0e, 0x99, 0x84, 0x5a, 0x34, 0x83, 0x99, 0x8c, 0x3f, 0x40, 0xe7, 0xe3, 0x54, 0x69, 0x92, 0x52, - 0x36, 0xa0, 0x9c, 0x28, 0x05, 0x3d, 0xd1, 0x0c, 0xce, 0x4d, 0xb5, 0x5b, 0x46, 0xd9, 0xf9, 0x1e, - 0xd5, 0xec, 0x90, 0x30, 0x27, 0x36, 0x22, 0x49, 0xcc, 0xa7, 0x33, 0xdd, 0x49, 0x10, 0x84, 0x13, - 0x3d, 0x12, 0x32, 0x71, 0x6f, 0xe2, 0x4c, 0x36, 0xa7, 0x39, 0x66, 0x52, 0x99, 0x0a, 0x7a, 0x60, - 0x9a, 0x8a, 0x9d, 0x5b, 0xa8, 0xee, 0xe8, 0x98, 0x23, 0xd4, 0x42, 0x13, 0x0e, 0xb8, 0x95, 0xc0, - 0x0a, 0x27, 0x3b, 0xb2, 0x0c, 0x96, 0x63, 0x45, 0xa7, 0x8f, 0x96, 0x8b, 0x93, 0xe0, 0x04, 0xd3, - 0xd2, 0x1c, 0xd3, 0x35, 0x54, 0x93, 0x2c, 0x32, 0x39, 0xd8, 0xf4, 0x9c, 0xd4, 0xef, 0x3d, 0xfb, - 0xab, 0xbd, 0xf4, 0xec, 0xa8, 0x5d, 0x7a, 0x7e, 0xd4, 0x2e, 0xbd, 0x38, 0x6a, 0x97, 0xfe, 0x3c, - 0x6a, 0x97, 0x7e, 0x79, 0xd9, 0x5e, 0x7a, 0xfe, 0xb2, 0xbd, 0xf4, 0xe2, 0x65, 0x7b, 0xe9, 0xc9, - 0xb9, 0x13, 0xf5, 0x1f, 0xd6, 0xe0, 0xdb, 0xfd, 0xe6, 0x3f, 0x01, 0x00, 0x00, 0xff, 0xff, 0x04, - 0xe9, 0x8f, 0xba, 0x9d, 0x0c, 0x00, 0x00, + proto.RegisterFile("server/diagnosticspb/diagnostics.proto", fileDescriptor_diagnostics_06065893ddb3d5bb) +} + +var fileDescriptor_diagnostics_06065893ddb3d5bb = []byte{ + // 1408 bytes of a gzipped FileDescriptorProto + 0x1f, 0x8b, 0x08, 0x00, 0x00, 0x00, 0x00, 0x00, 0x02, 0xff, 0xb4, 0x57, 0x4f, 0x6f, 0x1b, 0x45, + 0x14, 0x8f, 0xbd, 0xfe, 0x3b, 0x4e, 0xda, 0x74, 0x1a, 0xc2, 0x92, 0x22, 0xbb, 0xb8, 0xa2, 0x14, + 0x15, 0xd9, 0x6a, 0x0b, 0xe2, 0x8f, 0xa8, 0xaa, 0xd8, 0x09, 0xa5, 0x52, 0x48, 0xe9, 0xa6, 0xe5, + 0xd0, 0x03, 0xd6, 0x78, 0x77, 0xbc, 0x59, 0x32, 0xbb, 0xb3, 0xd9, 0x19, 0xbb, 0x72, 0x4e, 0x7c, + 0x03, 0xb8, 0x70, 0x40, 0x42, 0x82, 0x2f, 0x83, 0xd4, 0x63, 0x8f, 0x3d, 0x45, 0xe0, 0x7e, 0x0b, + 0x4e, 0x68, 0xde, 0x8c, 0xed, 0xb5, 0xdb, 0xc6, 0x0d, 0x2d, 0x87, 0x28, 0xfb, 0xde, 0xcc, 0xfb, + 0xbd, 0xff, 0xef, 0x8d, 0xd1, 0x65, 0x41, 0x93, 0x01, 0x4d, 0x9a, 0x5e, 0x40, 0xfc, 0x88, 0x0b, + 0x19, 0xb8, 0x22, 0xee, 0xa6, 0xa9, 0x46, 0x9c, 0x70, 0xc9, 0x71, 0xd5, 0xe5, 0xee, 0x41, 0xc2, + 0x89, 0xbb, 0xdf, 0xd0, 0x12, 0x8d, 0x19, 0x89, 0x8d, 0xd5, 0x6e, 0x3f, 0x60, 0x5e, 0x33, 0x88, + 0x7a, 0x5c, 0x4b, 0x6c, 0xd8, 0x2e, 0x8f, 0x7a, 0x81, 0xdf, 0x3c, 0xe2, 0x11, 0x8d, 0xbb, 0xf0, + 0xcf, 0x9c, 0xac, 0xf9, 0xdc, 0xe7, 0xf0, 0xd9, 0x54, 0x5f, 0x86, 0xfb, 0x36, 0xa0, 0xc7, 0xdd, + 0x26, 0x89, 0xe3, 0x8e, 0x90, 0x44, 0x1a, 0xd5, 0x1b, 0x97, 0xc4, 0x21, 0x6b, 0xba, 0x44, 0x12, + 0xc6, 0xfd, 0xa6, 0x47, 0x85, 0x1b, 0x77, 0x9b, 0x42, 0x26, 0x7d, 0x57, 0xf6, 0x13, 0xea, 0x99, + 0x4b, 0xeb, 0x63, 0xe9, 0x90, 0x4a, 0xe2, 0x11, 0x49, 0x34, 0xbf, 0xfe, 0x47, 0x05, 0xad, 0x6e, + 0x4d, 0x2c, 0x75, 0x68, 0xcc, 0x13, 0x89, 0x5b, 0x28, 0x17, 0x71, 0x8f, 0xda, 0x99, 0x8b, 0x99, + 0x2b, 0x95, 0xeb, 0x57, 0x1a, 0x27, 0xfb, 0xd6, 0xd8, 0xe5, 0x1e, 0xbd, 0x13, 0xf5, 0x78, 0x2b, + 0xf7, 0xf8, 0xb8, 0xb6, 0xe4, 0x80, 0x2c, 0xbe, 0x8d, 0x0a, 0x42, 0xf2, 0x84, 0x0a, 0x3b, 0x7b, + 0xd1, 0xba, 0x52, 0xb9, 0xfe, 0xe1, 0x22, 0x94, 0x3d, 0x75, 0x3b, 0x05, 0x63, 0xc4, 0xf1, 0x16, + 0x2a, 0x08, 0x77, 0x9f, 0x86, 0xc4, 0xb6, 0x00, 0xe8, 0x72, 0x1a, 0xe8, 0x90, 0xa9, 0xbf, 0x2e, + 0x11, 0xb4, 0x71, 0x9f, 0x74, 0x19, 0xdd, 0xa2, 0xc2, 0x4d, 0x82, 0x58, 0xf2, 0x64, 0x82, 0x02, + 0xb2, 0x78, 0x17, 0x95, 0xc5, 0x21, 0xd3, 0x71, 0xb3, 0x73, 0x00, 0x74, 0x75, 0x0e, 0xa8, 0xcd, + 0x19, 0xa3, 0xae, 0xa4, 0xde, 0x9e, 0x24, 0x92, 0x86, 0x34, 0x92, 0xea, 0x23, 0x00, 0xfb, 0x0c, + 0x5a, 0x49, 0x1c, 0x32, 0xc5, 0x14, 0xf8, 0xa7, 0x0c, 0xba, 0xc0, 0xa8, 0x4f, 0xdc, 0x61, 0xa7, + 0x1f, 0x05, 0x61, 0xcc, 0x40, 0x80, 0x7a, 0x1d, 0x9a, 0x24, 0x3c, 0x11, 0x76, 0x1e, 0x54, 0xdc, + 0x5d, 0xe4, 0xf4, 0x7c, 0xe8, 0x1b, 0x3b, 0x80, 0xf9, 0x20, 0x0d, 0xb9, 0x0d, 0x88, 0xdb, 0x91, + 0x4c, 0x86, 0xce, 0x3b, 0xec, 0x65, 0xe7, 0x38, 0x46, 0xab, 0x84, 0x49, 0x9a, 0x50, 0xaf, 0x23, + 0xa8, 0x94, 0x41, 0xe4, 0x0b, 0xbb, 0x00, 0x56, 0x6c, 0x9f, 0xda, 0x8a, 0x4d, 0x0d, 0xb4, 0x67, + 0x70, 0xb4, 0xee, 0xb3, 0x64, 0x96, 0x8b, 0x1f, 0xa1, 0xf3, 0x26, 0x04, 0xe0, 0x74, 0xc7, 0xe5, + 0xfd, 0x48, 0x0a, 0xbb, 0x08, 0x4a, 0x6f, 0xff, 0x47, 0xd7, 0xc1, 0x9b, 0x36, 0x20, 0x69, 0xb5, + 0xe7, 0xd8, 0x3c, 0x1f, 0xff, 0x80, 0x96, 0x55, 0xbb, 0x74, 0x74, 0x07, 0x09, 0xbb, 0x04, 0x1a, + 0x37, 0x4f, 0xad, 0xf1, 0x21, 0x8f, 0x68, 0x5b, 0x63, 0x80, 0x2e, 0x93, 0xe5, 0xca, 0xd1, 0x94, + 0x8f, 0x43, 0xb4, 0xd2, 0xa3, 0x44, 0xb5, 0x52, 0xa7, 0x2f, 0x88, 0x4f, 0xed, 0x32, 0x28, 0x6b, + 0x9d, 0x5a, 0xd9, 0x57, 0x1a, 0xe5, 0x81, 0x02, 0x49, 0x6b, 0x5b, 0xee, 0xa5, 0x0e, 0xf0, 0x2e, + 0xb2, 0xc4, 0x21, 0xb3, 0x11, 0x74, 0x5e, 0x73, 0x61, 0xcf, 0xdc, 0xdb, 0xb9, 0x13, 0x09, 0x49, + 0x22, 0x57, 0x77, 0x4e, 0x45, 0x21, 0x8e, 0x8e, 0x6b, 0xd6, 0xde, 0xbd, 0x1d, 0x47, 0x01, 0xe1, + 0x36, 0xb2, 0x68, 0x34, 0xb0, 0x2b, 0x80, 0x77, 0x75, 0x11, 0xde, 0x76, 0x34, 0x08, 0x12, 0x1e, + 0xa9, 0xba, 0x32, 0xd6, 0x29, 0xe9, 0x8d, 0x16, 0x5a, 0x7b, 0x51, 0x45, 0xe0, 0x55, 0x64, 0x1d, + 0xd0, 0x21, 0x8c, 0x89, 0xb2, 0xa3, 0x3e, 0xf1, 0x1a, 0xca, 0x0f, 0x08, 0xeb, 0x53, 0x3b, 0x0b, + 0x3c, 0x4d, 0x7c, 0x91, 0xfd, 0x2c, 0xb3, 0xe1, 0xa2, 0xd5, 0xf9, 0x70, 0xa7, 0xe5, 0x2d, 0x2d, + 0xff, 0x79, 0x5a, 0xbe, 0x72, 0xfd, 0x52, 0xca, 0x60, 0x9d, 0xec, 0x86, 0x1e, 0x97, 0xa9, 0xd4, + 0xa5, 0x95, 0xdc, 0x42, 0xe7, 0x9e, 0x0b, 0xf3, 0x22, 0x2b, 0xf3, 0x69, 0x80, 0x1d, 0x54, 0x3d, + 0xb9, 0x03, 0x17, 0xa1, 0x59, 0x69, 0xb4, 0x2d, 0xb4, 0xfe, 0xe2, 0xa2, 0x3e, 0x0d, 0x4a, 0xfd, + 0xcf, 0x1c, 0x2a, 0x8d, 0x47, 0x2c, 0x7e, 0x88, 0x8a, 0x6a, 0xbc, 0x76, 0x02, 0x0f, 0x84, 0xf3, + 0xad, 0xcd, 0xd1, 0x71, 0xad, 0x00, 0xc7, 0x5b, 0xff, 0x1c, 0xd7, 0x6e, 0xf8, 0x81, 0xdc, 0xef, + 0x77, 0x1b, 0x2e, 0x0f, 0x9b, 0x93, 0xd0, 0x79, 0xdd, 0xe9, 0x77, 0x33, 0x3e, 0xf0, 0x9b, 0x66, + 0x17, 0xe8, 0xc1, 0xbd, 0xe5, 0x14, 0x14, 0xe2, 0x1d, 0x4f, 0x99, 0xd0, 0x1d, 0x4a, 0x98, 0xd8, + 0x60, 0x02, 0x10, 0xf8, 0x02, 0x2a, 0x1f, 0xd0, 0xa1, 0x6e, 0x6e, 0xdb, 0x82, 0x93, 0xd2, 0x01, + 0x1d, 0x82, 0x37, 0xb8, 0x86, 0x2a, 0x09, 0x89, 0x7c, 0x6a, 0x8e, 0x73, 0x70, 0x8c, 0x80, 0xa5, + 0x2f, 0xdc, 0x44, 0x25, 0xc6, 0x5d, 0xc2, 0x02, 0x39, 0xb4, 0xf3, 0x90, 0xd3, 0x0b, 0xa9, 0x9c, + 0x8e, 0x0d, 0xd9, 0x31, 0x57, 0xc6, 0x63, 0x76, 0x2c, 0x82, 0x77, 0x51, 0x69, 0x9f, 0x24, 0xde, + 0x23, 0x92, 0x50, 0xbb, 0x00, 0xe2, 0x1f, 0x2d, 0xaa, 0xe1, 0xaf, 0xcd, 0xfd, 0xd4, 0x2a, 0x99, + 0x60, 0xe0, 0x2f, 0x51, 0x96, 0xab, 0x09, 0x95, 0x99, 0x5f, 0x24, 0x2f, 0x42, 0xba, 0xbb, 0x97, + 0xc2, 0xc8, 0x72, 0x81, 0xaf, 0xa1, 0x3c, 0xac, 0x71, 0xbb, 0x04, 0x00, 0x6f, 0xa5, 0x00, 0x80, + 0xdf, 0x48, 0xdd, 0xd7, 0x37, 0xf1, 0x3a, 0x2a, 0xf4, 0x63, 0x19, 0x84, 0x6a, 0x6e, 0xa8, 0xd8, + 0x18, 0x0a, 0xbf, 0x87, 0x96, 0x59, 0xe0, 0xd2, 0x48, 0xd0, 0x8e, 0x1c, 0xc6, 0x14, 0x1a, 0xbe, + 0xec, 0x54, 0x0c, 0xef, 0xfe, 0x30, 0x56, 0xa3, 0xa0, 0x24, 0x79, 0xcc, 0x19, 0xf7, 0x87, 0xa6, + 0x7f, 0x17, 0xfa, 0x7e, 0xdf, 0xdc, 0x4f, 0xfb, 0x3e, 0xc6, 0xa8, 0xff, 0x66, 0xa1, 0xf2, 0x64, + 0xc9, 0xfe, 0xaf, 0x85, 0xf4, 0x3d, 0x2a, 0xc1, 0xf2, 0x56, 0xe0, 0xd0, 0x62, 0xad, 0xf6, 0xe8, + 0xb8, 0x56, 0xd4, 0xca, 0x15, 0xfa, 0xc7, 0xa7, 0x42, 0x37, 0x72, 0x4e, 0x11, 0x40, 0xd3, 0x85, + 0x6a, 0xbd, 0xb4, 0x50, 0x73, 0x27, 0x17, 0x6a, 0xfe, 0xb9, 0x42, 0xdd, 0x40, 0x25, 0x97, 0xc4, + 0xc4, 0x55, 0x85, 0x5a, 0xd0, 0xc2, 0x63, 0x1a, 0xbf, 0x8b, 0xca, 0x64, 0x40, 0x02, 0xa6, 0x5e, + 0x18, 0x50, 0x3c, 0x96, 0x33, 0x65, 0x60, 0x8c, 0x72, 0x7d, 0x41, 0x75, 0x51, 0x58, 0x0e, 0x7c, + 0xe3, 0x6b, 0x68, 0x8d, 0x46, 0x6e, 0x32, 0x8c, 0x65, 0xc0, 0xa3, 0x0e, 0x61, 0x3e, 0x4f, 0x02, + 0xb9, 0x1f, 0x9a, 0x22, 0x38, 0x3f, 0x3d, 0xdb, 0x1c, 0x1f, 0xd5, 0x7f, 0xcf, 0xa0, 0xb3, 0x73, + 0xf3, 0x1c, 0x4b, 0x74, 0x56, 0xbd, 0x5a, 0x02, 0xc3, 0x9b, 0x26, 0x6b, 0x67, 0x74, 0x5c, 0x5b, + 0x49, 0xdf, 0x56, 0x51, 0xfd, 0xf4, 0x95, 0xa2, 0x0a, 0x8f, 0xa5, 0x19, 0x51, 0x67, 0x45, 0x1c, + 0xb2, 0x09, 0x99, 0xae, 0xd9, 0x6c, 0xba, 0x66, 0xeb, 0x3f, 0x5a, 0xa8, 0x92, 0xda, 0x10, 0x33, + 0xbd, 0x9d, 0x79, 0xbd, 0xde, 0xce, 0xbe, 0xb1, 0xde, 0xb6, 0x5e, 0xb7, 0xb7, 0x73, 0xaf, 0xdc, + 0xdb, 0xf3, 0x3d, 0x9c, 0x3f, 0xb9, 0x87, 0x0b, 0x6f, 0xa0, 0x87, 0x7f, 0xc9, 0xa0, 0x62, 0xfb, + 0xdb, 0x07, 0x50, 0x1c, 0xeb, 0xa8, 0x10, 0xf5, 0x43, 0x37, 0xee, 0xeb, 0x9a, 0x70, 0x0c, 0x85, + 0x6d, 0x54, 0x14, 0xdc, 0x3d, 0xa0, 0x52, 0x98, 0xfd, 0x36, 0x26, 0x55, 0xdf, 0xb8, 0xf0, 0x24, + 0xb7, 0xf4, 0xde, 0x03, 0x42, 0x71, 0x43, 0xee, 0x51, 0x06, 0x9e, 0x97, 0x1d, 0x4d, 0xa8, 0x0d, + 0x15, 0xee, 0x1f, 0x81, 0x4f, 0x59, 0x47, 0x7d, 0xaa, 0x0e, 0x31, 0x4f, 0x15, 0xfd, 0xb0, 0x2c, + 0x3b, 0x13, 0xba, 0xfe, 0x6b, 0x16, 0x2d, 0xa7, 0x93, 0x83, 0x2f, 0xa3, 0x33, 0x83, 0x20, 0x91, + 0x7d, 0xc2, 0x82, 0x23, 0xa2, 0x0a, 0xdd, 0xec, 0xba, 0x39, 0x2e, 0xbe, 0x85, 0x2c, 0xe5, 0x81, + 0xce, 0xff, 0x07, 0x8b, 0x62, 0x63, 0x5c, 0x1f, 0xbf, 0x4d, 0x94, 0xb7, 0xb7, 0x90, 0x15, 0xd2, + 0xd0, 0xa4, 0x7d, 0x21, 0xc0, 0x37, 0x34, 0x4c, 0x03, 0x84, 0x34, 0x54, 0xcd, 0xcd, 0x38, 0xf1, + 0xc8, 0xc0, 0xbf, 0xf6, 0x09, 0x84, 0x20, 0xeb, 0x4c, 0x19, 0xca, 0xe9, 0x38, 0xe1, 0x83, 0xc0, + 0xa3, 0x89, 0xc9, 0xef, 0x84, 0xc6, 0xef, 0xa3, 0x33, 0x93, 0xce, 0x74, 0x19, 0x11, 0x02, 0x52, + 0x5c, 0x76, 0x56, 0xc6, 0xdc, 0xb6, 0x62, 0xd6, 0xbf, 0x43, 0x05, 0x5d, 0x6d, 0x2a, 0x63, 0x3d, + 0x12, 0x06, 0x6c, 0xbc, 0xf8, 0x0d, 0x05, 0x4a, 0x18, 0x91, 0x3d, 0x9e, 0x84, 0xe6, 0xe1, 0x34, + 0xa1, 0x55, 0x36, 0x07, 0x34, 0x11, 0x2a, 0x82, 0x16, 0x1c, 0x8d, 0xc9, 0xfa, 0x4d, 0x54, 0x34, + 0xee, 0xa8, 0x14, 0x4a, 0x2e, 0x09, 0x03, 0xdc, 0x9c, 0xa3, 0x89, 0xd9, 0xb1, 0x95, 0x85, 0x93, + 0x29, 0xa3, 0xde, 0x42, 0xcb, 0xe9, 0x52, 0x9b, 0xf1, 0x34, 0x33, 0xe7, 0xe9, 0x3a, 0x2a, 0x24, + 0xd4, 0x57, 0x36, 0x68, 0xf3, 0x0c, 0xd5, 0x6a, 0x3e, 0xfe, 0xbb, 0xba, 0xf4, 0x78, 0x54, 0xcd, + 0x3c, 0x19, 0x55, 0x33, 0x4f, 0x47, 0xd5, 0xcc, 0x5f, 0xa3, 0x6a, 0xe6, 0xe7, 0x67, 0xd5, 0xa5, + 0x27, 0xcf, 0xaa, 0x4b, 0x4f, 0x9f, 0x55, 0x97, 0x1e, 0xae, 0xcc, 0xc4, 0xbf, 0x5b, 0x80, 0x5f, + 0x9d, 0x37, 0xfe, 0x0d, 0x00, 0x00, 0xff, 0xff, 0x44, 0x06, 0x5a, 0x4b, 0x57, 0x0f, 0x00, 0x00, } diff --git a/pkg/server/diagnosticspb/diagnostics.proto b/pkg/server/diagnosticspb/diagnostics.proto index d101bd6c3838..b6f1e7de63d4 100644 --- a/pkg/server/diagnosticspb/diagnostics.proto +++ b/pkg/server/diagnosticspb/diagnostics.proto @@ -27,23 +27,31 @@ message DiagnosticReport { map altered_settings = 6; map zone_configs = 8 [(gogoproto.nullable) = false]; map feature_usage = 9 [(gogoproto.nullable) = false]; - + SQLInstanceInfo sql = 10 [(gogoproto.nullable) = false, (gogoproto.customname) = "SQL"]; + Environment env = 11 [(gogoproto.nullable) = false]; map legacy_unimplemented_errors = 5; map legacy_error_counts = 7; - } +} +// NodeInfo describes the CRDB node which is reporting diagnostics. NodeInfo +// will be set to its empty value (e.g. NodeID = 0) when the report is generated +// by a SQL-only instance. message NodeInfo { int32 node_id = 1 [(gogoproto.customname) = "NodeID", (gogoproto.casttype) = "github.com/cockroachdb/cockroach/pkg/roachpb.NodeID"]; int64 bytes = 2; int64 key_count = 3; int64 range_count = 4; - roachpb.Locality locality = 5 [(gogoproto.nullable) = false]; + // Uptime is the number of seconds since the CRDB node was started. + int64 uptime = 9; + + // The following fields are deprecated and will be removed. They are replaced + // by the separated DiagnosticsReport.Env field. + roachpb.Locality locality = 5 [(gogoproto.nullable) = false]; HardwareInfo hardware = 6 [(gogoproto.nullable) = false]; OSInfo os = 7 [(gogoproto.nullable) = false]; build.Info build = 8 [(gogoproto.nullable) = false]; - int64 uptime = 9; string license_type = 10; TopologyInfo topology = 11 [(gogoproto.nullable) = false]; } @@ -64,6 +72,36 @@ message StoreInfo { // Want: sync times, observed read/write speeds } +// SQLInstanceInfo describes the SQL instance which is reporting diagnostics. +// This is always set, even for full CRDB nodes running both KV and SQL. +message SQLInstanceInfo { + // SQLInstanceID is the ephemeral identifier for the SQL instance which is + // reporting diagnostics. This is unique across all running SQL instances in + // the cluster (physical or tenant). + int32 sql_instance_id = 1 [(gogoproto.customname) = "SQLInstanceID", (gogoproto.casttype) = "github.com/cockroachdb/cockroach/pkg/base.SQLInstanceID"]; + // Uptime is the number of seconds since the SQL instance was started. + int64 uptime = 2; +} + +// Environment describes the context in which the CRDB node or SQL instance is +// running. +message Environment { + // Locality is an ordered set of key/value tiers that describe the location of + // this CRDB node or SQL instance. + roachpb.Locality locality = 1 [(gogoproto.nullable) = false]; + // Hardware describes the physical machine, virtual machine, or Linux + // container on which CRDB is running. + HardwareInfo hardware = 2 [(gogoproto.nullable) = false]; + // OS describes the operating system software that hosts CRDB. + OSInfo os = 3 [(gogoproto.nullable) = false]; + // Build describes the running CockroachDB binary. + build.Info build = 4 [(gogoproto.nullable) = false]; + // LicenseType describes the license that is in effect (OSS, Enterprise, etc). + string license_type = 5; + // Topology provides the cloud provider and region name that is hosting CRDB. + TopologyInfo topology = 6 [(gogoproto.nullable) = false]; +} + message CPUInfo { int32 numcpu = 1; // go's reported runtime.NUMCPU() int32 sockets = 2; // number of cpus reported diff --git a/pkg/server/server.go b/pkg/server/server.go index 3bf0543347f1..fbef9d4a3e73 100644 --- a/pkg/server/server.go +++ b/pkg/server/server.go @@ -150,7 +150,7 @@ type Server struct { protectedtsProvider protectedts.Provider protectedtsReconciler *ptreconcile.Reconciler - sqlServer *sqlServer + sqlServer *SQLServer // Created in NewServer but initialized (made usable) in `(*Server).Start`. externalStorageBuilder *externalStorageBuilder @@ -1874,7 +1874,7 @@ func (s *Server) startServeUI( } // TODO(tbg): move into server_sql.go. -func (s *sqlServer) startServeSQL( +func (s *SQLServer) startServeSQL( ctx context.Context, stopper *stop.Stopper, connManager netutil.Server, diff --git a/pkg/server/server_sql.go b/pkg/server/server_sql.go index 722a5ca26972..f21d927542ff 100644 --- a/pkg/server/server_sql.go +++ b/pkg/server/server_sql.go @@ -36,6 +36,7 @@ import ( "github.com/cockroachdb/cockroach/pkg/rpc" "github.com/cockroachdb/cockroach/pkg/rpc/nodedialer" "github.com/cockroachdb/cockroach/pkg/scheduledjobs" + "github.com/cockroachdb/cockroach/pkg/server/diagnostics" "github.com/cockroachdb/cockroach/pkg/server/serverpb" "github.com/cockroachdb/cockroach/pkg/server/status" "github.com/cockroachdb/cockroach/pkg/sql" @@ -66,12 +67,20 @@ import ( "github.com/cockroachdb/cockroach/pkg/util/mon" "github.com/cockroachdb/cockroach/pkg/util/netutil" "github.com/cockroachdb/cockroach/pkg/util/stop" + "github.com/cockroachdb/cockroach/pkg/util/timeutil" "github.com/cockroachdb/errors" "github.com/marusama/semaphore" "google.golang.org/grpc" ) -type sqlServer struct { +// SQLServer encapsulates the part of a CRDB server that is dedicated to SQL +// processing. All SQL commands are reduced to primitive operations on the +// lower-level KV layer. Multi-tenant installations of CRDB run zero or more +// standalone SQLServer instances per tenant (the KV layer is shared across all +// tenants). +type SQLServer struct { + stopper *stop.Stopper + sqlIDContainer *base.SQLIDContainer pgServer *pgwire.Server distSQLServer *distsql.ServerImpl execCfg *sql.ExecutorConfig @@ -92,6 +101,7 @@ type sqlServer struct { stmtDiagnosticsRegistry *stmtdiagnostics.Registry sqlLivenessProvider sqlliveness.Provider metricsRegistry *metric.Registry + diagnosticsReporter *diagnostics.Reporter } // sqlServerOptionalKVArgs are the arguments supplied to newSQLServer which are @@ -196,15 +206,18 @@ type sqlServerArgs struct { sqlStatusServer serverpb.SQLStatusServer } -func newSQLServer(ctx context.Context, cfg sqlServerArgs) (*sqlServer, error) { +func newSQLServer(ctx context.Context, cfg sqlServerArgs) (*SQLServer, error) { // NB: ValidateAddrs also fills in defaults. if err := cfg.Config.ValidateAddrs(ctx); err != nil { return nil, err } execCfg := &sql.ExecutorConfig{} codec := keys.MakeSQLCodec(cfg.SQLConfig.TenantID) - if override := cfg.SQLConfig.TenantIDCodecOverride; override != (roachpb.TenantID{}) { - codec = keys.MakeSQLCodec(override) + if knobs := cfg.TestingKnobs.TenantTestingKnobs; knobs != nil { + override := knobs.(*sql.TenantTestingKnobs).TenantIDCodecOverride + if override != (roachpb.TenantID{}) { + codec = keys.MakeSQLCodec(override) + } } // Create blob service for inter-node file sharing. @@ -629,7 +642,30 @@ func newSQLServer(ctx context.Context, cfg sqlServerArgs) (*sqlServer, error) { sqlExecutorTestingKnobs, ) - return &sqlServer{ + var reporter *diagnostics.Reporter + if cfg.tenantConnect != nil { + reporter = &diagnostics.Reporter{ + StartTime: timeutil.Now(), + AmbientCtx: &cfg.AmbientCtx, + Config: cfg.BaseConfig.Config, + Settings: cfg.Settings, + ClusterID: cfg.rpcContext.ClusterID.Get, + TenantID: cfg.rpcContext.TenantID, + SQLInstanceID: cfg.nodeIDContainer.SQLInstanceID, + SQLServer: pgServer.SQLServer, + InternalExec: cfg.circularInternalExecutor, + DB: cfg.db, + Recorder: cfg.recorder, + Locality: cfg.Locality, + } + if cfg.TestingKnobs.Server != nil { + reporter.TestingKnobs = &cfg.TestingKnobs.Server.(*TestingKnobs).DiagnosticsTestingKnobs + } + } + + return &SQLServer{ + stopper: cfg.stopper, + sqlIDContainer: cfg.nodeIDContainer, pgServer: pgServer, distSQLServer: distSQLServer, execCfg: execCfg, @@ -646,10 +682,11 @@ func newSQLServer(ctx context.Context, cfg sqlServerArgs) (*sqlServer, error) { stmtDiagnosticsRegistry: stmtDiagnosticsRegistry, sqlLivenessProvider: cfg.sqlLivenessProvider, metricsRegistry: cfg.registry, + diagnosticsReporter: reporter, }, nil } -func (s *sqlServer) start( +func (s *SQLServer) start( ctx context.Context, stopper *stop.Stopper, knobs base.TestingKnobs, @@ -790,3 +827,17 @@ func (s *sqlServer) start( return nil } + +// SQLInstanceID returns the ephemeral ID assigned to each SQL instance. The ID +// is guaranteed to be unique across all currently running instances, but may be +// reused once an instance is stopped. +func (s *SQLServer) SQLInstanceID() base.SQLInstanceID { + return s.sqlIDContainer.SQLInstanceID() +} + +// StartDiagnostics starts periodic diagnostics reporting. +// NOTE: This is not called in start so that it's disabled by default for +// testing. +func (s *SQLServer) StartDiagnostics(ctx context.Context) { + s.diagnosticsReporter.PeriodicallyReportDiagnostics(ctx, s.stopper) +} diff --git a/pkg/server/testserver.go b/pkg/server/testserver.go index 55e5efce0b92..88d5c3f8b464 100644 --- a/pkg/server/testserver.go +++ b/pkg/server/testserver.go @@ -53,6 +53,7 @@ import ( "github.com/cockroachdb/cockroach/pkg/sqlmigrations" "github.com/cockroachdb/cockroach/pkg/storage" "github.com/cockroachdb/cockroach/pkg/storage/cloud" + "github.com/cockroachdb/cockroach/pkg/testutils/serverutils" "github.com/cockroachdb/cockroach/pkg/ts" "github.com/cockroachdb/cockroach/pkg/util" "github.com/cockroachdb/cockroach/pkg/util/hlc" @@ -567,25 +568,56 @@ func makeSQLServerArgs( }, nil } +// TestTenant is an in-memory instantiation of the SQL-only process created for +// each active Cockroach tenant. TestTenant provides tests with access to +// internal methods and state on SQLServer. It is typically started in tests by +// calling the TestServerInterface.StartTenant method or by calling the wrapper +// serverutils.StartTenant method. +type TestTenant struct { + *SQLServer + sqlAddr string + httpAddr string +} + +// SQLAddr is part of the TestTenantInterface interface. +func (t *TestTenant) SQLAddr() string { + return t.sqlAddr +} + +// HTTPAddr is part of the TestTenantInterface interface. +func (t *TestTenant) HTTPAddr() string { + return t.httpAddr +} + +// PGServer is part of the TestTenantInterface interface. +func (t *TestTenant) PGServer() interface{} { + return t.pgServer +} + +// DiagnosticsReporter is part of the TestTenantInterface interface. +func (t *TestTenant) DiagnosticsReporter() interface{} { + return t.diagnosticsReporter +} + // StartTenant starts a SQL tenant communicating with this TestServer. func (ts *TestServer) StartTenant( params base.TestTenantArgs, -) (pgAddr string, httpAddr string, _ error) { +) (serverutils.TestTenantInterface, error) { ctx := context.Background() if !params.Existing { if _, err := ts.InternalExecutor().(*sql.InternalExecutor).Exec( ctx, "testserver-create-tenant", nil /* txn */, "SELECT crdb_internal.create_tenant($1)", params.TenantID.ToUint64(), ); err != nil { - return "", "", err + return nil, err } } st := cluster.MakeTestingClusterSettings() sqlCfg := makeTestSQLConfig(st, params.TenantID) sqlCfg.TenantKVAddrs = []string{ts.ServingRPCAddr()} - sqlCfg.TenantIDCodecOverride = params.TenantIDCodecOverride baseCfg := makeTestBaseConfig(st) + baseCfg.TestingKnobs = params.TestingKnobs if params.AllowSettingClusterSettings { baseCfg.TestingKnobs.TenantTestingKnobs = &sql.TenantTestingKnobs{ ClusterSettingsUpdater: st.MakeUpdater(), @@ -595,13 +627,14 @@ func (ts *TestServer) StartTenant( if stopper == nil { stopper = ts.Stopper() } - return StartTenant( + sqlServer, addr, httpAddr, err := StartTenant( ctx, stopper, ts.Cfg.ClusterName, baseCfg, sqlCfg, ) + return &TestTenant{SQLServer: sqlServer, sqlAddr: addr, httpAddr: httpAddr}, err } // StartTenant starts a stand-alone SQL server against a KV backend. @@ -611,14 +644,14 @@ func StartTenant( kvClusterName string, // NB: gone after https://github.com/cockroachdb/cockroach/issues/42519 baseCfg BaseConfig, sqlCfg SQLConfig, -) (pgAddr string, httpAddr string, _ error) { +) (sqlServer *SQLServer, pgAddr string, httpAddr string, _ error) { args, err := makeSQLServerArgs(stopper, kvClusterName, baseCfg, sqlCfg) if err != nil { - return "", "", err + return nil, "", "", err } s, err := newSQLServer(ctx, args) if err != nil { - return "", "", err + return nil, "", "", err } // TODO(asubiotto): remove this. Right now it is needed to initialize the @@ -635,7 +668,7 @@ func StartTenant( pgL, err := listen(ctx, &args.Config.SQLAddr, &args.Config.SQLAdvertiseAddr, "sql") if err != nil { - return "", "", err + return nil, "", "", err } args.stopper.RunWorker(ctx, func(ctx context.Context) { @@ -649,7 +682,7 @@ func StartTenant( httpL, err := listen(ctx, &args.Config.HTTPAddr, &args.Config.HTTPAdvertiseAddr, "http") if err != nil { - return "", "", err + return nil, "", "", err } args.stopper.RunWorker(ctx, func(ctx context.Context) { @@ -699,7 +732,7 @@ func StartTenant( heapProfileDirName: args.HeapProfileDirName, runtime: args.runtime, }); err != nil { - return "", "", err + return nil, "", "", err } if err := s.start(ctx, @@ -710,10 +743,10 @@ func StartTenant( socketFile, orphanedLeasesTimeThresholdNanos, ); err != nil { - return "", "", err + return nil, "", "", err } - return pgLAddr, httpLAddr, nil + return s, pgLAddr, httpLAddr, nil } // ExpectedInitialRangeCount returns the expected number of ranges that should @@ -1242,6 +1275,11 @@ func (ts *TestServer) ScratchRange() (roachpb.Key, error) { return scratchKey, nil } +// MetricsRecorder is part of TestServerInterface. +func (ts *TestServer) MetricsRecorder() interface{} { + return ts.node.recorder +} + type testServerFactoryImpl struct{} // TestServerFactory can be passed to serverutils.InitTestServerFactory diff --git a/pkg/server/updates.go b/pkg/server/updates.go index a27c60ac3027..d0e488babac2 100644 --- a/pkg/server/updates.go +++ b/pkg/server/updates.go @@ -28,6 +28,7 @@ import ( "github.com/cockroachdb/cockroach/pkg/keys" "github.com/cockroachdb/cockroach/pkg/roachpb" "github.com/cockroachdb/cockroach/pkg/security" + "github.com/cockroachdb/cockroach/pkg/server/diagnostics" "github.com/cockroachdb/cockroach/pkg/server/diagnosticspb" "github.com/cockroachdb/cockroach/pkg/server/telemetry" "github.com/cockroachdb/cockroach/pkg/settings" @@ -59,12 +60,6 @@ const ( updateCheckJitterSeconds = 120 ) -var diagnosticReportFrequency = settings.RegisterPublicNonNegativeDurationSetting( - "diagnostics.reporting.interval", - "interval at which diagnostics data should be reported", - time.Hour, -) - // randomly shift `d` to be up to `jitterSec` shorter or longer. func addJitter(d time.Duration, jitterSec int) time.Duration { j := time.Duration(rand.Intn(jitterSec*2)-jitterSec) * time.Second @@ -193,6 +188,7 @@ func (s *Server) checkForUpdates(ctx context.Context) bool { clusterInfo := diagnosticspb.ClusterInfo{ ClusterID: s.ClusterID(), + TenantID: s.rpcContext.TenantID, IsInsecure: s.cfg.Insecure, IsInternal: sql.ClusterIsInternal(&s.st.SV), } @@ -255,7 +251,7 @@ func (s *Server) maybeReportDiagnostics(ctx context.Context, now, scheduled time s.ReportDiagnostics(ctx) } - return scheduled.Add(diagnosticReportFrequency.Get(&s.st.SV)) + return scheduled.Add(diagnostics.ReportFrequency.Get(&s.st.SV)) } func (s *Server) collectNodeInfo(ctx context.Context) diagnosticspb.NodeInfo { @@ -423,6 +419,7 @@ func (s *Server) ReportDiagnostics(ctx context.Context) { clusterInfo := diagnosticspb.ClusterInfo{ ClusterID: s.ClusterID(), + TenantID: s.rpcContext.TenantID, IsInsecure: s.cfg.Insecure, IsInternal: sql.ClusterIsInternal(&s.st.SV), } @@ -430,7 +427,7 @@ func (s *Server) ReportDiagnostics(ctx context.Context) { if s.cfg.TestingKnobs.Server != nil { knobs = &s.cfg.TestingKnobs.Server.(*TestingKnobs).DiagnosticsTestingKnobs } - reportingURL := diagnosticspb.BuildReportingURL(&clusterInfo, &report.Node, knobs) + reportingURL := diagnosticspb.BuildReportingURL(&clusterInfo, report, knobs) if reportingURL == nil { return } diff --git a/pkg/sql/exec_util.go b/pkg/sql/exec_util.go index 97e940cfaba3..83fa54766578 100644 --- a/pkg/sql/exec_util.go +++ b/pkg/sql/exec_util.go @@ -870,6 +870,10 @@ type TenantTestingKnobs struct { // in-memory cluster settings. SQL tenants are otherwise prohibited from // setting cluster settings. ClusterSettingsUpdater settings.Updater + + // TenantIDCodecOverride overrides the tenant ID used to construct the SQL + // server's codec, but nothing else (e.g. its certs). Used for testing. + TenantIDCodecOverride roachpb.TenantID } var _ base.ModuleTestingKnobs = &TenantTestingKnobs{} diff --git a/pkg/sql/logictest/logic.go b/pkg/sql/logictest/logic.go index a9a18e07072e..de28f2084ac6 100644 --- a/pkg/sql/logictest/logic.go +++ b/pkg/sql/logictest/logic.go @@ -1331,10 +1331,15 @@ func (t *logicTest) setup(cfg testClusterConfig, serverArgs TestServerArgs) { connsForClusterSettingChanges := []*gosql.DB{t.cluster.ServerConn(0)} if cfg.useTenant { var err error - t.tenantAddr, _, err = t.cluster.Server(t.nodeIdx).StartTenant(base.TestTenantArgs{TenantID: roachpb.MakeTenantID(10), AllowSettingClusterSettings: true}) + tenantArgs := base.TestTenantArgs{ + TenantID: roachpb.MakeTenantID(10), + AllowSettingClusterSettings: true, + } + tenant, err := t.cluster.Server(t.nodeIdx).StartTenant(tenantArgs) if err != nil { t.rootT.Fatalf("%+v", err) } + t.tenantAddr = tenant.SQLAddr() // Open a connection to this tenant to set any cluster settings specified // by the test config. diff --git a/pkg/sql/run_control_test.go b/pkg/sql/run_control_test.go index 14b8e037a881..b8202b62bf79 100644 --- a/pkg/sql/run_control_test.go +++ b/pkg/sql/run_control_test.go @@ -59,7 +59,7 @@ func makeRunControlTestCases(t *testing.T) ([]runControlTestCase, func()) { testCases[0].conn1 = tc.ServerConn(0).Conn testCases[0].conn2 = tc.ServerConn(1).Conn - tenantDB := serverutils.StartTenant(t, tc.Server(0), base.TestTenantArgs{TenantID: roachpb.MakeTenantID(10)}) + _, tenantDB := serverutils.StartTenant(t, tc.Server(0), base.TestTenantArgs{TenantID: roachpb.MakeTenantID(10)}) testCases[1].name = "Tenant" testCases[1].conn1 = tenantDB.Conn testCases[1].conn2 = tenantDB.Conn diff --git a/pkg/sql/telemetry_test.go b/pkg/sql/telemetry_test.go index 13de95776af7..09b4500db7ad 100644 --- a/pkg/sql/telemetry_test.go +++ b/pkg/sql/telemetry_test.go @@ -13,6 +13,7 @@ package sql_test import ( "bytes" "context" + gosql "database/sql" "fmt" "regexp" "sort" @@ -22,13 +23,16 @@ import ( "github.com/cockroachdb/cockroach/pkg/base" "github.com/cockroachdb/cockroach/pkg/roachpb" + "github.com/cockroachdb/cockroach/pkg/security" "github.com/cockroachdb/cockroach/pkg/server" + "github.com/cockroachdb/cockroach/pkg/server/diagnostics" "github.com/cockroachdb/cockroach/pkg/server/diagnosticspb" "github.com/cockroachdb/cockroach/pkg/sql" "github.com/cockroachdb/cockroach/pkg/sql/catalog/catconstants" "github.com/cockroachdb/cockroach/pkg/sql/catalog/descpb" "github.com/cockroachdb/cockroach/pkg/testutils/diagutils" "github.com/cockroachdb/cockroach/pkg/testutils/serverutils" + "github.com/cockroachdb/cockroach/pkg/testutils/skip" "github.com/cockroachdb/cockroach/pkg/testutils/sqlutils" "github.com/cockroachdb/cockroach/pkg/util/cloudinfo" "github.com/cockroachdb/cockroach/pkg/util/leaktest" @@ -75,125 +79,185 @@ func TestTelemetry(t *testing.T) { defer leaktest.AfterTest(t)() defer log.Scope(t).Close(t) - ctx := context.Background() // Note: these tests cannot be run in parallel (with each other or with other // tests) because telemetry counters are global. datadriven.Walk(t, "testdata/telemetry", func(t *testing.T, path string) { // Disable cloud info reporting (it would make these tests really slow). defer cloudinfo.Disable()() - diagSrv := diagutils.NewServer() - defer diagSrv.Close() + var test telemetryTest + test.Start(t) + defer test.Close() + + // Run test against physical CRDB cluster. + t.Run("server", func(t *testing.T) { + datadriven.RunTest(t, path, func(t *testing.T, td *datadriven.TestData) string { + sqlServer := test.server.SQLServer().(*sql.Server) + return test.RunTest(td, test.serverDB, test.server.ReportDiagnostics, sqlServer) + }) + }) - diagSrvURL := diagSrv.URL() - params := base.TestServerArgs{ - Knobs: base.TestingKnobs{ - Server: &server.TestingKnobs{ - DiagnosticsTestingKnobs: diagnosticspb.TestingKnobs{ - OverrideReportingURL: &diagSrvURL, - }, + // Run test against logical tenant cluster. + t.Run("tenant", func(t *testing.T) { + // TODO(andyk): Re-enable these tests once tenant clusters fully + // support the features they're using. + switch path { + case "testdata/telemetry/execution", + "testdata/telemetry/planning", + "testdata/telemetry/sql-stats": + skip.WithIssue(t, 47893, "tenant clusters do not support SQL features used by this test") + } + + datadriven.RunTest(t, path, func(t *testing.T, td *datadriven.TestData) string { + sqlServer := test.server.SQLServer().(*sql.Server) + reporter := test.tenant.DiagnosticsReporter().(*diagnostics.Reporter) + return test.RunTest(td, test.tenantDB, reporter.ReportDiagnostics, sqlServer) + }) + }) + }) +} + +type telemetryTest struct { + t *testing.T + diagSrv *diagutils.Server + server serverutils.TestServerInterface + serverDB *gosql.DB + tenant serverutils.TestTenantInterface + tenantDB *gosql.DB + allowlist featureAllowlist +} + +func (tt *telemetryTest) Start(t *testing.T) { + tt.t = t + tt.diagSrv = diagutils.NewServer() + + diagSrvURL := tt.diagSrv.URL() + params := base.TestServerArgs{ + Knobs: base.TestingKnobs{ + Server: &server.TestingKnobs{ + DiagnosticsTestingKnobs: diagnosticspb.TestingKnobs{ + OverrideReportingURL: &diagSrvURL, }, }, - } - s, sqlConn, _ := serverutils.StartServer(t, params) - defer s.Stopper().Stop(ctx) + }, + } + tt.server, tt.serverDB, _ = serverutils.StartServer(tt.t, params) + tt.prepareCluster(tt.serverDB) - runner := sqlutils.MakeSQLRunner(sqlConn) - // Disable automatic reporting so it doesn't interfere with the test. - runner.Exec(t, "SET CLUSTER SETTING diagnostics.reporting.enabled = false") - runner.Exec(t, "SET CLUSTER SETTING diagnostics.reporting.send_crash_reports = false") - // Disable plan caching to get accurate counts if the same statement is - // issued multiple times. - runner.Exec(t, "SET CLUSTER SETTING sql.query_cache.enabled = false") + tt.tenant, tt.tenantDB = serverutils.StartTenant(tt.t, tt.server, base.TestTenantArgs{ + TenantID: roachpb.MakeTenantID(security.EmbeddedTenantIDs()[0]), + AllowSettingClusterSettings: true, + TestingKnobs: params.Knobs, + }) + tt.prepareCluster(tt.tenantDB) +} - var allowlist featureAllowlist - datadriven.RunTest(t, path, func(t *testing.T, td *datadriven.TestData) string { - switch td.Cmd { - case "exec": - _, err := sqlConn.Exec(td.Input) - if err != nil { - if errors.HasAssertionFailure(err) { - td.Fatalf(t, "%+v", err) - } - return fmt.Sprintf("error: %v\n", err) - } - return "" +func (tt *telemetryTest) Close() { + tt.server.Stopper().Stop(context.Background()) + tt.diagSrv.Close() +} - case "schema": - s.ReportDiagnostics(ctx) - last := diagSrv.LastRequestData() - var buf bytes.Buffer - for i := range last.Schema { - buf.WriteString(formatTableDescriptor(&last.Schema[i])) - } - return buf.String() +func (tt *telemetryTest) RunTest( + td *datadriven.TestData, + db *gosql.DB, + reportDiags func(ctx context.Context), + sqlServer *sql.Server, +) string { + ctx := context.Background() + switch td.Cmd { + case "exec": + _, err := db.Exec(td.Input) + if err != nil { + if errors.HasAssertionFailure(err) { + td.Fatalf(tt.t, "%+v", err) + } + return fmt.Sprintf("error: %v\n", err) + } + return "" - case "feature-allowlist": - var err error - allowlist, err = makeAllowlist(strings.Split(td.Input, "\n")) - if err != nil { - td.Fatalf(t, "error parsing feature regex: %s", err) - } - return "" + case "schema": + reportDiags(ctx) + last := tt.diagSrv.LastRequestData() + var buf bytes.Buffer + for i := range last.Schema { + buf.WriteString(formatTableDescriptor(&last.Schema[i])) + } + return buf.String() - case "feature-usage", "feature-counters": - // Report diagnostics once to reset the counters. - s.ReportDiagnostics(ctx) - _, err := sqlConn.Exec(td.Input) - var buf bytes.Buffer - if err != nil { - fmt.Fprintf(&buf, "error: %v\n", err) - } - s.ReportDiagnostics(ctx) - last := diagSrv.LastRequestData() - usage := last.FeatureUsage - keys := make([]string, 0, len(usage)) - for k, v := range usage { - if v == 0 { - // Ignore zero values (shouldn't happen in practice) - continue - } - if !allowlist.Match(k) { - // Feature key not in allowlist. - continue - } - keys = append(keys, k) - } - sort.Strings(keys) - tw := tabwriter.NewWriter(&buf, 2, 1, 2, ' ', 0) - for _, k := range keys { - // Report either just the key or the key and the count. - if td.Cmd == "feature-counters" { - fmt.Fprintf(tw, "%s\t%d\n", k, usage[k]) - } else { - fmt.Fprintf(tw, "%s\n", k) - } - } - _ = tw.Flush() - return buf.String() + case "feature-allowlist": + var err error + tt.allowlist, err = makeAllowlist(strings.Split(td.Input, "\n")) + if err != nil { + td.Fatalf(tt.t, "error parsing feature regex: %s", err) + } + return "" + + case "feature-usage", "feature-counters": + // Report diagnostics once to reset the counters. + reportDiags(ctx) + _, err := db.Exec(td.Input) + var buf bytes.Buffer + if err != nil { + fmt.Fprintf(&buf, "error: %v\n", err) + } + reportDiags(ctx) + last := tt.diagSrv.LastRequestData() + usage := last.FeatureUsage + keys := make([]string, 0, len(usage)) + for k, v := range usage { + if v == 0 { + // Ignore zero values (shouldn't happen in practice) + continue + } + if !tt.allowlist.Match(k) { + // Feature key not in allowlist. + continue + } + keys = append(keys, k) + } + sort.Strings(keys) + tw := tabwriter.NewWriter(&buf, 2, 1, 2, ' ', 0) + for _, k := range keys { + // Report either just the key or the key and the count. + if td.Cmd == "feature-counters" { + fmt.Fprintf(tw, "%s\t%d\n", k, usage[k]) + } else { + fmt.Fprintf(tw, "%s\n", k) + } + } + _ = tw.Flush() + return buf.String() - case "sql-stats": - // Report diagnostics once to reset the stats. - s.SQLServer().(*sql.Server).ResetSQLStats(ctx) - s.ReportDiagnostics(ctx) + case "sql-stats": + // Report diagnostics once to reset the stats. + sqlServer.ResetSQLStats(ctx) + reportDiags(ctx) - _, err := sqlConn.Exec(td.Input) - var buf bytes.Buffer - if err != nil { - fmt.Fprintf(&buf, "error: %v\n", err) - } - s.SQLServer().(*sql.Server).ResetSQLStats(ctx) - s.ReportDiagnostics(ctx) - last := diagSrv.LastRequestData() - buf.WriteString(formatSQLStats(last.SqlStats)) - return buf.String() + _, err := db.Exec(td.Input) + var buf bytes.Buffer + if err != nil { + fmt.Fprintf(&buf, "error: %v\n", err) + } + sqlServer.ResetSQLStats(ctx) + reportDiags(ctx) + last := tt.diagSrv.LastRequestData() + buf.WriteString(formatSQLStats(last.SqlStats)) + return buf.String() - default: - td.Fatalf(t, "unknown command %s", td.Cmd) - return "" - } - }) - }) + default: + td.Fatalf(tt.t, "unknown command %s", td.Cmd) + return "" + } +} + +func (tt *telemetryTest) prepareCluster(db *gosql.DB) { + runner := sqlutils.MakeSQLRunner(db) + // Disable automatic reporting so it doesn't interfere with the test. + runner.Exec(tt.t, "SET CLUSTER SETTING diagnostics.reporting.enabled = false") + runner.Exec(tt.t, "SET CLUSTER SETTING diagnostics.reporting.send_crash_reports = false") + // Disable plan caching to get accurate counts if the same statement is + // issued multiple times. + runner.Exec(tt.t, "SET CLUSTER SETTING sql.query_cache.enabled = false") } type featureAllowlist []*regexp.Regexp diff --git a/pkg/testutils/diagutils/diag_test_server.go b/pkg/testutils/diagutils/diag_test_server.go index 50ea01380a54..5445b148b196 100644 --- a/pkg/testutils/diagutils/diag_test_server.go +++ b/pkg/testutils/diagutils/diag_test_server.go @@ -38,6 +38,9 @@ type Server struct { // RequestData stores the data provided by a diagnostics request. type RequestData struct { UUID string + TenantID string + NodeID string + SQLInstanceID string Version string LicenseType string Internal string @@ -64,6 +67,9 @@ func NewServer() *Server { data := &RequestData{ UUID: r.URL.Query().Get("uuid"), + TenantID: r.URL.Query().Get("tenantid"), + NodeID: r.URL.Query().Get("nodeid"), + SQLInstanceID: r.URL.Query().Get("sqlid"), Version: r.URL.Query().Get("version"), LicenseType: r.URL.Query().Get("licensetype"), Internal: r.URL.Query().Get("internal"), diff --git a/pkg/testutils/serverutils/test_server_shim.go b/pkg/testutils/serverutils/test_server_shim.go index edc6f4728f1a..0efccd22fd24 100644 --- a/pkg/testutils/serverutils/test_server_shim.go +++ b/pkg/testutils/serverutils/test_server_shim.go @@ -35,6 +35,7 @@ import ( "github.com/cockroachdb/cockroach/pkg/util/httputil" "github.com/cockroachdb/cockroach/pkg/util/protoutil" "github.com/cockroachdb/cockroach/pkg/util/stop" + "github.com/cockroachdb/cockroach/pkg/util/uuid" ) // TestServerInterface defines test server functionality that tests need; it is @@ -50,6 +51,10 @@ type TestServerInterface interface { // NodeID returns the ID of this node within its cluster. NodeID() roachpb.NodeID + // ClusterID returns the cluster ID as understood by this node in the + // cluster. + ClusterID() uuid.UUID + // ServingRPCAddr returns the server's advertised address. ServingRPCAddr() string @@ -203,7 +208,7 @@ type TestServerInterface interface { ReportDiagnostics(ctx context.Context) // StartTenant spawns off tenant process connecting to this TestServer. - StartTenant(params base.TestTenantArgs) (pgAddr string, httpAddr string, _ error) + StartTenant(params base.TestTenantArgs) (TestTenantInterface, error) // ScratchRange splits off a range suitable to be used as KV scratch space. // (it doesn't overlap system spans or SQL tables). @@ -211,6 +216,10 @@ type TestServerInterface interface { // Calling this multiple times is undefined (but see // TestCluster.ScratchRange() which is idempotent). ScratchRange() (roachpb.Key, error) + + // MetricsRecorder returns this node's *status.MetricsRecorder as an + // interface{}. + MetricsRecorder() interface{} } // TestServerFactory encompasses the actual implementation of the shim @@ -239,7 +248,8 @@ func StartServer( if err := server.Start(); err != nil { t.Fatalf("%+v", err) } - goDB := OpenDBConn(t, server, params, server.Stopper()) + goDB := OpenDBConn( + t, server.ServingSQLAddr(), params.UseDatabase, params.Insecure, server.Stopper()) return server, goDB, server.DB() } @@ -255,12 +265,12 @@ func NewServer(params base.TestServerArgs) TestServerInterface { // OpenDBConn sets up a gosql DB connection to the given server. func OpenDBConn( - t testing.TB, server TestServerInterface, params base.TestServerArgs, stopper *stop.Stopper, + t testing.TB, sqlAddr string, useDatabase string, insecure bool, stopper *stop.Stopper, ) *gosql.DB { pgURL, cleanupGoDB := sqlutils.PGUrl( - t, server.ServingSQLAddr(), "StartServer" /* prefix */, url.User(security.RootUser)) - pgURL.Path = params.UseDatabase - if params.Insecure { + t, sqlAddr, "StartServer" /* prefix */, url.User(security.RootUser)) + pgURL.Path = useDatabase + if insecure { pgURL.RawQuery = "sslmode=disable" } goDB, err := gosql.Open("postgres", pgURL.String()) @@ -290,27 +300,22 @@ func StartServerRaw(args base.TestServerArgs) (TestServerInterface, error) { // StartTenant starts a tenant SQL server connecting to the supplied test // server. It uses the server's stopper to shut down automatically. However, // the returned DB is for the caller to close. -func StartTenant(t testing.TB, ts TestServerInterface, params base.TestTenantArgs) *gosql.DB { - pgAddr, _, err := ts.StartTenant(params) +func StartTenant( + t testing.TB, ts TestServerInterface, params base.TestTenantArgs, +) (TestTenantInterface, *gosql.DB) { + tenant, err := ts.StartTenant(params) if err != nil { t.Fatal(err) } - pgURL, cleanupGoDB := sqlutils.PGUrl( - t, pgAddr, t.Name() /* prefix */, url.User(security.RootUser)) - - db, err := gosql.Open("postgres", pgURL.String()) - if err != nil { - t.Fatal(err) - } stopper := params.Stopper if stopper == nil { stopper = ts.Stopper() } - stopper.AddCloser(stop.CloserFn(func() { - cleanupGoDB() - })) - return db + + goDB := OpenDBConn( + t, tenant.SQLAddr(), "", false /* insecure */, stopper) + return tenant, goDB } // GetJSONProto uses the supplied client to GET the URL specified by the parameters diff --git a/pkg/testutils/serverutils/test_tenant_shim.go b/pkg/testutils/serverutils/test_tenant_shim.go new file mode 100644 index 000000000000..f5d9d0252da3 --- /dev/null +++ b/pkg/testutils/serverutils/test_tenant_shim.go @@ -0,0 +1,37 @@ +// Copyright 2016 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. +// +// This file provides generic interfaces that allow tests to set up test tenants +// without importing the server package (avoiding circular dependencies). This + +package serverutils + +import "github.com/cockroachdb/cockroach/pkg/base" + +// TestTenantInterface defines SQL-only tenant functionality that tests need; it +// is implemented by server.TestTenant. +type TestTenantInterface interface { + // SQLInstanceID is the ephemeral ID assigned to a running instance of the + // SQLServer. Each tenant can have zero or more running SQLServer instances. + SQLInstanceID() base.SQLInstanceID + + // SQLAddr returns the tenant's SQL address. + SQLAddr() string + + // HTTPAddr returns the tenant's http address. + HTTPAddr() string + + // PGServer returns the tenant's *pgwire.Server as an interface{}. + PGServer() interface{} + + // DiagnosticsReporter returns the tenant's *diagnostics.Reporter as an + // interface{}. + DiagnosticsReporter() interface{} +} diff --git a/pkg/testutils/testcluster/testcluster.go b/pkg/testutils/testcluster/testcluster.go index 7de3b28c9bd6..77931209de96 100644 --- a/pkg/testutils/testcluster/testcluster.go +++ b/pkg/testutils/testcluster/testcluster.go @@ -394,7 +394,8 @@ func (tc *TestCluster) StartServer( t.Fatalf("%+v", err) } - dbConn := serverutils.OpenDBConn(t, server, serverArgs, server.Stopper()) + dbConn := serverutils.OpenDBConn( + t, server.ServingSQLAddr(), serverArgs.UseDatabase, serverArgs.Insecure, server.Stopper()) tc.mu.Lock() defer tc.mu.Unlock()