-
Notifications
You must be signed in to change notification settings - Fork 3.8k
/
tracer.go
807 lines (721 loc) · 26.2 KB
/
tracer.go
1
2
3
4
5
6
7
8
9
10
11
12
13
14
15
16
17
18
19
20
21
22
23
24
25
26
27
28
29
30
31
32
33
34
35
36
37
38
39
40
41
42
43
44
45
46
47
48
49
50
51
52
53
54
55
56
57
58
59
60
61
62
63
64
65
66
67
68
69
70
71
72
73
74
75
76
77
78
79
80
81
82
83
84
85
86
87
88
89
90
91
92
93
94
95
96
97
98
99
100
101
102
103
104
105
106
107
108
109
110
111
112
113
114
115
116
117
118
119
120
121
122
123
124
125
126
127
128
129
130
131
132
133
134
135
136
137
138
139
140
141
142
143
144
145
146
147
148
149
150
151
152
153
154
155
156
157
158
159
160
161
162
163
164
165
166
167
168
169
170
171
172
173
174
175
176
177
178
179
180
181
182
183
184
185
186
187
188
189
190
191
192
193
194
195
196
197
198
199
200
201
202
203
204
205
206
207
208
209
210
211
212
213
214
215
216
217
218
219
220
221
222
223
224
225
226
227
228
229
230
231
232
233
234
235
236
237
238
239
240
241
242
243
244
245
246
247
248
249
250
251
252
253
254
255
256
257
258
259
260
261
262
263
264
265
266
267
268
269
270
271
272
273
274
275
276
277
278
279
280
281
282
283
284
285
286
287
288
289
290
291
292
293
294
295
296
297
298
299
300
301
302
303
304
305
306
307
308
309
310
311
312
313
314
315
316
317
318
319
320
321
322
323
324
325
326
327
328
329
330
331
332
333
334
335
336
337
338
339
340
341
342
343
344
345
346
347
348
349
350
351
352
353
354
355
356
357
358
359
360
361
362
363
364
365
366
367
368
369
370
371
372
373
374
375
376
377
378
379
380
381
382
383
384
385
386
387
388
389
390
391
392
393
394
395
396
397
398
399
400
401
402
403
404
405
406
407
408
409
410
411
412
413
414
415
416
417
418
419
420
421
422
423
424
425
426
427
428
429
430
431
432
433
434
435
436
437
438
439
440
441
442
443
444
445
446
447
448
449
450
451
452
453
454
455
456
457
458
459
460
461
462
463
464
465
466
467
468
469
470
471
472
473
474
475
476
477
478
479
480
481
482
483
484
485
486
487
488
489
490
491
492
493
494
495
496
497
498
499
500
501
502
503
504
505
506
507
508
509
510
511
512
513
514
515
516
517
518
519
520
521
522
523
524
525
526
527
528
529
530
531
532
533
534
535
536
537
538
539
540
541
542
543
544
545
546
547
548
549
550
551
552
553
554
555
556
557
558
559
560
561
562
563
564
565
566
567
568
569
570
571
572
573
574
575
576
577
578
579
580
581
582
583
584
585
586
587
588
589
590
591
592
593
594
595
596
597
598
599
600
601
602
603
604
605
606
607
608
609
610
611
612
613
614
615
616
617
618
619
620
621
622
623
624
625
626
627
628
629
630
631
632
633
634
635
636
637
638
639
640
641
642
643
644
645
646
647
648
649
650
651
652
653
654
655
656
657
658
659
660
661
662
663
664
665
666
667
668
669
670
671
672
673
674
675
676
677
678
679
680
681
682
683
684
685
686
687
688
689
690
691
692
693
694
695
696
697
698
699
700
701
702
703
704
705
706
707
708
709
710
711
712
713
714
715
716
717
718
719
720
721
722
723
724
725
726
727
728
729
730
731
732
733
734
735
736
737
738
739
740
741
742
743
744
745
746
747
748
749
750
751
752
753
754
755
756
757
758
759
760
761
762
763
764
765
766
767
768
769
770
771
772
773
774
775
776
777
778
779
780
781
782
783
784
785
786
787
788
789
790
791
792
793
794
795
796
797
798
799
800
801
802
803
804
805
806
807
// 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 tracing
import (
"context"
"fmt"
"math/rand"
"strconv"
"strings"
"sync"
"sync/atomic"
"time"
"unsafe"
"github.com/cockroachdb/cockroach/pkg/settings"
"github.com/cockroachdb/cockroach/pkg/util/envutil"
"github.com/cockroachdb/cockroach/pkg/util/iterutil"
"github.com/cockroachdb/cockroach/pkg/util/syncutil"
"github.com/cockroachdb/errors"
"github.com/cockroachdb/logtags"
opentracing "github.com/opentracing/opentracing-go"
"github.com/petermattis/goid"
"golang.org/x/net/trace"
)
// verboseTracingBaggageKey is set as Baggage on traces which are used for verbose tracing,
// meaning that a) spans derived from this one will not be no-op spans and b) they will
// start recording.
//
// This is "sb" for historical reasons; this concept used to be called "[S]now[b]all" tracing
// and since this string goes on the wire, it's a hassle to change it now.
const verboseTracingBaggageKey = "sb"
const (
// maxLogBytesPerSpan limits the size of logs in a span; use a comfortable
// limit.
maxLogBytesPerSpan = 10 * (1 << 10) // 10 KiB
// maxStructuredBytesPerSpan limits the size of structured events in a span;
// use a comfortable limit.
maxStructuredBytesPerSpan = 1 << 10 // 1 KiB
// maxChildrenPerSpan limits the number of (direct) child spans in a Span.
maxChildrenPerSpan = 1000
// maxLogsPerSpanExternal limits the number of logs in a Span for external
// tracers (net/trace, lightstep); use a comfortable limit.
maxLogsPerSpanExternal = 1000
)
// These constants are used to form keys to represent tracing context
// information in carriers supporting opentracing.HTTPHeaders format.
const (
prefixTracerState = "crdb-tracer-"
prefixBaggage = "crdb-baggage-"
// prefixShadow is prepended to the keys for the context of the shadow tracer
// (e.g. LightStep).
prefixShadow = "crdb-shadow-"
fieldNameTraceID = prefixTracerState + "traceid"
fieldNameSpanID = prefixTracerState + "spanid"
// fieldNameShadow is the name of the shadow tracer.
fieldNameShadowType = prefixTracerState + "shadowtype"
)
var enableNetTrace = settings.RegisterBoolSetting(
"trace.debug.enable",
"if set, traces for recent requests can be seen at https://<ui>/debug/requests",
false,
).WithPublic()
var lightstepToken = settings.RegisterStringSetting(
"trace.lightstep.token",
"if set, traces go to Lightstep using this token",
envutil.EnvOrDefaultString("COCKROACH_TEST_LIGHTSTEP_TOKEN", ""),
).WithPublic()
// ZipkinCollector is the cluster setting that specifies the Zipkin instance
// to send traces to, if any.
var ZipkinCollector = settings.RegisterStringSetting(
"trace.zipkin.collector",
"if set, traces go to the given Zipkin instance (example: '127.0.0.1:9411'); ignored if trace.lightstep.token is set",
envutil.EnvOrDefaultString("COCKROACH_TEST_ZIPKIN_COLLECTOR", ""),
).WithPublic()
// Tracer is our own custom implementation of opentracing.Tracer. It supports:
//
// - forwarding events to x/net/trace instances
//
// - recording traces. Recording is started automatically for spans that have
// the verboseTracingBaggageKey baggage and can be started explicitly as well. Recorded
// events can be retrieved at any time.
//
// - lightstep traces. This is implemented by maintaining a "shadow" lightstep
// Span inside each of our spans.
//
// Even when tracing is disabled, we still use this Tracer (with x/net/trace and
// lightstep disabled) because of its recording capability (verbose tracing needs
// to work in all cases).
//
// Tracer is currently stateless so we could have a single instance; however,
// this won't be the case if the cluster settings move away from using global
// state.
type Tracer struct {
// Preallocated noopSpan, used to avoid creating spans when we are not using
// x/net/trace or lightstep and we are not recording.
noopSpan *Span
// True if tracing to the debug/requests endpoint. Accessed via t.useNetTrace().
_useNetTrace int32 // updated atomically
// Pointer to shadowTracer, if using one.
shadowTracer unsafe.Pointer
// activeSpans is a map that references all non-Finish'ed local root spans,
// i.e. those for which no WithLocalParent(<non-nil>) option was supplied.
// It also elides spans created using WithBypassRegistry.
// The map is keyed on the span ID, which is deterministically unique.
//
// In normal operation, a local root Span is inserted on creation and
// removed on .Finish().
//
// The map can be introspected by `Tracer.VisitSpans`. A Span can also be
// retrieved from its ID by `Tracer.GetActiveSpanFromID`.
activeSpans struct {
// NB: it might be tempting to use a sync.Map here, but
// this incurs an allocation per Span (sync.Map does
// not use a sync.Pool for its internal *entry type).
//
// The bare map approach is essentially allocation-free once the map
// has grown to accommodate the usual number of active local root spans,
// and the critical sections of the mutex are very small.
syncutil.Mutex
m map[uint64]*Span
}
// TracingVerbosityIndependentSemanticsIsActive is really
// version.IsActive(TracingVerbosityIndependentSemanticsIsActive)
// but gets injected this way to avoid import cycles. It defaults
// to a function that returns `true`.
TracingVerbosityIndependentSemanticsIsActive func() bool
includeAsyncSpansInRecordings bool // see TestingIncludeAsyncSpansInRecordings
}
// NewTracer creates a Tracer. It initially tries to run with minimal overhead
// and collects essentially nothing; use Configure() to enable various tracing
// backends.
func NewTracer() *Tracer {
t := &Tracer{}
t.activeSpans.m = make(map[uint64]*Span)
t.TracingVerbosityIndependentSemanticsIsActive = func() bool { return true }
// The noop span is marked as finished so that even in the case of a bug,
// it won't soak up data.
t.noopSpan = &Span{numFinishCalled: 1, i: spanInner{tracer: t}}
return t
}
// Configure sets up the Tracer according to the cluster settings (and keeps
// it updated if they change).
func (t *Tracer) Configure(sv *settings.Values) {
reconfigure := func() {
if lsToken := lightstepToken.Get(sv); lsToken != "" {
t.setShadowTracer(createLightStepTracer(lsToken))
} else if zipkinAddr := ZipkinCollector.Get(sv); zipkinAddr != "" {
t.setShadowTracer(createZipkinTracer(zipkinAddr))
} else {
t.setShadowTracer(nil, nil)
}
var nt int32
if enableNetTrace.Get(sv) {
nt = 1
}
atomic.StoreInt32(&t._useNetTrace, nt)
}
reconfigure()
enableNetTrace.SetOnChange(sv, reconfigure)
lightstepToken.SetOnChange(sv, reconfigure)
ZipkinCollector.SetOnChange(sv, reconfigure)
}
// HasExternalSink returns whether the tracer is configured to report
// to an external tracing collector.
func (t *Tracer) HasExternalSink() bool {
return t.getShadowTracer() != nil || t.useNetTrace()
}
func (t *Tracer) useNetTrace() bool {
return atomic.LoadInt32(&t._useNetTrace) != 0
}
// Close cleans up any resources associated with a Tracer.
func (t *Tracer) Close() {
// Clean up any shadow tracer.
t.setShadowTracer(nil, nil)
}
func (t *Tracer) setShadowTracer(manager shadowTracerManager, tr opentracing.Tracer) {
var shadow *shadowTracer
if manager != nil && tr != nil {
shadow = &shadowTracer{
Tracer: tr,
manager: manager,
}
}
if old := atomic.SwapPointer(&t.shadowTracer, unsafe.Pointer(shadow)); old != nil {
(*shadowTracer)(old).Close()
}
}
func (t *Tracer) getShadowTracer() *shadowTracer {
return (*shadowTracer)(atomic.LoadPointer(&t.shadowTracer))
}
// StartSpan starts a Span. See SpanOption for details.
func (t *Tracer) StartSpan(operationName string, os ...SpanOption) *Span {
_, sp := t.StartSpanCtx(noCtx, operationName, os...)
return sp
}
// StartSpanCtx starts a Span and returns it alongside a wrapping Context
// derived from the supplied Context. Any log tags found in the supplied
// Context are propagated into the Span; this behavior can be modified by
// passing WithLogTags explicitly.
//
// See SpanOption for other options that can be passed.
func (t *Tracer) StartSpanCtx(
ctx context.Context, operationName string, os ...SpanOption,
) (context.Context, *Span) {
// NB: apply takes and returns a value to avoid forcing
// `opts` on the heap here.
var opts spanOptions
for _, o := range os {
opts = o.apply(opts)
}
return t.startSpanGeneric(ctx, operationName, opts)
}
// AlwaysTrace returns true if operations should be traced regardless of the
// context.
func (t *Tracer) AlwaysTrace() bool {
shadowTracer := t.getShadowTracer()
return t.useNetTrace() || shadowTracer != nil
}
// startSpanGeneric is the implementation of StartSpanCtx and StartSpan. In
// the latter case, ctx == noCtx and the returned Context is the supplied one;
// otherwise the returned Context embeds the returned Span.
func (t *Tracer) startSpanGeneric(
ctx context.Context, opName string, opts spanOptions,
) (context.Context, *Span) {
if opts.RefType != opentracing.ChildOfRef && opts.RefType != opentracing.FollowsFromRef {
panic(fmt.Sprintf("unexpected RefType %v", opts.RefType))
}
if opts.Parent != nil {
if opts.RemoteParent != nil {
panic("can't specify both Parent and RemoteParent")
}
}
if opts.LogTags == nil {
opts.LogTags = logtags.FromContext(ctx)
}
// Are we tracing everything, or have a parent, or want a real span? Then
// we create a real trace span. In all other cases, a noop span will do.
if !(t.AlwaysTrace() || opts.parentTraceID() != 0 || opts.ForceRealSpan) {
return maybeWrapCtx(ctx, nil /* octx */, t.noopSpan)
}
if opts.LogTags == nil && opts.Parent != nil && !opts.Parent.i.isNoop() {
// If no log tags are specified in the options, use the parent
// span's, if any. This behavior is the reason logTags are
// fundamentally different from tags, which are strictly per span,
// for better or worse.
opts.LogTags = opts.Parent.i.crdb.logTags
}
startTime := time.Now()
// First, create any external spans that we may need (opentracing, net/trace).
// We do this early so that they are available when we construct the main Span,
// which makes it easier to avoid one-offs when populating the tags and baggage
// items for the top-level Span.
var ot otSpan
{
shadowTr := t.getShadowTracer()
// Make sure not to derive spans created using an old
// shadow tracer via a new one.
typ1, ok1 := opts.shadowTrTyp() // old
typ2, ok2 := shadowTr.Type() // new
// If both are set and don't agree, ignore shadow tracer
// for the new span to avoid compat issues between the
// two underlying tracers.
if ok2 && (!ok1 || typ1 == typ2) {
var shadowCtx opentracing.SpanContext
if opts.Parent != nil && opts.Parent.i.ot.shadowSpan != nil {
shadowCtx = opts.Parent.i.ot.shadowSpan.Context()
}
ot = makeShadowSpan(shadowTr, shadowCtx, opts.RefType, opName, startTime)
// If LogTags are given, pass them as tags to the shadow span.
// Regular tags are populated later, via the top-level Span.
if opts.LogTags != nil {
setLogTags(opts.LogTags.Get(), func(remappedKey string, tag *logtags.Tag) {
_ = ot.shadowSpan.SetTag(remappedKey, tag.Value())
})
}
}
}
var netTr trace.Trace
if t.useNetTrace() {
netTr = trace.New("tracing", opName)
netTr.SetMaxEvents(maxLogsPerSpanExternal)
// If LogTags are given, pass them as tags to the shadow span.
// Regular tags are populated later, via the top-level Span.
if opts.LogTags != nil {
setLogTags(opts.LogTags.Get(), func(remappedKey string, tag *logtags.Tag) {
netTr.LazyPrintf("%s:%v", remappedKey, tag)
})
}
}
// Now that `ot` and `netTr` are properly set up, make the Span.
traceID := opts.parentTraceID()
if traceID == 0 {
// NB: it is tempting to use the traceID and spanID from the
// possibly populated otSpan in this case, but the opentracing
// interface doesn't give us a good way to extract these.
traceID = uint64(rand.Int63())
}
spanID := uint64(rand.Int63())
goroutineID := uint64(goid.Get())
// Now allocate the main *Span and contained crdbSpan.
// Allocate these together to save on individual allocs.
//
// NB: at the time of writing, it's not possible to start a Span
// that *only* contains `ot` or `netTr`. This is just an artifact
// of the history of this code and may change in the future.
helper := struct {
span Span
crdbSpan crdbSpan
octx optimizedContext
}{}
helper.crdbSpan = crdbSpan{
traceID: traceID,
spanID: spanID,
goroutineID: goroutineID,
operation: opName,
startTime: startTime,
parentSpanID: opts.parentSpanID(),
logTags: opts.LogTags,
mu: crdbSpanMu{
duration: -1, // unfinished
},
}
helper.span.i = spanInner{
tracer: t,
crdb: &helper.crdbSpan,
ot: ot,
netTr: netTr,
}
s := &helper.span
{
// Link the newly created span to the parent, if necessary,
// and start recording, if requested.
// We inherit the recording type of the local parent, if any,
// over the remote parent, if any. If neither are specified, we're not recording.
var p *crdbSpan
if opts.Parent != nil {
p = opts.Parent.i.crdb
}
s.i.crdb.enableRecording(p, opts.recordingType())
}
// Set initial tags. These will propagate to the crdbSpan, ot, and netTr
// as appropriate.
//
// NB: this could be optimized.
for k, v := range opts.Tags {
s.SetTag(k, v)
}
// Copy baggage from parent. This similarly fans out over the various
// spans contained in Span.
//
// NB: this could be optimized.
if opts.Parent != nil {
if !opts.Parent.i.isNoop() {
opts.Parent.i.crdb.mu.Lock()
m := opts.Parent.i.crdb.mu.baggage
for k, v := range m {
s.SetBaggageItem(k, v)
}
opts.Parent.i.crdb.mu.Unlock()
}
} else {
if !opts.BypassRegistry {
// Local root span - put it into the registry of active local root
// spans. `Span.Finish` takes care of deleting it again.
t.activeSpans.Lock()
t.activeSpans.m[spanID] = s
t.activeSpans.Unlock()
}
if opts.RemoteParent != nil {
for k, v := range opts.RemoteParent.Baggage {
s.SetBaggageItem(k, v)
}
}
}
return maybeWrapCtx(ctx, &helper.octx, s)
}
// serializationFormat is the format used by the Tracer to {de,}serialize span
// metadata across process boundaries. This takes place within
// Tracer.{InjectMetaInto,ExtractMetaFrom}. Each format is inextricably linked
// to a corresponding Carrier, which is the thing that actually captures the
// serialized data and crosses process boundaries.
//
// The usage pattern is as follows:
//
// // One end of the RPC.
// carrier := MapCarrier{...}
// tracer.InjectMetaInto(sp.Meta(), carrier)
//
// // carrier crosses RPC boundary.
//
// // Other end of the RPC.
// spMeta, _ := Tracer.ExtractMetaFrom(carrier)
// ctx, sp := tracer.StartSpanCtx(..., spMeta)
//
type serializationFormat = opentracing.BuiltinFormat
const (
_ serializationFormat = iota
// metadataFormat is used to {de,}serialize data as HTTP header string
// pairs. It's used with gRPC (the carrier must be metadataCarrier), for
// when operations straddle RPC boundaries.
metadataFormat = opentracing.HTTPHeaders
// mapFormat is used to serialize data as a map of string pairs. The carrier
// must be MapCarrier.
mapFormat = opentracing.TextMap
)
// Carrier is what's used to capture the serialized data. Each carrier is
// inextricably linked to a corresponding format. See serializationFormat for
// more details.
type Carrier interface {
Set(key, val string)
ForEach(fn func(key, val string) error) error
}
// MapCarrier is an implementation of the Carrier interface for a map of string
// pairs.
type MapCarrier struct {
Map map[string]string
}
// Set implements the Carrier interface.
func (c MapCarrier) Set(key, val string) {
c.Map[key] = val
}
// ForEach implements the Carrier interface.
func (c MapCarrier) ForEach(fn func(key, val string) error) error {
for k, v := range c.Map {
if err := fn(k, v); err != nil {
return err
}
}
return nil
}
type textMapWriterFn func(key, val string)
var _ opentracing.TextMapWriter = textMapWriterFn(nil)
// Set is part of the opentracing.TextMapWriter interface.
func (fn textMapWriterFn) Set(key, val string) {
fn(key, val)
}
// InjectMetaInto is used to serialize the given span metadata into the given
// Carrier. This, alongside ExtractMetaFrom, can be used to carry span metadata
// across process boundaries. See serializationFormat for more details.
func (t *Tracer) InjectMetaInto(sm *SpanMeta, carrier Carrier) error {
if sm == nil {
// Fast path when tracing is disabled. ExtractMetaFrom will accept an
// empty map as a noop context.
return nil
}
var format serializationFormat
switch carrier.(type) {
case MapCarrier:
format = mapFormat
case metadataCarrier:
format = metadataFormat
default:
return errors.New("unsupported carrier")
}
carrier.Set(fieldNameTraceID, strconv.FormatUint(sm.traceID, 16))
carrier.Set(fieldNameSpanID, strconv.FormatUint(sm.spanID, 16))
for k, v := range sm.Baggage {
carrier.Set(prefixBaggage+k, v)
}
shadowTr := t.getShadowTracer()
if shadowTr != nil {
// Don't use a different shadow tracer than the one that created the parent span
// to put information on the wire. If something changes out from under us, forget
// about shadow tracing.
curTyp, _ := shadowTr.Type()
if typ := sm.shadowTracerType; typ == curTyp {
carrier.Set(fieldNameShadowType, sm.shadowTracerType)
// Encapsulate the shadow text map, prepending a prefix to the keys.
if err := shadowTr.Inject(sm.shadowCtx, format, textMapWriterFn(func(key, val string) {
carrier.Set(prefixShadow+key, val)
})); err != nil {
return err
}
}
}
return nil
}
var noopSpanMeta = (*SpanMeta)(nil)
// ExtractMetaFrom is used to deserialize a span metadata (if any) from the
// given Carrier. This, alongside InjectMetaFrom, can be used to carry span
// metadata across process boundaries. See serializationFormat for more details.
func (t *Tracer) ExtractMetaFrom(carrier Carrier) (*SpanMeta, error) {
var format serializationFormat
switch carrier.(type) {
case MapCarrier:
format = mapFormat
case metadataCarrier:
format = metadataFormat
default:
return noopSpanMeta, errors.New("unsupported carrier")
}
var shadowType string
var shadowCarrier opentracing.TextMapCarrier
var traceID uint64
var spanID uint64
var baggage map[string]string
// TODO(tbg): ForeachKey forces things on the heap. We can do better
// by using an explicit carrier.
err := carrier.ForEach(func(k, v string) error {
switch k = strings.ToLower(k); k {
case fieldNameTraceID:
var err error
traceID, err = strconv.ParseUint(v, 16, 64)
if err != nil {
return opentracing.ErrSpanContextCorrupted
}
case fieldNameSpanID:
var err error
spanID, err = strconv.ParseUint(v, 16, 64)
if err != nil {
return opentracing.ErrSpanContextCorrupted
}
case fieldNameShadowType:
shadowType = v
default:
if strings.HasPrefix(k, prefixBaggage) {
if baggage == nil {
baggage = make(map[string]string)
}
baggage[strings.TrimPrefix(k, prefixBaggage)] = v
} else if strings.HasPrefix(k, prefixShadow) {
if shadowCarrier == nil {
shadowCarrier = make(opentracing.TextMapCarrier)
}
// We build a shadow textmap with the original shadow keys.
shadowCarrier.Set(strings.TrimPrefix(k, prefixShadow), v)
}
}
return nil
})
if err != nil {
return noopSpanMeta, err
}
if traceID == 0 && spanID == 0 {
return noopSpanMeta, nil
}
var recordingType RecordingType
if baggage[verboseTracingBaggageKey] != "" {
recordingType = RecordingVerbose
}
var shadowCtx opentracing.SpanContext
if shadowType != "" {
shadowTr := t.getShadowTracer()
curShadowTyp, _ := shadowTr.Type()
if shadowType != curShadowTyp {
// If either the incoming context or tracer disagree on which
// shadow tracer (if any) is active, scrub shadow tracing from
// consideration.
shadowType = ""
} else {
// Shadow tracing is active on this node and the incoming information
// was created using the same type of tracer.
//
// Extract the shadow context using the un-encapsulated textmap.
shadowCtx, err = shadowTr.Extract(format, shadowCarrier)
if err != nil {
return noopSpanMeta, err
}
}
}
return &SpanMeta{
traceID: traceID,
spanID: spanID,
shadowTracerType: shadowType,
shadowCtx: shadowCtx,
recordingType: recordingType,
Baggage: baggage,
}, nil
}
// GetActiveSpanFromID retrieves any active span given its span ID.
func (t *Tracer) GetActiveSpanFromID(spanID uint64) (*Span, bool) {
t.activeSpans.Lock()
span, found := t.activeSpans.m[spanID]
t.activeSpans.Unlock()
return span, found
}
// VisitSpans invokes the visitor with all active Spans. The function will
// gracefully exit if the visitor returns iterutil.StopIteration().
func (t *Tracer) VisitSpans(visitor func(*Span) error) error {
t.activeSpans.Lock()
sl := make([]*Span, 0, len(t.activeSpans.m))
for _, sp := range t.activeSpans.m {
sl = append(sl, sp)
}
t.activeSpans.Unlock()
for _, sp := range sl {
if err := visitor(sp); err != nil {
if iterutil.Done(err) {
return nil
}
return err
}
}
return nil
}
// TestingIncludeAsyncSpansInRecordings is a test-only helper that configures
// the tracer to include recordings from forked/async child spans, when
// retrieving the recording for a parent span.
func (t *Tracer) TestingIncludeAsyncSpansInRecordings() {
t.includeAsyncSpansInRecordings = true
}
// ForkSpan forks the current span, if any[1]. Forked spans "follow from" the
// original, and are typically used to trace operations that may outlive the
// parent (think async tasks). See the package-level documentation for more
// details.
//
// The recordings from these spans will not be automatically propagated to the
// parent span[2]. Also see `ChildSpan`, for the other kind of derived span
// relation.
//
// A context wrapping the newly created span is returned, along with the span
// itself. If non-nil, the caller is responsible for eventually Finish()ing it.
//
// [1]: Looking towards the provided context to see if one exists.
// [2]: Unless configured differently by tests, see
// TestingIncludeAsyncSpansInRecordings.
func ForkSpan(ctx context.Context, opName string) (context.Context, *Span) {
sp := SpanFromContext(ctx)
if sp == nil {
return ctx, nil
}
collectionOpt := WithParentAndManualCollection(sp.Meta())
if sp.Tracer().includeAsyncSpansInRecordings {
// Using auto collection here ensures that recordings from async spans
// also show up at the parent.
collectionOpt = WithParentAndAutoCollection(sp)
}
return sp.Tracer().StartSpanCtx(ctx, opName, WithFollowsFrom(), collectionOpt)
}
// ChildSpan creates a child span of the current one, if any. Recordings from
// child spans are automatically propagated to the parent span, and the tags are
// inherited from the context's log tags automatically. Also see `ForkSpan`,
// for the other kind of derived span relation.
//
// A context wrapping the newly created span is returned, along with the span
// itself. If non-nil, the caller is responsible for eventually Finish()ing it.
func ChildSpan(ctx context.Context, opName string) (context.Context, *Span) {
sp := SpanFromContext(ctx)
if sp == nil {
return ctx, nil
}
return sp.Tracer().StartSpanCtx(ctx, opName, WithParentAndAutoCollection(sp))
}
// ChildSpanRemote is like ChildSpan but the new Span is created using
// WithParentAndManualCollection instead of WithParentAndAutoCollection. When
// this is used, it's the caller's duty to collect this span's recording and
// return it to the root span of the trace.
func ChildSpanRemote(ctx context.Context, opName string) (context.Context, *Span) {
sp := SpanFromContext(ctx)
if sp == nil {
return ctx, nil
}
return sp.Tracer().StartSpanCtx(ctx, opName, WithParentAndManualCollection(sp.Meta()))
}
// EnsureChildSpan looks at the supplied Context. If it contains a Span, returns
// a child span via the WithParentAndAutoCollection option; otherwise starts a
// new Span. In both cases, a context wrapping the Span is returned along with
// the newly created Span.
//
// The caller is responsible for closing the Span (via Span.Finish).
func EnsureChildSpan(
ctx context.Context, tr *Tracer, name string, os ...SpanOption,
) (context.Context, *Span) {
slp := optsPool.Get().(*[]SpanOption)
*slp = append(*slp, WithParentAndAutoCollection(SpanFromContext(ctx)))
*slp = append(*slp, os...)
ctx, sp := tr.StartSpanCtx(ctx, name, *slp...)
// Clear and zero-length the slice. Note that we have to clear
// explicitly or the options will continue to be referenced by
// the slice.
for i := range *slp {
(*slp)[i] = nil
}
*slp = (*slp)[0:0:cap(*slp)]
optsPool.Put(slp)
return ctx, sp
}
var optsPool = sync.Pool{
New: func() interface{} {
// It is unusual to pass more than 5 SpanOptions.
sl := make([]SpanOption, 0, 5)
return &sl
},
}
// StartVerboseTrace takes in a context and returns a derived one with a
// Span in it that is recording verbosely. The caller takes ownership of
// this Span from the returned context and is in charge of Finish()ing it.
//
// TODO(tbg): remove this method. It adds very little over EnsureChildSpan.
func StartVerboseTrace(ctx context.Context, tr *Tracer, opName string) (context.Context, *Span) {
ctx, sp := EnsureChildSpan(ctx, tr, opName, WithForceRealSpan())
sp.SetVerbose(true)
return ctx, sp
}
// ContextWithRecordingSpan returns a context with an embedded trace Span.
// The Span is derived from the provided Tracer. The Span returns its contents
// when `getRecording` is called, and must be stopped using `cancel`, when done
// with the context (`getRecording` needs to be called before `cancel`).
//
// Note that to convert the recorded spans into text, you can use
// Recording.String(). Tests can also use FindMsgInRecording().
func ContextWithRecordingSpan(
ctx context.Context, tr *Tracer, opName string,
) (_ context.Context, getRecording func() Recording, cancel func()) {
ctx, sp := tr.StartSpanCtx(ctx, opName, WithForceRealSpan())
sp.SetVerbose(true)
ctx, cancelCtx := context.WithCancel(ctx)
cancel = func() {
cancelCtx()
sp.SetVerbose(false)
sp.Finish()
tr.Close()
}
return ctx, sp.GetRecording, cancel
}