Skip to content

Commit

Permalink
storage: implement iterator to interleave separated intents
Browse files Browse the repository at this point in the history
intentInterleavingIter allows for both physically separate
and physically interleaved intents, and makes both look
like interleaved intents. It uses two underlying iterators,
a MVCCIterator and an EngineIterator, that are kept
synchronized.

Informs #41720

Release note: None
  • Loading branch information
sumeerbhola committed Nov 13, 2020
1 parent ec6cdbc commit 95b836d
Show file tree
Hide file tree
Showing 11 changed files with 1,496 additions and 17 deletions.
28 changes: 20 additions & 8 deletions pkg/keys/keys.go
Original file line number Diff line number Diff line change
Expand Up @@ -393,22 +393,28 @@ func QueueLastProcessedKey(key roachpb.RKey, queue string) roachpb.Key {
}

// LockTableSingleKey creates a key under which all single-key locks for the
// given key can be found. Note that there can be multiple locks for the given
// key, but those are distinguished using the "version" which is not in scope
// of the keys package.
// given key can be found. buf is used as scratch-space to avoid allocations
// -- its contents will be overwritten and not appended to.
// Note that there can be multiple locks for the given key, but those are
// distinguished using the "version" which is not in scope of the keys
// package.
// For a scan [start, end) the corresponding lock table scan is
// [LTSK(start), LTSK(end)).
func LockTableSingleKey(key roachpb.Key) roachpb.Key {
func LockTableSingleKey(key roachpb.Key, buf []byte) (roachpb.Key, []byte) {
// The +3 accounts for the bytesMarker and terminator.
keyLen := len(LocalRangeLockTablePrefix) + len(LockTableSingleKeyInfix) + len(key) + 3
if cap(buf) < keyLen {
buf = make([]byte, 0, keyLen)
} else {
buf = buf[:0]
}
// Don't unwrap any local prefix on key using Addr(key). This allow for
// doubly-local lock table keys. For example, local range descriptor keys can
// be locked during split and merge transactions.
// The +3 account for the bytesMarker and terminator.
buf := make(roachpb.Key, 0,
len(LocalRangeLockTablePrefix)+len(LockTableSingleKeyInfix)+len(key)+3)
buf = append(buf, LocalRangeLockTablePrefix...)
buf = append(buf, LockTableSingleKeyInfix...)
buf = encoding.EncodeBytesAscending(buf, key)
return buf
return buf, buf
}

// DecodeLockTableSingleKey decodes the single-key lock table key to return the key
Expand Down Expand Up @@ -448,6 +454,12 @@ func IsLocal(k roachpb.Key) bool {
return bytes.HasPrefix(k, localPrefix)
}

// IsLocalStoreKey performs a cheap check that returns true iff the parameter
// is a local store key.
func IsLocalStoreKey(k roachpb.Key) bool {
return bytes.HasPrefix(k, localStorePrefix)
}

// Addr returns the address for the key, used to lookup the range containing the
// key. In the normal case, this is simply the key's value. However, for local
// keys, such as transaction records, the address is the inner encoded key, with
Expand Down
2 changes: 1 addition & 1 deletion pkg/keys/keys_test.go
Original file line number Diff line number Diff line change
Expand Up @@ -718,7 +718,7 @@ func TestLockTableKeyEncodeDecode(t *testing.T) {
}
for _, test := range testCases {
t.Run("", func(t *testing.T) {
ltKey := LockTableSingleKey(test.key)
ltKey, _ := LockTableSingleKey(test.key, nil)
require.True(t, bytes.HasPrefix(ltKey, expectedPrefix))
k, err := DecodeLockTableSingleKey(ltKey)
require.NoError(t, err)
Expand Down
4 changes: 4 additions & 0 deletions pkg/storage/BUILD.bazel
Original file line number Diff line number Diff line change
Expand Up @@ -14,6 +14,7 @@ go_library(
"error.go",
"file_util.go",
"in_mem.go",
"intent_interleaving_iter.go",
"multi_iterator.go",
"mvcc.go",
"mvcc_incremental_iterator.go",
Expand Down Expand Up @@ -81,6 +82,7 @@ go_library(
"//pkg/settings/cluster",
"//pkg/storage/enginepb",
"//pkg/storage/fs",
"//pkg/util",
"//pkg/util/bufalloc",
"//pkg/util/encoding",
"//pkg/util/envutil",
Expand Down Expand Up @@ -116,6 +118,7 @@ go_test(
"disk_map_test.go",
"engine_key_test.go",
"engine_test.go",
"intent_interleaving_iter_test.go",
"main_test.go",
"multi_iterator_test.go",
"mvcc_history_test.go",
Expand Down Expand Up @@ -147,6 +150,7 @@ go_test(
"//pkg/testutils",
"//pkg/testutils/skip",
"//pkg/testutils/zerofields",
"//pkg/util",
"//pkg/util/caller",
"//pkg/util/encoding",
"//pkg/util/fileutil",
Expand Down
9 changes: 6 additions & 3 deletions pkg/storage/engine.go
Original file line number Diff line number Diff line change
Expand Up @@ -60,7 +60,8 @@ type SimpleMVCCIterator interface {
// NextKey advances the iterator to the next MVCC key. This operation is
// distinct from Next which advances to the next version of the current key
// or the next key if the iterator is currently located at the last version
// for a key.
// for a key. NextKey must not be used to switch iteration direction from
// reverse iteration to forward iteration.
NextKey()
// UnsafeKey returns the same value as Key, but the memory is invalidated on
// the next call to {Next,NextKey,Prev,SeekGE,SeekLT,Close}.
Expand Down Expand Up @@ -125,7 +126,8 @@ type MVCCIterator interface {
// and the encoded SST data specified, within the provided key range. Returns
// stats on skipped KVs, or an error if a collision is found.
CheckForKeyCollisions(sstData []byte, start, end roachpb.Key) (enginepb.MVCCStats, error)
// SetUpperBound installs a new upper bound for this iterator.
// SetUpperBound installs a new upper bound for this iterator. The caller can modify
// the parameter after this function returns.
SetUpperBound(roachpb.Key)
// Stats returns statistics about the iterator.
Stats() IteratorStats
Expand Down Expand Up @@ -294,7 +296,8 @@ type Reader interface {
NewMVCCIterator(iterKind MVCCIterKind, opts IterOptions) MVCCIterator
// NewEngineIterator returns a new instance of an EngineIterator over this
// engine. The caller must invoke EngineIterator.Close() when finished
// with the iterator to free resources.
// with the iterator to free resources. The caller can change IterOptions
// after this function returns.
NewEngineIterator(opts IterOptions) EngineIterator
}

Expand Down
3 changes: 2 additions & 1 deletion pkg/storage/engine_key.go
Original file line number Diff line number Diff line change
Expand Up @@ -251,8 +251,9 @@ func (lk LockTableKey) ToEngineKey() EngineKey {
if lk.Strength != lock.Exclusive {
panic("unsupported lock strength")
}
ltKey, _ := keys.LockTableSingleKey(lk.Key, nil)
k := EngineKey{
Key: keys.LockTableSingleKey(lk.Key),
Key: ltKey,
Version: make([]byte, engineKeyVersionLockTableLen),
}
k.Version[0] = byte(lk.Strength)
Expand Down
Loading

0 comments on commit 95b836d

Please sign in to comment.