Skip to content

Commit e15b1ec

Browse files
authored
Merge pull request #159213 from cockroachdb/blathers/backport-release-26.1-158349
release-26.1: kvnemesis: report splits to raft asserter
2 parents 7232097 + 9587fd6 commit e15b1ec

File tree

4 files changed

+61
-6
lines changed

4 files changed

+61
-6
lines changed

pkg/kv/kvnemesis/kvnemesis_test.go

Lines changed: 7 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -170,6 +170,13 @@ func (cfg kvnemesisTestCfg) testClusterArgs(
170170
}
171171
return 0, nil
172172
}
173+
storeKnobs.AfterSplitApplication = func(desc roachpb.ReplicaDescriptor, state kvserverpb.ReplicaState) {
174+
asserter.ApplySplitRHS(
175+
state.Desc.RangeID, desc.ReplicaID,
176+
state.RaftAppliedIndex, state.RaftAppliedIndexTerm,
177+
state.LeaseAppliedIndex, state.RaftClosedTimestamp,
178+
)
179+
}
173180
storeKnobs.AfterSnapshotApplication = func(
174181
desc roachpb.ReplicaDescriptor, state kvserverpb.ReplicaState, snap kvserver.IncomingSnapshot,
175182
) {

pkg/kv/kvserver/apply/asserter.go

Lines changed: 42 additions & 6 deletions
Original file line numberDiff line numberDiff line change
@@ -342,6 +342,33 @@ func (r *rangeAsserter) apply(
342342
r.seedAppliedAs[seedID] = cmdID
343343
}
344344

345+
// ApplySplitRHS tracks and asserts command applications which split out a new
346+
// range. Accepts the state of the RHS.
347+
func (a *Asserter) ApplySplitRHS(
348+
rangeID roachpb.RangeID,
349+
replicaID roachpb.ReplicaID,
350+
raftAppliedIndex kvpb.RaftIndex,
351+
raftAppliedIndexTerm kvpb.RaftTerm,
352+
leaseAppliedIndex kvpb.LeaseAppliedIndex,
353+
closedTS hlc.Timestamp,
354+
) {
355+
// Use a special command ID that signifies a pseudo-proposal that splits this
356+
// range out. This ID does not match any "real" IDs (they are of a different
357+
// size, and sufficiently random even if were of the same size).
358+
const cmdID = kvserverbase.CmdIDKey("__split__")
359+
r := a.forRange(rangeID)
360+
// Circumvent the check in r.apply that the command ID must be proposed.
361+
func() {
362+
r.Lock()
363+
defer r.Unlock()
364+
r.proposedCmds[cmdID] = 0
365+
}()
366+
// Pretend that we are applying this pseudo-command.
367+
r.apply(replicaID, cmdID, raftpb.Entry{
368+
Index: uint64(raftAppliedIndex), Term: uint64(raftAppliedIndexTerm),
369+
}, leaseAppliedIndex, closedTS)
370+
}
371+
345372
// ApplySnapshot tracks and asserts snapshot application.
346373
func (a *Asserter) ApplySnapshot(
347374
rangeID roachpb.RangeID,
@@ -385,13 +412,22 @@ func (r *rangeAsserter) applySnapshot(
385412
}
386413
r.replicaAppliedIndex[replicaID] = index
387414

388-
// We can't have a snapshot without any applied log entries, except when this
389-
// is an initial snapshot. It's possible that the initial snapshot follows an
390-
// empty entry appended by the raft leader at the start of this term. Since we
391-
// don't register this entry as applied, r.log can be empty here.
415+
// Typically, we can't have a snapshot without any applied entries. For most
416+
// ranges, there is at least a pseudo-command (see ApplySplitRHS) that created
417+
// this range as part of a split.
418+
//
419+
// For ranges that were not a result of a split (those created during the
420+
// cluster bootstrap), the only way to see an empty r.log here is that this
421+
// snapshot follows only empty entries appended by the raft leader at the
422+
// start of its term or other no-op entries (see comment in r.apply explaining
423+
// that we do not register such commands).
424+
//
425+
// The latter situation appears impossible, because the bootstrapped ranges
426+
// can't send a snapshot to anyone without applying a config change adding
427+
// another replica - so r.log can't be empty.
392428
//
393-
// See the comment in r.apply() method, around the empty cmdID check, and the
394-
// comment for r.log saying that there can be gaps in the observed applies.
429+
// TODO(pav-kv): consider dropping this condition, or reporting the initial
430+
// state during the cluster bootstrap for completeness.
395431
if len(r.log) == 0 {
396432
return
397433
}

pkg/kv/kvserver/store_split.go

Lines changed: 9 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -313,6 +313,15 @@ func prepareRightReplicaForSplit(
313313
// Raft group, there might not be any Raft traffic for quite a while.
314314
rightRepl.maybeUnquiesceLocked(true /* wakeLeader */, true /* mayCampaign */)
315315

316+
if fn := r.store.TestingKnobs().AfterSplitApplication; fn != nil {
317+
// TODO(pav-kv): we have already checked up the stack that rightDesc exists,
318+
// but maybe it would be better to carry a "bus" struct with these kinds of
319+
// data post checks so that we (a) don't need to repeat the computation /
320+
// validation, (b) can be sure that it's consistent.
321+
rightDesc, _ := split.RightDesc.GetReplicaDescriptor(r.StoreID())
322+
fn(rightDesc, rightRepl.shMu.state)
323+
}
324+
316325
return rightRepl
317326
}
318327

pkg/kv/kvserver/testing_knobs.go

Lines changed: 3 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -379,6 +379,9 @@ type StoreTestingKnobs struct {
379379
// BeforeSnapshotSSTIngestion is run just before the SSTs are ingested when
380380
// applying a snapshot.
381381
BeforeSnapshotSSTIngestion func(IncomingSnapshot, []string) error
382+
// AfterSplitApplication is called on the newly created replica's state after
383+
// a split is applied. Called iff the RHS replica is not already destroyed.
384+
AfterSplitApplication func(roachpb.ReplicaDescriptor, kvserverpb.ReplicaState)
382385
// AfterSnapshotApplication is run after a snapshot is applied, before
383386
// releasing the replica mutex.
384387
AfterSnapshotApplication func(roachpb.ReplicaDescriptor, kvserverpb.ReplicaState, IncomingSnapshot)

0 commit comments

Comments
 (0)