From ebc33fa0823ccf9a9c6095e3d8d487261af20ef5 Mon Sep 17 00:00:00 2001 From: Nathan VanBenschoten Date: Thu, 2 Feb 2023 17:44:47 -0500 Subject: [PATCH] kv: rename r.mu.last{Index,Term} to r.mu.last{Index,Term}NotDurable Simple rename to make the semantics of these fields more clear and harder to misinterpret. --- pkg/kv/kvserver/helpers_test.go | 2 +- pkg/kv/kvserver/raft_log_queue.go | 2 +- pkg/kv/kvserver/replica.go | 12 ++++++------ pkg/kv/kvserver/replica_init.go | 4 ++-- pkg/kv/kvserver/replica_raft.go | 18 +++++++++--------- pkg/kv/kvserver/replica_raftstorage.go | 20 ++++++++++---------- pkg/kv/kvserver/replica_sideload_test.go | 2 +- 7 files changed, 30 insertions(+), 30 deletions(-) diff --git a/pkg/kv/kvserver/helpers_test.go b/pkg/kv/kvserver/helpers_test.go index 00564d86034a..5b4b112a5a8b 100644 --- a/pkg/kv/kvserver/helpers_test.go +++ b/pkg/kv/kvserver/helpers_test.go @@ -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 { diff --git a/pkg/kv/kvserver/raft_log_queue.go b/pkg/kv/kvserver/raft_log_queue.go index eb305fcae327..70ec7779bf8b 100644 --- a/pkg/kv/kvserver/raft_log_queue.go +++ b/pkg/kv/kvserver/raft_log_queue.go @@ -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 => diff --git a/pkg/kv/kvserver/replica.go b/pkg/kv/kvserver/replica.go index eb72b9c650e2..270b137fd799 100644 --- a/pkg/kv/kvserver/replica.go +++ b/pkg/kv/kvserver/replica.go @@ -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 @@ -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 diff --git a/pkg/kv/kvserver/replica_init.go b/pkg/kv/kvserver/replica_init.go index b5964c917b3b..c2494e35bba7 100644 --- a/pkg/kv/kvserver/replica_init.go +++ b/pkg/kv/kvserver/replica_init.go @@ -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. diff --git a/pkg/kv/kvserver/replica_raft.go b/pkg/kv/kvserver/replica_raft.go index 994471cb9564..9db90f1af469 100644 --- a/pkg/kv/kvserver/replica_raft.go +++ b/pkg/kv/kvserver/replica_raft.go @@ -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 @@ -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() @@ -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 { diff --git a/pkg/kv/kvserver/replica_raftstorage.go b/pkg/kv/kvserver/replica_raftstorage.go index 22cef1a6f55d..40a78cf4bb4a 100644 --- a/pkg/kv/kvserver/replica_raftstorage.go +++ b/pkg/kv/kvserver/replica_raftstorage.go @@ -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, @@ -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. @@ -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) diff --git a/pkg/kv/kvserver/replica_sideload_test.go b/pkg/kv/kvserver/replica_sideload_test.go index 74501c638a78..19633d18dd8d 100644 --- a/pkg/kv/kvserver/replica_sideload_test.go +++ b/pkg/kv/kvserver/replica_sideload_test.go @@ -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(