Skip to content

Commit

Permalink
kv: rename r.mu.last{Index,Term} to r.mu.last{Index,Term}NotDurable
Browse files Browse the repository at this point in the history
Simple rename to make the semantics of these fields more clear and
harder to misinterpret.
  • Loading branch information
nvanbenschoten committed Feb 2, 2023
1 parent 14ebedc commit ebc33fa
Show file tree
Hide file tree
Showing 7 changed files with 30 additions and 30 deletions.
2 changes: 1 addition & 1 deletion pkg/kv/kvserver/helpers_test.go
Original file line number Diff line number Diff line change
Expand Up @@ -356,7 +356,7 @@ func (r *Replica) GetRaftLogSize() (int64, bool) {
func (r *Replica) GetCachedLastTerm() uint64 {
r.mu.RLock()
defer r.mu.RUnlock()
return r.mu.lastTerm
return r.mu.lastTermNotDurable
}

func (r *Replica) IsRaftGroupInitialized() bool {
Expand Down
2 changes: 1 addition & 1 deletion pkg/kv/kvserver/raft_log_queue.go
Original file line number Diff line number Diff line change
Expand Up @@ -263,7 +263,7 @@ func newTruncateDecision(ctx context.Context, r *Replica) (truncateDecision, err

const anyRecipientStore roachpb.StoreID = 0
pendingSnapshotIndex := r.getSnapshotLogTruncationConstraintsRLocked(anyRecipientStore)
lastIndex := r.mu.lastIndex
lastIndex := r.mu.lastIndexNotDurable
// NB: raftLogSize above adjusts for pending truncations that have already
// been successfully replicated via raft, but logSizeTrusted does not see if
// those pending truncations would cause a transition from trusted =>
Expand Down
12 changes: 6 additions & 6 deletions pkg/kv/kvserver/replica.go
Original file line number Diff line number Diff line change
Expand Up @@ -397,12 +397,12 @@ type Replica struct {
mergeTxnID uuid.UUID
// The state of the Raft state machine.
state kvserverpb.ReplicaState
// Last index/term written to the raft log (not necessarily durable
// locally or committed by the group). Note that lastTerm may be 0 (and
// thus invalid) even when lastIndex is known, in which case the term
// will have to be retrieved from the Raft log entry. Use the
// Last index/term written to the raft log (not necessarily durable locally
// or committed by the group). Note that lastTermNotDurable may be 0 (and
// thus invalid) even when lastIndexNotDurable is known, in which case the
// term will have to be retrieved from the Raft log entry. Use the
// invalidLastTerm constant for this case.
lastIndex, lastTerm uint64
lastIndexNotDurable, lastTermNotDurable uint64
// A map of raft log index of pending snapshots to deadlines.
// Used to prohibit raft log truncations that would leave a gap between
// the snapshot and the new first index. The map entry has a zero
Expand Down Expand Up @@ -1294,7 +1294,7 @@ func (r *Replica) State(ctx context.Context) kvserverpb.RangeInfo {
r.mu.RLock()
defer r.mu.RUnlock()
ri.ReplicaState = *(protoutil.Clone(&r.mu.state)).(*kvserverpb.ReplicaState)
ri.LastIndex = r.mu.lastIndex
ri.LastIndex = r.mu.lastIndexNotDurable
ri.NumPending = uint64(r.numPendingProposalsRLocked())
ri.RaftLogSize = r.mu.raftLogSize
ri.RaftLogSizeTrusted = r.mu.raftLogSizeTrusted
Expand Down
4 changes: 2 additions & 2 deletions pkg/kv/kvserver/replica_init.go
Original file line number Diff line number Diff line change
Expand Up @@ -208,11 +208,11 @@ func (r *Replica) loadRaftMuLockedReplicaMuLocked(desc *roachpb.RangeDescriptor)
if r.mu.state, err = r.mu.stateLoader.Load(ctx, r.Engine(), desc); err != nil {
return err
}
r.mu.lastIndex, err = r.mu.stateLoader.LoadLastIndex(ctx, r.Engine())
r.mu.lastIndexNotDurable, err = r.mu.stateLoader.LoadLastIndex(ctx, r.Engine())
if err != nil {
return err
}
r.mu.lastTerm = invalidLastTerm
r.mu.lastTermNotDurable = invalidLastTerm

// Ensure that we're not trying to load a replica with a different ID than
// was used to construct this Replica.
Expand Down
18 changes: 9 additions & 9 deletions pkg/kv/kvserver/replica_raft.go
Original file line number Diff line number Diff line change
Expand Up @@ -720,8 +720,8 @@ func (r *Replica) handleRaftReadyRaftMuLocked(
var msgStorageAppend, msgStorageApply raftpb.Message
r.mu.Lock()
state := logstore.RaftState{ // used for append below
LastIndex: r.mu.lastIndex,
LastTerm: r.mu.lastTerm,
LastIndex: r.mu.lastIndexNotDurable,
LastTerm: r.mu.lastTermNotDurable,
ByteSize: r.mu.raftLogSize,
}
leaderID := r.mu.leaderID
Expand Down Expand Up @@ -890,13 +890,13 @@ func (r *Replica) handleRaftReadyRaftMuLocked(
stats.tSnapEnd = timeutil.Now()
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.lastIndexNotDurable, r.mu.lastTermNotDurable 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()
state = logstore.RaftState{
LastIndex: r.mu.lastIndex,
LastTerm: r.mu.lastTerm,
LastIndex: r.mu.lastIndexNotDurable,
LastTerm: r.mu.lastTermNotDurable,
ByteSize: r.mu.raftLogSize,
}
r.mu.RUnlock()
Expand Down Expand Up @@ -946,8 +946,8 @@ func (r *Replica) handleRaftReadyRaftMuLocked(
// leader ID.
r.mu.Lock()
// TODO(pavelkalinnikov): put logstore.RaftState to r.mu directly.
r.mu.lastIndex = state.LastIndex
r.mu.lastTerm = state.LastTerm
r.mu.lastIndexNotDurable = state.LastIndex
r.mu.lastTermNotDurable = state.LastTerm
r.mu.raftLogSize = state.ByteSize
var becameLeader bool
if r.mu.leaderID != leaderID {
Expand Down
20 changes: 10 additions & 10 deletions pkg/kv/kvserver/replica_raftstorage.go
Original file line number Diff line number Diff line change
Expand Up @@ -111,19 +111,19 @@ func (r *Replica) raftEntriesLocked(lo, hi, maxBytes uint64) ([]raftpb.Entry, er
return (*replicaRaftStorage)(r).Entries(lo, hi, maxBytes)
}

// invalidLastTerm is an out-of-band value for r.mu.lastTerm that
// invalidates lastTerm caching and forces retrieval of Term(lastTerm)
// from the raftEntryCache/RocksDB.
// invalidLastTerm is an out-of-band value for r.mu.lastTermNotDurable that
// invalidates lastTermNotDurable caching and forces retrieval of
// Term(lastIndexNotDurable) from the raftEntryCache/Pebble.
const invalidLastTerm = 0

// Term implements the raft.Storage interface.
// Term requires that r.mu is held for writing because it requires exclusive
// access to r.mu.stateLoader.
func (r *replicaRaftStorage) Term(i uint64) (uint64, error) {
// TODO(nvanbenschoten): should we set r.mu.lastTerm when
// r.mu.lastIndex == i && r.mu.lastTerm == invalidLastTerm?
if r.mu.lastIndex == i && r.mu.lastTerm != invalidLastTerm {
return r.mu.lastTerm, nil
// TODO(nvanbenschoten): should we set r.mu.lastTermNotDurable when
// r.mu.lastIndexNotDurable == i && r.mu.lastTermNotDurable == invalidLastTerm?
if r.mu.lastIndexNotDurable == i && r.mu.lastTermNotDurable != invalidLastTerm {
return r.mu.lastTermNotDurable, nil
}
ctx := r.AnnotateCtx(context.TODO())
return logstore.LoadTerm(ctx, r.mu.stateLoader.StateLoader, r.store.Engine(), r.RangeID,
Expand All @@ -145,7 +145,7 @@ func (r *Replica) GetTerm(i uint64) (uint64, error) {

// raftLastIndexRLocked requires that r.mu is held for reading.
func (r *Replica) raftLastIndexRLocked() uint64 {
return r.mu.lastIndex
return r.mu.lastIndexNotDurable
}

// LastIndex implements the raft.Storage interface.
Expand Down Expand Up @@ -731,13 +731,13 @@ func (r *Replica) applySnapshot(
// performance implications are not likely to be drastic. If our
// feelings about this ever change, we can add a LastIndex field to
// raftpb.SnapshotMetadata.
r.mu.lastIndex = state.RaftAppliedIndex
r.mu.lastIndexNotDurable = state.RaftAppliedIndex

// TODO(sumeer): We should be able to set this to
// nonemptySnap.Metadata.Term. See
// https://github.com/cockroachdb/cockroach/pull/75675#pullrequestreview-867926687
// for a discussion regarding this.
r.mu.lastTerm = invalidLastTerm
r.mu.lastTermNotDurable = invalidLastTerm
r.mu.raftLogSize = 0
// Update the store stats for the data in the snapshot.
r.store.metrics.subtractMVCCStats(ctx, r.tenantMetricsRef, *r.mu.state.Stats)
Expand Down
2 changes: 1 addition & 1 deletion pkg/kv/kvserver/replica_sideload_test.go
Original file line number Diff line number Diff line change
Expand Up @@ -179,7 +179,7 @@ func TestRaftSSTableSideloading(t *testing.T) {

rsl := logstore.NewStateLoader(tc.repl.RangeID)
lo := tc.repl.mu.state.TruncatedState.Index + 1
hi := tc.repl.mu.lastIndex + 1
hi := tc.repl.mu.lastIndexNotDurable + 1

tc.store.raftEntryCache.Clear(tc.repl.RangeID, hi)
ents, err := logstore.LoadEntries(
Expand Down

0 comments on commit ebc33fa

Please sign in to comment.