-
Notifications
You must be signed in to change notification settings - Fork 3.8k
/
replica_raft.go
2030 lines (1888 loc) · 80.1 KB
/
replica_raft.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
808
809
810
811
812
813
814
815
816
817
818
819
820
821
822
823
824
825
826
827
828
829
830
831
832
833
834
835
836
837
838
839
840
841
842
843
844
845
846
847
848
849
850
851
852
853
854
855
856
857
858
859
860
861
862
863
864
865
866
867
868
869
870
871
872
873
874
875
876
877
878
879
880
881
882
883
884
885
886
887
888
889
890
891
892
893
894
895
896
897
898
899
900
901
902
903
904
905
906
907
908
909
910
911
912
913
914
915
916
917
918
919
920
921
922
923
924
925
926
927
928
929
930
931
932
933
934
935
936
937
938
939
940
941
942
943
944
945
946
947
948
949
950
951
952
953
954
955
956
957
958
959
960
961
962
963
964
965
966
967
968
969
970
971
972
973
974
975
976
977
978
979
980
981
982
983
984
985
986
987
988
989
990
991
992
993
994
995
996
997
998
999
1000
// Copyright 2019 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 kvserver
import (
"context"
"fmt"
"math/rand"
"sort"
"strings"
"time"
"github.com/cockroachdb/cockroach/pkg/keys"
"github.com/cockroachdb/cockroach/pkg/kv/kvserver/apply"
"github.com/cockroachdb/cockroach/pkg/kv/kvserver/concurrency"
"github.com/cockroachdb/cockroach/pkg/kv/kvserver/kvserverbase"
"github.com/cockroachdb/cockroach/pkg/kv/kvserver/kvserverpb"
"github.com/cockroachdb/cockroach/pkg/kv/kvserver/liveness"
"github.com/cockroachdb/cockroach/pkg/kv/kvserver/stateloader"
"github.com/cockroachdb/cockroach/pkg/roachpb"
"github.com/cockroachdb/cockroach/pkg/storage"
"github.com/cockroachdb/cockroach/pkg/util"
"github.com/cockroachdb/cockroach/pkg/util/encoding"
"github.com/cockroachdb/cockroach/pkg/util/hlc"
"github.com/cockroachdb/cockroach/pkg/util/humanizeutil"
"github.com/cockroachdb/cockroach/pkg/util/log"
"github.com/cockroachdb/cockroach/pkg/util/protoutil"
"github.com/cockroachdb/cockroach/pkg/util/timeutil"
"github.com/cockroachdb/cockroach/pkg/util/tracing"
"github.com/cockroachdb/cockroach/pkg/util/uuid"
"github.com/cockroachdb/errors"
"go.etcd.io/etcd/raft/v3"
"go.etcd.io/etcd/raft/v3/raftpb"
"go.etcd.io/etcd/raft/v3/tracker"
)
func makeIDKey() kvserverbase.CmdIDKey {
idKeyBuf := make([]byte, 0, raftCommandIDLen)
idKeyBuf = encoding.EncodeUint64Ascending(idKeyBuf, uint64(rand.Int63()))
return kvserverbase.CmdIDKey(idKeyBuf)
}
// evalAndPropose prepares the necessary pending command struct and initializes
// a client command ID if one hasn't been. A verified lease is supplied as a
// parameter if the command requires a lease; nil otherwise. It then evaluates
// the command and proposes it to Raft on success.
//
// The method accepts a concurrency guard, which it assumes responsibility for
// if it succeeds in proposing a command into Raft. If the method does not
// return an error, the guard is guaranteed to be eventually freed and the
// caller should relinquish all ownership of it. If it does return an error, the
// caller retains full ownership over the guard.
//
// evalAndPropose takes ownership of the supplied token; the caller should
// tok.Move() it into this method. It will be used to untrack the request once
// it comes out of the proposal buffer.
//
// Nothing here or below can take out a raftMu lock, since executeWriteBatch()
// is already holding readOnlyCmdMu when calling this. Locking raftMu after it
// would violate the locking order specified for Store.mu.
//
// Return values:
// - a channel which receives a response or error upon application
// - a closure used to attempt to abandon the command. When called, it unbinds
// the command's context from its Raft proposal. The client is then free to
// terminate execution, although it is given no guarantee that the proposal
// won't still go on to commit and apply at some later time.
// - the proposal's ID.
// - any error obtained during the creation or proposal of the command, in
// which case the other returned values are zero.
func (r *Replica) evalAndPropose(
ctx context.Context,
ba *roachpb.BatchRequest,
g *concurrency.Guard,
st kvserverpb.LeaseStatus,
lul hlc.Timestamp,
tok TrackedRequestToken,
) (chan proposalResult, func(), kvserverbase.CmdIDKey, *roachpb.Error) {
defer tok.DoneIfNotMoved(ctx)
idKey := makeIDKey()
proposal, pErr := r.requestToProposal(ctx, idKey, ba, st, lul, g.LatchSpans(), g.LockSpans())
log.Event(proposal.ctx, "evaluated request")
// If the request hit a server-side concurrency retry error, immediately
// proagate the error. Don't assume ownership of the concurrency guard.
if isConcurrencyRetryError(pErr) {
pErr = maybeAttachLease(pErr, &st.Lease)
return nil, nil, "", pErr
} else if _, ok := pErr.GetDetail().(*roachpb.ReplicaCorruptionError); ok {
return nil, nil, "", pErr
}
// Attach the endCmds to the proposal and assume responsibility for
// releasing the concurrency guard if the proposal makes it to Raft.
proposal.ec = endCmds{repl: r, g: g, st: st}
// Pull out proposal channel to return. proposal.doneCh may be set to
// nil if it is signaled in this function.
proposalCh := proposal.doneCh
// There are two cases where request evaluation does not lead to a Raft
// proposal:
// 1. proposal.command == nil indicates that the evaluation was a no-op
// and that no Raft command needs to be proposed.
// 2. pErr != nil corresponds to a failed proposal - the command resulted
// in an error.
if proposal.command == nil {
intents := proposal.Local.DetachEncounteredIntents()
endTxns := proposal.Local.DetachEndTxns(pErr != nil /* alwaysOnly */)
r.handleReadWriteLocalEvalResult(ctx, *proposal.Local, false /* raftMuHeld */)
pr := proposalResult{
Reply: proposal.Local.Reply,
Err: pErr,
EncounteredIntents: intents,
EndTxns: endTxns,
}
proposal.finishApplication(ctx, pr)
return proposalCh, func() {}, "", nil
}
log.VEventf(proposal.ctx, 2,
"proposing command to write %d new keys, %d new values, %d new intents, "+
"write batch size=%d bytes",
proposal.command.ReplicatedEvalResult.Delta.KeyCount,
proposal.command.ReplicatedEvalResult.Delta.ValCount,
proposal.command.ReplicatedEvalResult.Delta.IntentCount,
proposal.command.WriteBatch.Size(),
)
// If the request requested that Raft consensus be performed asynchronously,
// return a proposal result immediately on the proposal's done channel.
// The channel's capacity will be large enough to accommodate this.
if ba.AsyncConsensus {
if ets := proposal.Local.DetachEndTxns(false /* alwaysOnly */); len(ets) != 0 {
// Disallow async consensus for commands with EndTxnIntents because
// any !Always EndTxnIntent can't be cleaned up until after the
// command succeeds.
return nil, nil, "", roachpb.NewErrorf("cannot perform consensus asynchronously for "+
"proposal with EndTxnIntents=%v; %v", ets, ba)
}
// 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")
// Signal the proposal's response channel immediately.
reply := *proposal.Local.Reply
reply.Responses = append([]roachpb.ResponseUnion(nil), reply.Responses...)
pr := proposalResult{
Reply: &reply,
EncounteredIntents: proposal.Local.DetachEncounteredIntents(),
}
proposal.signalProposalResult(pr)
// Continue with proposal...
}
// Attach information about the proposer to the command.
proposal.command.ProposerLeaseSequence = st.Lease.Sequence
// Perform a sanity check that the lease is owned by this replica.
if !st.Lease.OwnedBy(r.store.StoreID()) && !ba.IsLeaseRequest() {
log.Fatalf(ctx, "cannot propose %s on follower with remotely owned lease %s", ba, st.Lease)
}
// Once a command is written to the raft log, it must be loaded into memory
// and replayed on all replicas. If a command is too big, stop it here. If
// the command is not too big, acquire an appropriate amount of quota from
// the replica's proposal quota pool.
//
// TODO(tschottdorf): blocking a proposal here will leave it dangling in the
// closed timestamp tracker for an extended period of time, which will in turn
// prevent the node-wide closed timestamp from making progress. This is quite
// unfortunate; we should hoist the quota pool before the reference with the
// closed timestamp tracker is acquired. This is better anyway; right now many
// commands can evaluate but then be blocked on quota, which has worse memory
// behavior.
quotaSize := uint64(proposal.command.Size())
if maxSize := uint64(MaxCommandSize.Get(&r.store.cfg.Settings.SV)); quotaSize > maxSize {
return nil, nil, "", roachpb.NewError(errors.Errorf(
"command is too large: %d bytes (max: %d)", quotaSize, maxSize,
))
}
var err error
proposal.quotaAlloc, err = r.maybeAcquireProposalQuota(ctx, quotaSize)
if err != nil {
return nil, nil, "", roachpb.NewError(err)
}
// Make sure we clean up the proposal if we fail to insert it into the
// proposal buffer successfully. This ensures that we always release any
// quota that we acquire.
defer func() {
if pErr != nil {
proposal.releaseQuota()
}
}()
if filter := r.store.TestingKnobs().TestingProposalFilter; filter != nil {
filterArgs := kvserverbase.ProposalFilterArgs{
Ctx: ctx,
Cmd: *proposal.command,
QuotaAlloc: proposal.quotaAlloc,
CmdID: idKey,
Req: *ba,
}
if pErr = filter(filterArgs); pErr != nil {
return nil, nil, "", pErr
}
}
pErr = r.propose(ctx, proposal, tok.Move(ctx))
if pErr != nil {
return nil, nil, "", pErr
}
// Abandoning a proposal unbinds its context so that the proposal's client
// is free to terminate execution. However, it does nothing to try to
// prevent the command from succeeding. In particular, endCmds will still be
// invoked when the command is applied. There are a handful of cases where
// the command may not be applied (or even processed): the process crashes
// or the local replica is removed from the range.
abandon := func() {
// The proposal may or may not be in the Replica's proposals map.
// Instead of trying to look it up, simply modify the captured object
// directly. The raftMu must be locked to modify the context of a
// proposal because as soon as we propose a command to Raft, ownership
// passes to the "below Raft" machinery.
r.raftMu.Lock()
defer r.raftMu.Unlock()
r.mu.Lock()
defer r.mu.Unlock()
// TODO(radu): Should this context be created via tracer.ForkSpan?
// We'd need to make sure the span is finished eventually.
proposal.ctx = r.AnnotateCtx(context.TODO())
}
return proposalCh, abandon, idKey, nil
}
// propose encodes a command, starts tracking it, and proposes it to Raft.
//
// The method hands ownership of the command over to the Raft machinery. After
// the method returns, all access to the command must be performed while holding
// Replica.mu and Replica.raftMu.
//
// propose takes ownership of the supplied token; the caller should tok.Move()
// it into this method. It will be used to untrack the request once it comes out
// of the proposal buffer.
func (r *Replica) propose(
ctx context.Context, p *ProposalData, tok TrackedRequestToken,
) (pErr *roachpb.Error) {
defer tok.DoneIfNotMoved(ctx)
// If an error occurs reset the command's MaxLeaseIndex to its initial value.
// Failure to propose will propagate to the client. An invariant of this
// package is that proposals which are finished carry a raft command with a
// MaxLeaseIndex equal to the proposal command's max lease index.
defer func(prev uint64) {
if pErr != nil {
p.command.MaxLeaseIndex = prev
}
}(p.command.MaxLeaseIndex)
// Make sure the maximum lease index is unset. This field will be set in
// propBuf.Insert and its encoded bytes will be appended to the encoding
// buffer as a MaxLeaseFooter.
p.command.MaxLeaseIndex = 0
// Determine the encoding style for the Raft command.
prefix := true
version := raftVersionStandard
if crt := p.command.ReplicatedEvalResult.ChangeReplicas; crt != nil {
// EndTxnRequest with a ChangeReplicasTrigger is special because Raft
// needs to understand it; it cannot simply be an opaque command. To
// permit this, the command is proposed by the proposal buffer using
// ProposeConfChange. For that reason, we also don't need a Raft command
// prefix because the command ID is stored in a field in
// raft.ConfChange.
log.Infof(p.ctx, "proposing %s", crt)
prefix = false
// Ensure that we aren't trying to remove ourselves from the range without
// having previously given up our lease, since the range won't be able
// to make progress while the lease is owned by a removed replica (and
// leases can stay in such a state for a very long time when using epoch-
// based range leases). This shouldn't happen often, but has been seen
// before (#12591).
//
// Note that due to atomic replication changes, when a removal is initiated,
// the replica remains in the descriptor, but as VOTER_{OUTGOING,DEMOTING}.
// We want to block it from getting into that state in the first place,
// since there's no stopping the actual removal/demotion once it's there.
// The Removed() field has contains these replicas when this first
// transition is initiated, so its use here is copacetic.
replID := r.ReplicaID()
for _, rDesc := range crt.Removed() {
if rDesc.ReplicaID == replID {
err := errors.Mark(errors.Newf("received invalid ChangeReplicasTrigger %s to remove self (leaseholder)", crt),
errMarkInvalidReplicationChange)
log.Errorf(p.ctx, "%v", err)
return roachpb.NewError(err)
}
}
} else if p.command.ReplicatedEvalResult.AddSSTable != nil {
log.VEvent(p.ctx, 4, "sideloadable proposal detected")
version = raftVersionSideloaded
r.store.metrics.AddSSTableProposals.Inc(1)
if p.command.ReplicatedEvalResult.AddSSTable.Data == nil {
return roachpb.NewErrorf("cannot sideload empty SSTable")
}
} else if log.V(4) {
log.Infof(p.ctx, "proposing command %x: %s", p.idKey, p.Request.Summary())
}
// Create encoding buffer.
preLen := 0
if prefix {
preLen = raftCommandPrefixLen
}
cmdLen := p.command.Size()
// Allocate the data slice with enough capacity to eventually hold the two
// "footers" that are filled later.
needed := preLen + cmdLen + kvserverpb.MaxRaftCommandFooterSize()
data := make([]byte, preLen, needed)
// Encode prefix with command ID, if necessary.
if prefix {
encodeRaftCommandPrefix(data, version, p.idKey)
}
// Encode body of command.
data = data[:preLen+cmdLen]
if _, err := protoutil.MarshalTo(p.command, data[preLen:]); err != nil {
return roachpb.NewError(err)
}
p.encodedCommand = data
// Too verbose even for verbose logging, so manually enable if you want to
// debug proposal sizes.
if false {
log.Infof(p.ctx, `%s: proposal: %d
RaftCommand.ReplicatedEvalResult: %d
RaftCommand.ReplicatedEvalResult.Delta: %d
RaftCommand.WriteBatch: %d
`, p.Request.Summary(), cmdLen,
p.command.ReplicatedEvalResult.Size(),
p.command.ReplicatedEvalResult.Delta.Size(),
p.command.WriteBatch.Size(),
)
}
// Log an event if this is a large proposal. These are more likely to cause
// blips or worse, and it's good to be able to pick them from traces.
//
// TODO(tschottdorf): can we mark them so lightstep can group them?
const largeProposalEventThresholdBytes = 2 << 19 // 512kb
if cmdLen > largeProposalEventThresholdBytes {
log.Eventf(p.ctx, "proposal is large: %s", humanizeutil.IBytes(int64(cmdLen)))
}
// Insert into the proposal buffer, which passes the command to Raft to be
// proposed. The proposal buffer assigns the command a maximum lease index
// when it sequences it.
//
// NB: we must not hold r.mu while using the proposal buffer, see comment
// on the field.
err := r.mu.proposalBuf.Insert(ctx, p, tok.Move(ctx))
if err != nil {
return roachpb.NewError(err)
}
return nil
}
func (r *Replica) numPendingProposalsRLocked() int {
return len(r.mu.proposals) + r.mu.proposalBuf.AllocatedIdx()
}
// hasPendingProposalsRLocked is part of the quiescer interface.
// It returns true if this node has any outstanding proposals. A client might be
// waiting for the outcome of these proposals, so we definitely don't want to
// quiesce while such proposals are in-flight.
//
// Note that this method says nothing about other node's outstanding proposals:
// if this node is the current leaseholders, previous leaseholders might have
// proposals on which they're waiting. If this node is not the current
// leaseholder, then obviously whoever is the current leaseholder might have
// pending proposals. This method is called in two places: on the current
// leaseholder when deciding whether the leaseholder should attempt to quiesce
// the range, and then on every follower to confirm that the range can indeed be
// quiesced.
func (r *Replica) hasPendingProposalsRLocked() bool {
return r.numPendingProposalsRLocked() > 0
}
// hasPendingProposalQuotaRLocked is part of the quiescer interface. It returns
// true if there are any commands that haven't completed replicating that are
// tracked by this node's quota pool (i.e. commands that haven't been acked by
// all live replicas).
// We can't quiesce while there's outstanding quota because the respective quota
// would not be released while quiesced, and it might prevent the range from
// unquiescing (leading to deadlock). See #46699.
func (r *Replica) hasPendingProposalQuotaRLocked() bool {
if r.mu.proposalQuota == nil {
return true
}
return !r.mu.proposalQuota.Full()
}
var errRemoved = errors.New("replica removed")
// stepRaftGroup calls Step on the replica's RawNode with the provided request's
// message. Before doing so, it assures that the replica is unquiesced and ready
// to handle the request.
func (r *Replica) stepRaftGroup(req *RaftMessageRequest) error {
// We're processing an incoming raft message (from a batch that may
// include MsgVotes), so don't campaign if we wake up our raft
// group.
return r.withRaftGroup(false, func(raftGroup *raft.RawNode) (bool, error) {
// We're processing a message from another replica which means that the
// other replica is not quiesced, so we don't need to wake the leader.
// Note that we avoid campaigning when receiving raft messages, because
// we expect the originator to campaign instead.
r.unquiesceWithOptionsLocked(false /* campaignOnWake */)
r.mu.lastUpdateTimes.update(req.FromReplica.ReplicaID, timeutil.Now())
if req.Message.Type == raftpb.MsgSnap {
// Occasionally a snapshot message may arrive under an outdated term,
// which would lead to Raft discarding the snapshot. This should be
// really rare in practice, but it does happen in tests and in particular
// can happen to the synchronous snapshots on the learner path, which
// will then have to wait for the raft snapshot queue to send another
// snapshot. However, in some tests it is desirable to disable the
// raft snapshot queue. This workaround makes that possible.
//
// See TestReportUnreachableRemoveRace for the test that prompted
// this addition.
if term := raftGroup.BasicStatus().Term; term > req.Message.Term {
req.Message.Term = term
}
}
err := raftGroup.Step(req.Message)
if errors.Is(err, raft.ErrProposalDropped) {
// A proposal was forwarded to this replica but we couldn't propose it.
// Swallow the error since we don't have an effective way of signaling
// this to the sender.
// TODO(bdarnell): Handle ErrProposalDropped better.
// https://github.com/cockroachdb/cockroach/issues/21849
err = nil
}
return false /* unquiesceAndWakeLeader */, err
})
}
// raftSchedulerCtx annotates a given Raft scheduler context with information
// about the replica. The method may return a cached instance of this context.
func (r *Replica) raftSchedulerCtx(schedulerCtx context.Context) context.Context {
if v := r.schedulerCtx.Load(); v != nil {
return v.(context.Context)
}
schedulerCtx = r.AnnotateCtx(schedulerCtx)
r.schedulerCtx.Store(schedulerCtx)
return schedulerCtx
}
type handleSnapshotStats struct {
offered bool
applied bool
}
type handleRaftReadyStats struct {
applyCommittedEntriesStats
snap handleSnapshotStats
}
// noSnap can be passed to handleRaftReady when no snapshot should be processed.
var noSnap IncomingSnapshot
// handleRaftReady processes a raft.Ready containing entries and messages that
// are ready to read, be saved to stable storage, committed, or sent to other
// peers. It takes a non-empty IncomingSnapshot to indicate that it is
// about to process a snapshot.
//
// The returned string is nonzero whenever an error is returned to give a
// non-sensitive cue as to what happened.
func (r *Replica) handleRaftReady(
ctx context.Context, inSnap IncomingSnapshot,
) (handleRaftReadyStats, string, error) {
r.raftMu.Lock()
defer r.raftMu.Unlock()
return r.handleRaftReadyRaftMuLocked(ctx, inSnap)
}
// handleRaftReadyLocked is the same as handleRaftReady but requires that the
// replica's raftMu be held.
//
// The returned string is nonzero whenever an error is returned to give a
// non-sensitive cue as to what happened.
func (r *Replica) handleRaftReadyRaftMuLocked(
ctx context.Context, inSnap IncomingSnapshot,
) (_ handleRaftReadyStats, _ string, foo error) {
var stats handleRaftReadyStats
if inSnap.State != nil {
stats.snap.offered = true
}
var hasReady bool
var rd raft.Ready
r.mu.Lock()
lastIndex := r.mu.lastIndex // used for append below
lastTerm := r.mu.lastTerm
raftLogSize := r.mu.raftLogSize
leaderID := r.mu.leaderID
lastLeaderID := leaderID
err := r.withRaftGroupLocked(true, func(raftGroup *raft.RawNode) (bool, error) {
numFlushed, err := r.mu.proposalBuf.FlushLockedWithRaftGroup(ctx, raftGroup)
if err != nil {
return false, err
}
if hasReady = raftGroup.HasReady(); hasReady {
rd = raftGroup.Ready()
}
// We unquiesce if we have a Ready (= there's work to do). We also have
// to unquiesce if we just flushed some proposals but there isn't a
// Ready, which can happen if the proposals got dropped (raft does this
// if it doesn't know who the leader is). And, for extra defense in depth,
// we also unquiesce if there are outstanding proposals.
//
// NB: if we had the invariant that the group can only be in quiesced
// state if it knows the leader (state.Lead) AND we knew that raft would
// never give us an empty ready here (i.e. the only reason to drop a
// proposal is not knowing the leader) then numFlushed would not be
// necessary. The latter is likely true but we don't want to rely on
// it. The former is maybe true, but there's no easy way to enforce it.
unquiesceAndWakeLeader := hasReady || numFlushed > 0 || len(r.mu.proposals) > 0
return unquiesceAndWakeLeader, nil
})
r.mu.applyingEntries = len(rd.CommittedEntries) > 0
r.mu.Unlock()
if errors.Is(err, errRemoved) {
// If we've been removed then just return.
return stats, "", nil
} else if err != nil {
const expl = "while checking raft group for Ready"
return stats, expl, errors.Wrap(err, expl)
}
if !hasReady {
// We must update the proposal quota even if we don't have a ready.
// Consider the case when our quota is of size 1 and two out of three
// replicas have committed one log entry while the third is lagging
// behind. When the third replica finally does catch up and sends
// along a MsgAppResp, since the entry is already committed on the
// leader replica, no Ready is emitted. But given that the third
// replica has caught up, we can release
// some quota back to the pool.
r.updateProposalQuotaRaftMuLocked(ctx, lastLeaderID)
return stats, "", nil
}
logRaftReady(ctx, rd)
refreshReason := noReason
if rd.SoftState != nil && leaderID != roachpb.ReplicaID(rd.SoftState.Lead) {
// Refresh pending commands if the Raft leader has changed. This is usually
// the first indication we have of a new leader on a restarted node.
//
// TODO(peter): Re-proposing commands when SoftState.Lead changes can lead
// to wasteful multiple-reproposals when we later see an empty Raft command
// indicating a newly elected leader or a conf change. Replay protection
// prevents any corruption, so the waste is only a performance issue.
if log.V(3) {
log.Infof(ctx, "raft leader changed: %d -> %d", leaderID, rd.SoftState.Lead)
}
if !r.store.TestingKnobs().DisableRefreshReasonNewLeader {
refreshReason = reasonNewLeader
}
leaderID = roachpb.ReplicaID(rd.SoftState.Lead)
}
if inSnap.State != nil {
if !raft.IsEmptySnap(rd.Snapshot) {
snapUUID, err := uuid.FromBytes(rd.Snapshot.Data)
if err != nil {
const expl = "invalid snapshot id"
return stats, expl, errors.Wrap(err, expl)
}
if inSnap.SnapUUID == (uuid.UUID{}) {
log.Fatalf(ctx, "programming error: a snapshot application was attempted outside of the streaming snapshot codepath")
}
if snapUUID != inSnap.SnapUUID {
log.Fatalf(ctx, "incoming snapshot id doesn't match raft snapshot id: %s != %s", snapUUID, inSnap.SnapUUID)
}
// Applying this snapshot may require us to subsume one or more of our right
// neighbors. This occurs if this replica is informed about the merges via a
// Raft snapshot instead of a MsgApp containing the merge commits, e.g.,
// because it went offline before the merge commits applied and did not come
// back online until after the merge commits were truncated away.
subsumedRepls, releaseMergeLock := r.maybeAcquireSnapshotMergeLock(ctx, inSnap)
defer releaseMergeLock()
if err := r.applySnapshot(ctx, inSnap, rd.Snapshot, rd.HardState, subsumedRepls); err != nil {
const expl = "while applying snapshot"
return stats, expl, errors.Wrap(err, expl)
}
stats.snap.applied = true
// r.mu.lastIndex, r.mu.lastTerm and r.mu.raftLogSize were updated in
// applySnapshot, but we also want to make sure we reflect these changes in
// the local variables we're tracking here.
r.mu.RLock()
lastIndex = r.mu.lastIndex
lastTerm = r.mu.lastTerm
raftLogSize = r.mu.raftLogSize
r.mu.RUnlock()
// We refresh pending commands after applying a snapshot because this
// replica may have been temporarily partitioned from the Raft group and
// missed leadership changes that occurred. Suppose node A is the leader,
// and then node C gets partitioned away from the others. Leadership passes
// back and forth between A and B during the partition, but when the
// partition is healed node A is leader again.
if !r.store.TestingKnobs().DisableRefreshReasonSnapshotApplied &&
refreshReason == noReason {
refreshReason = reasonSnapshotApplied
}
}
} else if !raft.IsEmptySnap(rd.Snapshot) {
// If we didn't expect Raft to have a snapshot but it has one
// regardless, that is unexpected and indicates a programming
// error.
err := makeNonDeterministicFailure(
"have inSnap=nil, but raft has a snapshot %s",
raft.DescribeSnapshot(rd.Snapshot),
)
return stats, getNonDeterministicFailureExplanation(err), err
}
// If the ready struct includes entries that have been committed, these
// entries will be applied to the Replica's replicated state machine down
// below, after appending new entries to the raft log and sending messages
// to peers. However, the process of appending new entries to the raft log
// and then applying committed entries to the state machine can take some
// time - and these entries are already durably committed. If they have
// clients waiting on them, we'd like to acknowledge their success as soon
// as possible. To facilitate this, we take a quick pass over the committed
// entries and acknowledge as many as we can trivially prove will not be
// rejected beneath raft.
//
// Note that we only acknowledge up to the current last index in the Raft
// log. The CommittedEntries slice may contain entries that are also in the
// Entries slice (to be appended in this ready pass), and we don't want to
// acknowledge them until they are durably in our local Raft log. This is
// most common in single node replication groups, but it is possible when a
// follower in a multi-node replication group is catching up after falling
// behind. In the first case, the entries are not yet committed so
// acknowledging them would be a lie. In the second case, the entries are
// committed so we could acknowledge them at this point, but doing so seems
// risky. To avoid complications in either case, we pass lastIndex for the
// maxIndex argument to AckCommittedEntriesBeforeApplication.
sm := r.getStateMachine()
dec := r.getDecoder()
appTask := apply.MakeTask(sm, dec)
appTask.SetMaxBatchSize(r.store.TestingKnobs().MaxApplicationBatchSize)
defer appTask.Close()
if err := appTask.Decode(ctx, rd.CommittedEntries); err != nil {
return stats, getNonDeterministicFailureExplanation(err), err
}
if err := appTask.AckCommittedEntriesBeforeApplication(ctx, lastIndex); err != nil {
return stats, getNonDeterministicFailureExplanation(err), err
}
// Separate the MsgApp messages from all other Raft message types so that we
// can take advantage of the optimization discussed in the Raft thesis under
// the section: `10.2.1 Writing to the leader’s disk in parallel`. The
// optimization suggests that instead of a leader writing new log entries to
// disk before replicating them to its followers, the leader can instead
// write the entries to disk in parallel with replicating to its followers
// and them writing to their disks.
//
// Here, we invoke this optimization by:
// 1. sending all MsgApps.
// 2. syncing all entries and Raft state to disk.
// 3. sending all other messages.
//
// Since this is all handled in handleRaftReadyRaftMuLocked, we're assured
// that even though we may sync new entries to disk after sending them in
// MsgApps to followers, we'll always have them synced to disk before we
// process followers' MsgAppResps for the corresponding entries because
// Ready processing is sequential (and because a restart of the leader would
// prevent the MsgAppResp from being handled by it). This is important
// because it makes sure that the leader always has all of the entries in
// the log for its term, which is required in etcd/raft for technical
// reasons[1].
//
// MsgApps are also used to inform followers of committed entries through
// the Commit index that they contain. Due to the optimization described
// above, a Commit index may be sent out to a follower before it is
// persisted on the leader. This is safe because the Commit index can be
// treated as volatile state, as is supported by raft.MustSync[2].
// Additionally, the Commit index can never refer to entries from the
// current Ready (due to the MsgAppResp argument above) except in
// single-node groups, in which as a result we have to be careful to not
// persist a Commit index without the entries its commit index might refer
// to (see the HardState update below for details).
//
// [1]: the Raft thesis states that this can be made safe:
//
// > The leader may even commit an entry before it has been written to its
// > own disk, if a majority of followers have written it to their disks;
// > this is still safe.
//
// [2]: Raft thesis section: `3.8 Persisted state and server restarts`:
//
// > Other state variables are safe to lose on a restart, as they can all be
// > recreated. The most interesting example is the commit index, which can
// > safely be reinitialized to zero on a restart.
//
// Note that this will change when joint quorums are implemented, at which
// point we have to introduce coupling between the Commit index and
// persisted config changes, and also require some commit indexes to be
// durably synced.
// See:
// https://github.com/etcd-io/etcd/issues/7625#issuecomment-489232411
msgApps, otherMsgs := splitMsgApps(rd.Messages)
r.traceMessageSends(msgApps, "sending msgApp")
r.sendRaftMessages(ctx, msgApps)
// Use a more efficient write-only batch because we don't need to do any
// reads from the batch. Any reads are performed on the underlying DB.
batch := r.store.Engine().NewUnindexedBatch(false /* writeOnly */)
defer batch.Close()
prevLastIndex := lastIndex
if len(rd.Entries) > 0 {
// All of the entries are appended to distinct keys, returning a new
// last index.
thinEntries, sideLoadedEntriesSize, err := r.maybeSideloadEntriesRaftMuLocked(ctx, rd.Entries)
if err != nil {
const expl = "during sideloading"
return stats, expl, errors.Wrap(err, expl)
}
raftLogSize += sideLoadedEntriesSize
if lastIndex, lastTerm, raftLogSize, err = r.append(
ctx, batch, lastIndex, lastTerm, raftLogSize, thinEntries,
); err != nil {
const expl = "during append"
return stats, expl, errors.Wrap(err, expl)
}
}
if !raft.IsEmptyHardState(rd.HardState) {
if !r.IsInitialized() && rd.HardState.Commit != 0 {
log.Fatalf(ctx, "setting non-zero HardState.Commit on uninitialized replica %s. HS=%+v", r, rd.HardState)
}
// NB: Note that without additional safeguards, it's incorrect to write
// the HardState before appending rd.Entries. When catching up, a follower
// will receive Entries that are immediately Committed in the same
// Ready. If we persist the HardState but happen to lose the Entries,
// assertions can be tripped.
//
// We have both in the same batch, so there's no problem. If that ever
// changes, we must write and sync the Entries before the HardState.
if err := r.raftMu.stateLoader.SetHardState(ctx, batch, rd.HardState); err != nil {
const expl = "during setHardState"
return stats, expl, errors.Wrap(err, expl)
}
}
// Synchronously commit the batch with the Raft log entries and Raft hard
// state as we're promising not to lose this data.
//
// Note that the data is visible to other goroutines before it is synced to
// disk. This is fine. The important constraints are that these syncs happen
// before Raft messages are sent and before the call to RawNode.Advance. Our
// regular locking is sufficient for this and if other goroutines can see the
// data early, that's fine. In particular, snapshots are not a problem (I
// think they're the only thing that might access log entries or HardState
// from other goroutines). Snapshots do not include either the HardState or
// uncommitted log entries, and even if they did include log entries that
// were not persisted to disk, it wouldn't be a problem because raft does not
// infer the that entries are persisted on the node that sends a snapshot.
commitStart := timeutil.Now()
if err := batch.Commit(rd.MustSync && !disableSyncRaftLog.Get(&r.store.cfg.Settings.SV)); err != nil {
const expl = "while committing batch"
return stats, expl, errors.Wrap(err, expl)
}
if rd.MustSync {
elapsed := timeutil.Since(commitStart)
r.store.metrics.RaftLogCommitLatency.RecordValue(elapsed.Nanoseconds())
}
if len(rd.Entries) > 0 {
// We may have just overwritten parts of the log which contain
// sideloaded SSTables from a previous term (and perhaps discarded some
// entries that we didn't overwrite). Remove any such leftover on-disk
// payloads (we can do that now because we've committed the deletion
// just above).
firstPurge := rd.Entries[0].Index // first new entry written
purgeTerm := rd.Entries[0].Term - 1
lastPurge := prevLastIndex // old end of the log, include in deletion
purgedSize, err := maybePurgeSideloaded(ctx, r.raftMu.sideloaded, firstPurge, lastPurge, purgeTerm)
if err != nil {
const expl = "while purging sideloaded storage"
return stats, expl, err
}
raftLogSize -= purgedSize
if raftLogSize < 0 {
// Might have gone negative if node was recently restarted.
raftLogSize = 0
}
}
// Update protected state - last index, last term, raft log size, and raft
// leader ID.
r.mu.Lock()
r.mu.lastIndex = lastIndex
r.mu.lastTerm = lastTerm
r.mu.raftLogSize = raftLogSize
var becameLeader bool
if r.mu.leaderID != leaderID {
r.mu.leaderID = leaderID
// Clear the remote proposal set. Would have been nil already if not
// previously the leader.
becameLeader = r.mu.leaderID == r.mu.replicaID
}
r.mu.Unlock()
// When becoming the leader, proactively add the replica to the replicate
// queue. We might have been handed leadership by a remote node which wanted
// to remove itself from the range.
if becameLeader && r.store.replicateQueue != nil {
r.store.replicateQueue.MaybeAddAsync(ctx, r, r.store.Clock().NowAsClockTimestamp())
}
// Update raft log entry cache. We clear any older, uncommitted log entries
// and cache the latest ones.
r.store.raftEntryCache.Add(r.RangeID, rd.Entries, true /* truncate */)
r.sendRaftMessages(ctx, otherMsgs)
r.traceEntries(rd.CommittedEntries, "committed, before applying any entries")
applicationStart := timeutil.Now()
if len(rd.CommittedEntries) > 0 {
err := appTask.ApplyCommittedEntries(ctx)
stats.applyCommittedEntriesStats = sm.moveStats()
if errors.Is(err, apply.ErrRemoved) {
// We know that our replica has been removed. All future calls to
// r.withRaftGroup() will return errRemoved so no future Ready objects
// will be processed by this Replica.
return stats, "", err
} else if err != nil {
return stats, getNonDeterministicFailureExplanation(err), err
}
// etcd raft occasionally adds a nil entry (our own commands are never
// empty). This happens in two situations: When a new leader is elected, and
// when a config change is dropped due to the "one at a time" rule. In both
// cases we may need to resubmit our pending proposals (In the former case
// we resubmit everything because we proposed them to a former leader that
// is no longer able to commit them. In the latter case we only need to
// resubmit pending config changes, but it's hard to distinguish so we
// resubmit everything anyway). We delay resubmission until after we have
// processed the entire batch of entries.
if stats.numEmptyEntries > 0 {
// Overwrite unconditionally since this is the most aggressive
// reproposal mode.
if !r.store.TestingKnobs().DisableRefreshReasonNewLeaderOrConfigChange {
refreshReason = reasonNewLeaderOrConfigChange
}
}
}
applicationElapsed := timeutil.Since(applicationStart).Nanoseconds()
r.store.metrics.RaftApplyCommittedLatency.RecordValue(applicationElapsed)
r.store.metrics.RaftCommandsApplied.Inc(int64(len(rd.CommittedEntries)))
if r.store.TestingKnobs().EnableUnconditionalRefreshesInRaftReady {
refreshReason = reasonNewLeaderOrConfigChange
}
if refreshReason != noReason {
r.mu.Lock()
r.refreshProposalsLocked(ctx, 0 /* refreshAtDelta */, refreshReason)
r.mu.Unlock()
}
// NB: if we just processed a command which removed this replica from the
// raft group we will early return before this point. This, combined with
// the fact that we'll refuse to process messages intended for a higher
// replica ID ensures that our replica ID could not have changed.
const expl = "during advance"
r.mu.Lock()
err = r.withRaftGroupLocked(true, func(raftGroup *raft.RawNode) (bool, error) {
raftGroup.Advance(rd)
if stats.numConfChangeEntries > 0 {
// If the raft leader got removed, campaign the first remaining voter.
//
// NB: this must be called after Advance() above since campaigning is
// a no-op in the presence of unapplied conf changes.
if shouldCampaignAfterConfChange(ctx, r.store.StoreID(), r.descRLocked(), raftGroup) {
r.campaignLocked(ctx)
}
}
// If the Raft group still has more to process then we immediately
// re-enqueue it for another round of processing. This is possible if
// the group's committed entries were paginated due to size limitations
// and we didn't apply all of them in this pass.
if raftGroup.HasReady() {
r.store.enqueueRaftUpdateCheck(r.RangeID)
}
return true, nil
})
r.mu.applyingEntries = false
r.mu.Unlock()
if err != nil {
return stats, expl, errors.Wrap(err, expl)
}
// NB: All early returns other than the one due to not having a ready
// which also makes the below call are due to fatal errors.
// We must also update the proposal quota when have a ready; consider the
// case where there are two replicas and we have a quota of size 1. We
// acquire the quota when the write gets proposed on the leader and expect it
// to be released when the follower commits it locally. In order to do so we
// need to have the entry 'come out of raft' and in the case of a two node
// raft group, this only happens if hasReady == true. If we don't release
// quota back at the end of handleRaftReadyRaftMuLocked, the next write will
// get blocked.
r.updateProposalQuotaRaftMuLocked(ctx, lastLeaderID)
return stats, "", nil
}
// splitMsgApps splits the Raft message slice into two slices, one containing
// MsgApps and one containing all other message types. Each slice retains the
// relative ordering between messages in the original slice.
func splitMsgApps(msgs []raftpb.Message) (msgApps, otherMsgs []raftpb.Message) {
splitIdx := 0
for i, msg := range msgs {
if msg.Type == raftpb.MsgApp {
msgs[i], msgs[splitIdx] = msgs[splitIdx], msgs[i]
splitIdx++
}
}
return msgs[:splitIdx], msgs[splitIdx:]
}
// maybeFatalOnRaftReadyErr will fatal if err is neither nil nor
// apply.ErrRemoved.
func maybeFatalOnRaftReadyErr(ctx context.Context, expl string, err error) (removed bool) {
switch {
case err == nil:
return false
case errors.Is(err, apply.ErrRemoved):
return true
default:
log.FatalfDepth(ctx, 1, "%s: %+v", log.Safe(expl), err)
panic("unreachable")
}
}
// tick the Raft group, returning true if the raft group exists and is
// unquiesced; false otherwise.
func (r *Replica) tick(ctx context.Context, livenessMap liveness.IsLiveMap) (bool, error) {
r.unreachablesMu.Lock()
remotes := r.unreachablesMu.remotes
r.unreachablesMu.remotes = nil
r.unreachablesMu.Unlock()
r.raftMu.Lock()
defer r.raftMu.Unlock()
r.mu.Lock()
defer r.mu.Unlock()
// If the raft group is uninitialized, do not initialize on tick.
if r.mu.internalRaftGroup == nil {
return false, nil
}
for remoteReplica := range remotes {
r.mu.internalRaftGroup.ReportUnreachable(uint64(remoteReplica))
}
if r.mu.quiescent {
return false, nil
}
if r.maybeQuiesceLocked(ctx, livenessMap) {
return false, nil
}
r.maybeTransferRaftLeadershipToLeaseholderLocked(ctx)
// For followers, we update lastUpdateTimes when we step a message from them
// into the local Raft group. The leader won't hit that path, so we update
// it whenever it ticks. In effect, this makes sure it always sees itself as
// alive.
if r.mu.replicaID == r.mu.leaderID {
r.mu.lastUpdateTimes.update(r.mu.replicaID, timeutil.Now())
}