Skip to content

Commit

Permalink
Browse files Browse the repository at this point in the history
59992: tracing: propagate non-recording spans across rpc boundaries r=knz,irfansharif,raduberinde a=tbg

We were previously propagating RPCs across boundaries only when they
were verbose. We now propagate any span (except the noop span)
regardless of current verbosity.
This ensures that SQL (which always creates real, non-verbose spans by
default) can trace the entirety of its operations.

This enables the background collection of contention metadata in 21.1
and more generally, the idea of always-on tracing. Note that we had
previously considered always-on tracing to mean avoiding the noop span
altogether; due to memory retention issues[1][2] encountered we are
taking a step back from that idea for the 21.1 release.

[1]: cockroachdb#59370
[2]: cockroachdb#59431

Release note: None


Co-authored-by: Tobias Grieger <tobias.b.grieger@gmail.com>
Co-authored-by: Tobias Schottdorf <tobias.schottdorf@gmail.com>
  • Loading branch information
3 people committed Feb 17, 2021
2 parents c2ef500 + f93e92b commit d12d281
Show file tree
Hide file tree
Showing 36 changed files with 1,010 additions and 90 deletions.
2 changes: 1 addition & 1 deletion docs/generated/settings/settings.html
Original file line number Diff line number Diff line change
Expand Up @@ -101,6 +101,6 @@
<tr><td><code>trace.debug.enable</code></td><td>boolean</td><td><code>false</code></td><td>if set, traces for recent requests can be seen at https://<ui>/debug/requests</td></tr>
<tr><td><code>trace.lightstep.token</code></td><td>string</td><td><code></code></td><td>if set, traces go to Lightstep using this token</td></tr>
<tr><td><code>trace.zipkin.collector</code></td><td>string</td><td><code></code></td><td>if set, traces go to the given Zipkin instance (example: '127.0.0.1:9411'); ignored if trace.lightstep.token is set</td></tr>
<tr><td><code>version</code></td><td>version</td><td><code>20.2-26</code></td><td>set the active cluster version in the format '<major>.<minor>'</td></tr>
<tr><td><code>version</code></td><td>version</td><td><code>20.2-28</code></td><td>set the active cluster version in the format '<major>.<minor>'</td></tr>
</tbody>
</table>
2 changes: 2 additions & 0 deletions docs/generated/sql/functions.md
Original file line number Diff line number Diff line change
Expand Up @@ -2648,6 +2648,8 @@ SELECT * FROM crdb_internal.check_consistency(true, ‘\x02’, ‘\x04’)</p>
</span></td></tr>
<tr><td><a name="crdb_internal.set_vmodule"></a><code>crdb_internal.set_vmodule(vmodule_string: <a href="string.html">string</a>) &rarr; <a href="int.html">int</a></code></td><td><span class="funcdesc"><p>Set the equivalent of the <code>--vmodule</code> flag on the gateway node processing this request; it affords control over the logging verbosity of different files. Example syntax: <code>crdb_internal.set_vmodule('recordio=2,file=1,gfs*=3')</code>. Reset with: <code>crdb_internal.set_vmodule('')</code>. Raising the verbosity can severely affect performance.</p>
</span></td></tr>
<tr><td><a name="crdb_internal.trace_id"></a><code>crdb_internal.trace_id() &rarr; <a href="int.html">int</a></code></td><td><span class="funcdesc"><p>Returns the current trace ID or an error if no trace is open.</p>
</span></td></tr>
<tr><td><a name="current_database"></a><code>current_database() &rarr; <a href="string.html">string</a></code></td><td><span class="funcdesc"><p>Returns the current database.</p>
</span></td></tr>
<tr><td><a name="current_schema"></a><code>current_schema() &rarr; <a href="string.html">string</a></code></td><td><span class="funcdesc"><p>Returns the current schema.</p>
Expand Down
8 changes: 8 additions & 0 deletions pkg/clusterversion/cockroach_versions.go
Original file line number Diff line number Diff line change
Expand Up @@ -237,6 +237,10 @@ const (
PostTruncatedAndRangeAppliedStateMigration
// SeparatedIntents allows the writing of separated intents/locks.
SeparatedIntents
// TracingVerbosityIndependentSemantics marks a change in which trace spans
// are propagated across RPC boundaries independently of their verbosity setting.
// This requires a version gate this violates implicit assumptions in v20.2.
TracingVerbosityIndependentSemantics

// Step (1): Add new versions here.
)
Expand Down Expand Up @@ -395,6 +399,10 @@ var versionsSingleton = keyedVersions([]keyedVersion{
Key: SeparatedIntents,
Version: roachpb.Version{Major: 20, Minor: 2, Internal: 26},
},
{
Key: TracingVerbosityIndependentSemantics,
Version: roachpb.Version{Major: 20, Minor: 2, Internal: 28},
},
// Step (2): Add new versions here.
})

Expand Down
5 changes: 3 additions & 2 deletions pkg/clusterversion/key_string.go

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

17 changes: 17 additions & 0 deletions pkg/cmd/roachtest/versionupgrade.go
Original file line number Diff line number Diff line change
Expand Up @@ -175,6 +175,14 @@ func runVersionUpgrade(ctx context.Context, t *test, c *cluster, buildVersion ve
testFeaturesStep,
// schemaChangeStep,
backupStep,
// Turn tracing on globally to give it a fighting chance at exposing any
// crash-inducing incompatibilities or horrendous memory leaks. (It won't
// catch most memory leaks since this test doesn't run for too long or does
// too much work). Then, run the previous tests again.
enableTracingGloballyStep,
testFeaturesStep,
// schemaChangeStep,
backupStep,
)

u.run(ctx, t)
Expand Down Expand Up @@ -354,6 +362,15 @@ func binaryUpgradeStep(nodes nodeListOption, newVersion string) versionStep {
}
}

func enableTracingGloballyStep(ctx context.Context, t *test, u *versionUpgradeTest) {
db := u.conn(ctx, t, 1)
// NB: this enables net/trace, and as a side effect creates verbose trace spans everywhere.
_, err := db.ExecContext(ctx, `SET CLUSTER SETTING trace.debug.enable = $1`, true)
if err != nil {
t.Fatal(err)
}
}

func preventAutoUpgradeStep(node int) versionStep {
return func(ctx context.Context, t *test, u *versionUpgradeTest) {
db := u.conn(ctx, t, node)
Expand Down
6 changes: 6 additions & 0 deletions pkg/jobs/jobs.go
Original file line number Diff line number Diff line change
Expand Up @@ -915,6 +915,12 @@ func (sj *StartableJob) CleanupOnRollback(ctx context.Context) error {
"cannot call CleanupOnRollback for a StartableJob with a non-finalized transaction")
}
sj.registry.unregister(*sj.ID())
if sj.span != nil {
sj.span.Finish()
}
if sj.cancel != nil {
sj.cancel()
}
return nil
}

Expand Down
8 changes: 8 additions & 0 deletions pkg/jobs/registry.go
Original file line number Diff line number Diff line change
Expand Up @@ -517,6 +517,14 @@ func (r *Registry) CreateStartableJobWithTxn(
_, span := tracing.ForkSpan(ctx, "job")
if span != nil {
resumerCtx = tracing.ContextWithSpan(resumerCtx, span)

// This trace span unfortunately is sometimes never finished.
// As a hack/workaround, finish it now so that it leaves the
// tracer registry.
//
// Remove this when this issue is fixed:
// https://github.com/cockroachdb/cockroach/issues/60671
span.Finish()
}

if r.startUsingSQLLivenessAdoption(ctx) {
Expand Down
3 changes: 3 additions & 0 deletions pkg/kv/kvnemesis/kvnemesis_test.go
Original file line number Diff line number Diff line change
Expand Up @@ -70,6 +70,9 @@ func TestKVNemesisMultiNode(t *testing.T) {
sqlDBs[i] = tc.ServerConn(i)
}
sqlutils.MakeSQLRunner(sqlDBs[0]).Exec(t, `SET CLUSTER SETTING kv.rangefeed.enabled = true`)
// Turn net/trace on, which results in real trace spans created throughout.
// This gives kvnemesis a chance to hit NPEs related to tracing.
sqlutils.MakeSQLRunner(sqlDBs[0]).Exec(t, `SET CLUSTER SETTING trace.debug.enable = true`)

config := NewDefaultConfig()
config.NumNodes, config.NumReplicas = numNodes, 3
Expand Down
7 changes: 7 additions & 0 deletions pkg/kv/kvserver/replica_raft.go
Original file line number Diff line number Diff line change
Expand Up @@ -129,6 +129,13 @@ func (r *Replica) evalAndPropose(
// Fork the proposal's context span so that the proposal's context
// can outlive the original proposer's context.
proposal.ctx, proposal.sp = tracing.ForkSpan(ctx, "async consensus")
{
// This span sometimes leaks. Disable it for the time being.
//
// Tracked in: https://github.com/cockroachdb/cockroach/issues/60677
proposal.sp.Finish()
proposal.sp = nil
}

// Signal the proposal's response channel immediately.
reply := *proposal.Local.Reply
Expand Down
4 changes: 4 additions & 0 deletions pkg/settings/cluster/cluster_settings.go
Original file line number Diff line number Diff line change
Expand Up @@ -121,6 +121,10 @@ func MakeClusterSettings() *Settings {
sv.Init(s.Version)

s.Tracer = tracing.NewTracer()
s.Tracer.TracingVerbosityIndependentSemanticsIsActive = func() bool {
return s.Version.IsActive(context.Background(),
clusterversion.TracingVerbosityIndependentSemantics)
}
s.Tracer.Configure(sv)

return s
Expand Down
7 changes: 4 additions & 3 deletions pkg/sql/conn_executor.go
Original file line number Diff line number Diff line change
Expand Up @@ -592,9 +592,10 @@ func (s *Server) newConnExecutor(
nodeIDOrZero: nodeIDOrZero,
clock: s.cfg.Clock,
// Future transaction's monitors will inherits from sessionRootMon.
connMon: sessionRootMon,
tracer: s.cfg.AmbientCtx.Tracer,
settings: s.cfg.Settings,
connMon: sessionRootMon,
tracer: s.cfg.AmbientCtx.Tracer,
settings: s.cfg.Settings,
execTestingKnobs: s.GetExecutorConfig().TestingKnobs,
},
memMetrics: memMetrics,
planner: planner{execCfg: s.cfg, alloc: &rowenc.DatumAlloc{}},
Expand Down
18 changes: 18 additions & 0 deletions pkg/sql/crdb_internal.go
Original file line number Diff line number Diff line change
Expand Up @@ -66,6 +66,7 @@ import (
"github.com/cockroachdb/cockroach/pkg/util/timeutil"
"github.com/cockroachdb/cockroach/pkg/util/tracing"
"github.com/cockroachdb/errors"
ptypes "github.com/gogo/protobuf/types"
)

// CrdbInternalName is the name of the crdb_internal schema.
Expand Down Expand Up @@ -1148,9 +1149,18 @@ CREATE TABLE crdb_internal.node_inflight_trace_spans (
duration INTERVAL, -- The span's duration, measured from start to Finish().
-- A span whose recording is collected before it's finished will
-- have the duration set as the "time of collection - start time".
num_payloads INT NOT NULL, -- The number of structured payloads in this span.
operation STRING NULL -- The span's operation.
)`,
populate: func(ctx context.Context, p *planner, _ *dbdesc.Immutable, addRow func(...tree.Datum) error) error {
hasAdmin, err := p.HasAdminRole(ctx)
if err != nil {
return err
}
if !hasAdmin {
return pgerror.Newf(pgcode.InsufficientPrivilege,
"only users with the admin role are allowed to read crdb_internal.node_inflight_trace_spans")
}
return p.ExecCfg().Settings.Tracer.VisitSpans(func(span *tracing.Span) error {
for _, rec := range span.GetRecording() {
traceID := rec.TraceID
Expand All @@ -1167,7 +1177,14 @@ CREATE TABLE crdb_internal.node_inflight_trace_spans (
spanDuration := rec.Duration
operation := rec.Operation

var numStructured int
rec.Structured(func(any *ptypes.Any) {
numStructured++
})

if err := addRow(
// TODO(angelapwen): we're casting uint64s to int64 here,
// is that ok?
tree.NewDInt(tree.DInt(traceID)),
tree.NewDInt(tree.DInt(parentSpanID)),
tree.NewDInt(tree.DInt(spanID)),
Expand All @@ -1178,6 +1195,7 @@ CREATE TABLE crdb_internal.node_inflight_trace_spans (
duration.MakeDuration(spanDuration.Nanoseconds(), 0, 0),
types.DefaultIntervalTypeMetadata,
),
tree.NewDInt(tree.DInt(numStructured)),
tree.NewDString(operation),
); err != nil {
return err
Expand Down
4 changes: 3 additions & 1 deletion pkg/sql/distsql_running.go
Original file line number Diff line number Diff line change
Expand Up @@ -670,7 +670,9 @@ func (r *DistSQLReceiver) Push(
r.contendedQueryMetric = nil
}
if err := r.contentionRegistry.AddContentionEvent(ev); err != nil {
r.resultWriter.SetError(errors.Wrap(err, "unable to add contention event to registry"))
// TODO(asubiotto): see https://github.com/cockroachdb/cockroach/issues/60669
// r.resultWriter.SetError(errors.Wrap(err, "unable to add contention event to registry"))
_ = err
}
})
}
Expand Down
6 changes: 6 additions & 0 deletions pkg/sql/exec_util.go
Original file line number Diff line number Diff line change
Expand Up @@ -921,6 +921,12 @@ type ExecutorTestingKnobs struct {
// DeterministicExplainAnalyze, if set, will result in overriding fields in
// EXPLAIN ANALYZE (PLAN) that can vary between runs (like elapsed times).
DeterministicExplainAnalyze bool

// Pretend59315IsFixed pretends that this issue is fixed:
// https://github.com/cockroachdb/cockroach/issues/59315
// which means that we don't need the WithBypassRegistry option
// in resetForNewSQLTxn.
Pretend59315IsFixed bool
}

// PGWireTestingKnobs contains knobs for the pgwire module.
Expand Down
11 changes: 10 additions & 1 deletion pkg/sql/logictest/logic.go
Original file line number Diff line number Diff line change
Expand Up @@ -465,7 +465,8 @@ type testClusterConfig struct {
isCCLConfig bool
// localities is set if nodes should be set to a particular locality.
// Nodes are 1-indexed.
localities map[int]roachpb.Locality
localities map[int]roachpb.Locality
pretend59315IsFixed bool
}

const threeNodeTenantConfigName = "3node-tenant"
Expand Down Expand Up @@ -562,6 +563,13 @@ var logicTestConfigs = []testClusterConfig{
overrideDistSQLMode: "on",
overrideAutoStats: "false",
},
{
name: "5node-pretend59315IsFixed",
numNodes: 5,
overrideDistSQLMode: "on",
overrideAutoStats: "false",
pretend59315IsFixed: true,
},
{
name: "5node-metadata",
numNodes: 5,
Expand Down Expand Up @@ -1332,6 +1340,7 @@ func (t *logicTest) newCluster(serverArgs TestServerArgs) {
},
SQLExecutor: &sql.ExecutorTestingKnobs{
DeterministicExplainAnalyze: true,
Pretend59315IsFixed: t.cfg.pretend59315IsFixed,
},
},
ClusterName: "testclustername",
Expand Down
53 changes: 24 additions & 29 deletions pkg/sql/logictest/testdata/logic_test/contention_event
Original file line number Diff line number Diff line change
@@ -1,3 +1,5 @@
# LogicTest: 5node-pretend59315IsFixed
#
# Verify that ContentionEvents are emitted. This is mostly a sanity check - look
# for the datadriven tests in `pkg/kv/kvserver/concurrency` for the actual events
# that do get emitted in various contention scenarios.
Expand All @@ -6,8 +8,8 @@ statement ok
GRANT ADMIN TO testuser

statement ok
CREATE TABLE kv (k VARCHAR PRIMARY KEY, v VARCHAR)

CREATE TABLE kv (k VARCHAR PRIMARY KEY, v VARCHAR);
ALTER TABLE kv SPLIT AT VALUES ('b'), ('d'), ('q'), ('z');

query TT
SELECT * FROM kv
Expand All @@ -23,41 +25,34 @@ INSERT INTO kv VALUES('k', 'v')

user root

# Scan all ranges of the table (note that we have intentionally
# split it into at least six ranges). This is better than a point
# lookup in that it gives tracing more of a chance to get things
# wrong due to DistSender parallelism.
statement ok
SET tracing=on

statement ok
SET TRACING=on;
BEGIN;
SET TRANSACTION PRIORITY HIGH;
INSERT INTO kv VALUES('k', 'my v');
COMMIT;
SET TRACING=off;

# Check for the message emitted by (roachpb.ContentionEvent).String() as proof that a ContentionEvent
# payload was added to the trace.
#
# NB: if the matcher here ever gets updated, make sure to update throughout this file.
query I
SELECT count(message) FROM [ SHOW TRACE FOR SESSION ] WHERE message LIKE '%conflicted with % on % for %'
----
1
SELECT * FROM kv;

# Clean up.
user testuser

statement ok
ROLLBACK

# Do a straightforward non-conflicting insert; we want to check that there's *no* ContentionEvent
# emitted for those. There's no reason to believe it would be - this is a pure sanity check.

statement ok
SET TRACING=on;
INSERT INTO kv VALUES ('l', 'lll');
SET TRACING=off;
user root

query I
SELECT count(message) FROM [ SHOW TRACE FOR SESSION ] WHERE message LIKE '%conflicted with % on % for %'
# Check that the number of payloads in the open trace is at least 1.
# TODO(angelapwen): when we have a way to pull particular payloads for
# a trace, we should verify that we're seeing the right contention event.
# As is, the payloads could be something else (though we verified manually
# and there is a contention event).
#
# NB: the contention event is not in our trace span but in one of its
# children, so it wouldn't be found if we filtered by the trace span ID.
#
# NB: this needs the 5node-pretend59315 config because otherwise the span is not
# tracked.
query B
SELECT count(num_payloads) > 0 FROM crdb_internal.node_inflight_trace_spans WHERE trace_id = crdb_internal.trace_id();
----
0
true
7 changes: 5 additions & 2 deletions pkg/sql/logictest/testdata/logic_test/crdb_internal
Original file line number Diff line number Diff line change
Expand Up @@ -250,10 +250,10 @@ SELECT * FROM crdb_internal.zones WHERE false
zone_id subzone_id target range_name database_name table_name index_name partition_name
raw_config_yaml raw_config_sql raw_config_protobuf full_config_yaml full_config_sql

query IIIIBTTT colnames
query IIIIBTTIT colnames
SELECT * FROM crdb_internal.node_inflight_trace_spans WHERE span_id < 0
----
trace_id parent_span_id span_id goroutine_id finished start_time duration operation
trace_id parent_span_id span_id goroutine_id finished start_time duration num_payloads operation

query ITTTTITTTTTTTTTI colnames
SELECT * FROM crdb_internal.ranges WHERE range_id < 0
Expand Down Expand Up @@ -477,6 +477,9 @@ select * from crdb_internal.kv_store_status
query error pq: only users with the admin role are allowed to read crdb_internal.gossip_alerts
select * from crdb_internal.gossip_alerts

query error pq: only users with the admin role are allowed to read crdb_internal.node_inflight_trace_spans
select * from crdb_internal.node_inflight_trace_spans

# Anyone can see the executable version.
query T
select regexp_replace(crdb_internal.node_executable_version()::string, '(-\d+)?$', '');
Expand Down
4 changes: 2 additions & 2 deletions pkg/sql/logictest/testdata/logic_test/crdb_internal_tenant
Original file line number Diff line number Diff line change
Expand Up @@ -262,10 +262,10 @@ SELECT * FROM crdb_internal.zones WHERE false
zone_id subzone_id target range_name database_name table_name index_name partition_name
raw_config_yaml raw_config_sql raw_config_protobuf full_config_yaml full_config_sql

query IIIIBTTT colnames
query IIIIBTTIT colnames
SELECT * FROM crdb_internal.node_inflight_trace_spans WHERE span_id < 0
----
trace_id parent_span_id span_id goroutine_id finished start_time duration operation
trace_id parent_span_id span_id goroutine_id finished start_time duration num_payloads operation

statement error not fully contained in tenant keyspace
SELECT * FROM crdb_internal.ranges WHERE range_id < 0
Expand Down

0 comments on commit d12d281

Please sign in to comment.