66package kvstorage
77
88import (
9+ "github.com/cockroachdb/cockroach/pkg/keys"
10+ "github.com/cockroachdb/cockroach/pkg/kv/kvserver/spanset"
11+ "github.com/cockroachdb/cockroach/pkg/roachpb"
912 "github.com/cockroachdb/cockroach/pkg/storage"
13+ "github.com/cockroachdb/cockroach/pkg/util"
1014 "github.com/cockroachdb/cockroach/pkg/util/buildutil"
15+ "github.com/cockroachdb/errors"
1116)
1217
1318// The following are type aliases that help annotating various storage
@@ -121,8 +126,20 @@ type Engines struct {
121126// MakeEngines creates an Engines handle in which both state machine and log
122127// engine reside in the same physical engine.
123128func MakeEngines (eng storage.Engine ) Engines {
124- // TODO(#158281): in test builds, wrap the engines in a way that allows
125- // verifying that all accesses touch the correct subset of keys.
129+ if util .RaceEnabled {
130+ // Wrap the engines with span set engines to catch incorrect engine
131+ // accesses.
132+ wrappedStateEngine := spanset .NewSpanSetEngine (eng )
133+ wrappedStateEngine .(* spanset.SpanSetEngine ).AddForbiddenMatcher (validateIsStateEngineSpan )
134+ wrappedLogEngine := spanset .NewSpanSetEngine (eng )
135+ wrappedLogEngine .(* spanset.SpanSetEngine ).AddForbiddenMatcher (validateIsRaftEngineSpan )
136+ return Engines {
137+ stateEngine : wrappedStateEngine ,
138+ logEngine : wrappedLogEngine ,
139+ todoEngine : eng ,
140+ }
141+ }
142+
126143 return Engines {
127144 stateEngine : eng ,
128145 todoEngine : eng ,
@@ -137,6 +154,20 @@ func MakeSeparatedEnginesForTesting(state, log storage.Engine) Engines {
137154 if ! buildutil .CrdbTestBuild {
138155 panic ("separated engines are not supported" )
139156 }
157+ if util .RaceEnabled {
158+ // Wrap the engines with span set engines to catch incorrect engine
159+ // accesses.
160+ wrappedStateEngine := spanset .NewSpanSetEngine (state )
161+ wrappedStateEngine .(* spanset.SpanSetEngine ).AddForbiddenMatcher (validateIsStateEngineSpan )
162+ wrappedLogEngine := spanset .NewSpanSetEngine (log )
163+ wrappedLogEngine .(* spanset.SpanSetEngine ).AddForbiddenMatcher (validateIsRaftEngineSpan )
164+ return Engines {
165+ stateEngine : wrappedStateEngine ,
166+ todoEngine : nil ,
167+ logEngine : wrappedLogEngine ,
168+ separated : true ,
169+ }
170+ }
140171 return Engines {
141172 stateEngine : state ,
142173 todoEngine : nil ,
@@ -178,3 +209,128 @@ func (e *Engines) TODOEngine() storage.Engine {
178209func (e * Engines ) Separated () bool {
179210 return e .separated
180211}
212+
213+ // validateIsStateEngineSpan asserts that the provided span only overlaps with
214+ // keys in the State engine and returns an error if not.
215+ // Note that we could receive the span with a nil startKey, which has a special
216+ // meaning that the span represents: [endKey.Prev(), endKey).
217+ func validateIsStateEngineSpan (span spanset.TrickySpan ) error {
218+ // If the provided span overlaps with local store span, it cannot be a
219+ // StateEngine span because Store-local keys belong to the LogEngine.
220+ if spanset .Overlaps (roachpb.Span {
221+ Key : keys .LocalStorePrefix ,
222+ EndKey : keys .LocalStoreMax ,
223+ }, span ) {
224+ return errors .Errorf ("overlaps with store local keys" )
225+ }
226+
227+ // If the provided span is completely outside the rangeID local spans for any
228+ // rangeID, then there is no overlap with any rangeID local keys.
229+ fullRangeIDLocalSpans := roachpb.Span {
230+ Key : keys .LocalRangeIDPrefix .AsRawKey (),
231+ EndKey : keys .LocalRangeIDPrefix .AsRawKey ().PrefixEnd (),
232+ }
233+ if ! spanset .Overlaps (fullRangeIDLocalSpans , span ) {
234+ return nil
235+ }
236+
237+ // At this point, we know that we overlap with fullRangeIDLocalSpans. If we
238+ // are not completely within fullRangeIDLocalSpans, return an error as we
239+ // make an assumption that spans should respect the local RangeID tree
240+ // structure, and that spans that partially overlaps with
241+ // fullRangeIDLocalSpans don't make logical sense.
242+ if ! spanset .Contains (fullRangeIDLocalSpans , span ) {
243+ return errors .Errorf ("overlapping an unreplicated rangeID key" )
244+ }
245+
246+ // If the span in inside fullRangeIDLocalSpans, we expect that both start and
247+ // end keys should be in the same rangeID.
248+ rangeIDKey := span .Key
249+ if rangeIDKey == nil {
250+ rangeIDKey = span .EndKey
251+ }
252+ rangeID , err := keys .DecodeRangeIDPrefix (rangeIDKey )
253+ if err != nil {
254+ return errors .NewAssertionErrorWithWrappedErrf (err ,
255+ "could not decode range ID for span: %s" , span )
256+ }
257+
258+ // If the span is inside RangeIDLocalSpans but outside RangeIDUnreplicated,
259+ // it cannot overlap local raft keys.
260+ rangeIDPrefixBuf := keys .MakeRangeIDPrefixBuf (rangeID )
261+ if ! spanset .Overlaps (roachpb.Span {
262+ Key : rangeIDPrefixBuf .UnreplicatedPrefix (),
263+ EndKey : rangeIDPrefixBuf .UnreplicatedPrefix ().PrefixEnd (),
264+ }, span ) {
265+ return nil
266+ }
267+
268+ // RangeTombstoneKey and RaftReplicaIDKey belong to the StateEngine, and can
269+ // be accessed as point keys.
270+ if roachpb .Span (span ).Equal (roachpb.Span {
271+ Key : rangeIDPrefixBuf .RangeTombstoneKey (),
272+ }) {
273+ return nil
274+ }
275+
276+ if roachpb .Span (span ).Equal (roachpb.Span {
277+ Key : rangeIDPrefixBuf .RaftReplicaIDKey (),
278+ }) {
279+ return nil
280+ }
281+
282+ return errors .Errorf ("overlapping an unreplicated rangeID span" )
283+ }
284+
285+ // validateIsRaftEngineSpan asserts that the provided span only overlaps with
286+ // keys in the Raft engine and returns an error if not.
287+ // Note that we could receive the span with a nil startKey, which has a special
288+ // meaning that the span represents: [endKey.Prev(), endKey).
289+ func validateIsRaftEngineSpan (span spanset.TrickySpan ) error {
290+ // The LogEngine owns only Store-local and RangeID-local raft keys. A span
291+ // inside Store-local is correct. If it's only partially inside, an error is
292+ // returned below, as part of checking RangeID-local spans.
293+ if spanset .Contains (roachpb.Span {
294+ Key : keys .LocalStorePrefix ,
295+ EndKey : keys .LocalStoreMax ,
296+ }, span ) {
297+ return nil
298+ }
299+
300+ // At this point, the remaining possible LogEngine keys are inside
301+ // LocalRangeID spans. If the span is not completely inside it, it must
302+ // overlap with some StateEngine keys.
303+ if ! spanset .Contains (roachpb.Span {
304+ Key : keys .LocalRangeIDPrefix .AsRawKey (),
305+ EndKey : keys .LocalRangeIDPrefix .AsRawKey ().PrefixEnd (),
306+ }, span ) {
307+ return errors .Errorf ("overlaps with state engine keys" )
308+ }
309+
310+ // If the span in inside LocalRangeID, we assume that both start and
311+ // end keys should be in the same rangeID.
312+ rangeIDKey := span .Key
313+ if rangeIDKey == nil {
314+ rangeIDKey = span .EndKey
315+ }
316+ rangeID , err := keys .DecodeRangeIDPrefix (rangeIDKey )
317+ if err != nil {
318+ return errors .NewAssertionErrorWithWrappedErrf (err ,
319+ "could not decode range ID for span: %s" , span )
320+ }
321+ rangeIDPrefixBuf := keys .MakeRangeIDPrefixBuf (rangeID )
322+ if ! spanset .Contains (roachpb.Span {
323+ Key : rangeIDPrefixBuf .UnreplicatedPrefix (),
324+ EndKey : rangeIDPrefixBuf .UnreplicatedPrefix ().PrefixEnd (),
325+ }, span ) {
326+ return errors .Errorf ("overlaps with state engine keys" )
327+ }
328+ if spanset .Overlaps (roachpb.Span {Key : rangeIDPrefixBuf .RangeTombstoneKey ()}, span ) {
329+ return errors .Errorf ("overlaps with state engine keys" )
330+ }
331+ if spanset .Overlaps (roachpb.Span {Key : rangeIDPrefixBuf .RaftReplicaIDKey ()}, span ) {
332+ return errors .Errorf ("overlaps with state engine keys" )
333+ }
334+
335+ return nil
336+ }
0 commit comments