diff --git a/pkg/kv/kvserver/concurrency/lock_table.go b/pkg/kv/kvserver/concurrency/lock_table.go index e1edd6e0e9dd..cfedbd1146e9 100644 --- a/pkg/kv/kvserver/concurrency/lock_table.go +++ b/pkg/kv/kvserver/concurrency/lock_table.go @@ -1093,6 +1093,9 @@ func (ulh *unreplicatedLockHolderInfo) rollbackIgnoredSeqNumbers( if len(ignoredSeqNums) == 0 { return } + // NOTE: this logic differs slightly from replicated lock acquisition, where + // we don't rollback locks at ignored sequence numbers unless they are the + // same strength as the lock acquisition. See the comment in MVCCAcquireLock. for strIdx, minSeqNumber := range ulh.strengths { if minSeqNumber == -1 { continue diff --git a/pkg/kv/kvserver/mvcc_gc_queue.go b/pkg/kv/kvserver/mvcc_gc_queue.go index b2dbcf3ab1f5..0d2675370c1e 100644 --- a/pkg/kv/kvserver/mvcc_gc_queue.go +++ b/pkg/kv/kvserver/mvcc_gc_queue.go @@ -288,9 +288,10 @@ func makeMVCCGCQueueScore( gcTTL time.Duration, canAdvanceGCThreshold bool, ) mvccGCQueueScore { - repl.mu.Lock() + repl.mu.RLock() ms := *repl.mu.state.Stats - repl.mu.Unlock() + hint := *repl.mu.state.GCHint + repl.mu.RUnlock() if repl.store.cfg.TestingKnobs.DisableLastProcessedCheck { lastGC = hlc.Timestamp{} @@ -301,7 +302,7 @@ func makeMVCCGCQueueScore( // trigger GC at the same time. r := makeMVCCGCQueueScoreImpl( ctx, int64(repl.RangeID), now, ms, gcTTL, lastGC, canAdvanceGCThreshold, - repl.GetGCHint(), gc.TxnCleanupThreshold.Get(&repl.ClusterSettings().SV), + hint, gc.TxnCleanupThreshold.Get(&repl.ClusterSettings().SV), ) return r } diff --git a/pkg/storage/BUILD.bazel b/pkg/storage/BUILD.bazel index 1c37f876d816..a879fc6f32f4 100644 --- a/pkg/storage/BUILD.bazel +++ b/pkg/storage/BUILD.bazel @@ -16,6 +16,7 @@ go_library( "in_mem.go", "intent_interleaving_iter.go", "lock_table_iterator.go", + "lock_table_key_scanner.go", "min_version.go", "mvcc.go", "mvcc_incremental_iterator.go", diff --git a/pkg/storage/lock_table_key_scanner.go b/pkg/storage/lock_table_key_scanner.go new file mode 100644 index 000000000000..33a96e534dc7 --- /dev/null +++ b/pkg/storage/lock_table_key_scanner.go @@ -0,0 +1,307 @@ +// Copyright 2023 The Cockroach Authors. +// +// Use of this software is governed by the Business Source License +// included in the file licenses/BSL.txt. +// +// As of the Change Date specified in that file, in accordance with +// the Business Source License, use of this software will be governed +// by the Apache License, Version 2.0, included in the file +// licenses/APL.txt. + +package storage + +import ( + "sync" + + "github.com/cockroachdb/cockroach/pkg/keys" + "github.com/cockroachdb/cockroach/pkg/kv/kvpb" + "github.com/cockroachdb/cockroach/pkg/kv/kvserver/concurrency/lock" + "github.com/cockroachdb/cockroach/pkg/roachpb" + "github.com/cockroachdb/cockroach/pkg/storage/enginepb" + "github.com/cockroachdb/cockroach/pkg/util/uuid" + "github.com/cockroachdb/errors" +) + +// Fixed length slice for all supported lock strengths for replicated locks. May +// be used to iterate supported lock strengths in strength order (strongest to +// weakest). +var replicatedLockStrengths = [...]lock.Strength{lock.Intent, lock.Exclusive, lock.Shared} + +func init() { + if replicatedLockStrengths[0] != lock.MaxStrength { + panic("replicatedLockStrengths[0] != lock.MaxStrength; update replicatedLockStrengths?") + } +} + +// replicatedLockStrengthToIndexMap returns a mapping between (strength, index) +// pairs that can be used to index into the lockTableScanner.ownLocks array. +// +// Trying to use a lock strength that isn't supported with replicated locks to +// index into the lockTableScanner.ownLocks array will cause a runtime error. +var replicatedLockStrengthToIndexMap = func() (m [lock.MaxStrength + 1]int) { + // Initialize all to -1. + for str := range m { + m[str] = -1 + } + // Set the indices of the valid strengths. + for i, str := range replicatedLockStrengths { + m[str] = i + } + return m +}() + +// strongerOrEqualStrengths returns all supported lock strengths for replicated +// locks that are as strong or stronger than the provided strength. The returned +// slice is ordered from strongest to weakest. +func strongerOrEqualStrengths(str lock.Strength) []lock.Strength { + return replicatedLockStrengths[:replicatedLockStrengthToIndexMap[str]+1] +} + +// minConflictLockStrength returns the minimum lock strength that conflicts with +// the provided lock strength. +func minConflictLockStrength(str lock.Strength) (lock.Strength, error) { + switch str { + case lock.Shared: + return lock.Exclusive, nil + case lock.Exclusive, lock.Intent: + return lock.Shared, nil + default: + return 0, errors.AssertionFailedf( + "lockTableKeyScanner: unexpected lock strength %s", str.String()) + } +} + +// lockTableKeyScanner is used to scan a single key in the replicated lock +// table. It searches for locks on the key that conflict with a (transaction, +// lock strength) pair and for locks that the transaction has already acquired +// on the key. +// +// The purpose of a lockTableKeyScanner is to determine whether a transaction +// can acquire a lock on a key or perform an MVCC mutation on a key, and if so, +// what lock table keys the transaction should write to perform the operation. +type lockTableKeyScanner struct { + iter *LockTableIterator + // The transaction attempting to acquire a lock. The ID will be zero if a + // non-transactional request is attempting to perform an MVCC mutation. + txnID uuid.UUID + // Stop adding conflicting locks and abort scan once the maxConflicts limit + // is reached. Ignored if zero. + maxConflicts int64 + + // Stores any error returned. If non-nil, iteration short circuits. + err error + // Stores any locks that conflict with the transaction and locking strength. + conflicts []roachpb.Lock + // Stores any locks that the transaction has already acquired. + ownLocks [len(replicatedLockStrengths)]*enginepb.MVCCMetadata + + // Avoids heap allocations. + ltKeyBuf []byte + ltValue enginepb.MVCCMetadata + firstOwnLock enginepb.MVCCMetadata +} + +var lockTableKeyScannerPool = sync.Pool{ + New: func() interface{} { return new(lockTableKeyScanner) }, +} + +// newLockTableKeyScanner creates a new lockTableKeyScanner. +// +// txn is the transaction attempting to acquire locks. If txn is not nil, locks +// held by the transaction with any strength will be accumulated into the +// ownLocks array. Otherwise, if txn is nil, the request is non-transactional +// and no locks will be accumulated into the ownLocks array. +// +// str is the strength of the lock that the transaction (or non-transactional +// request) is attempting to acquire. The scanner will search for locks held by +// other transactions that conflict with this strength. +// +// maxConflicts is the maximum number of conflicting locks that the scanner +// should accumulate before returning an error. If maxConflicts is zero, the +// scanner will accumulate all conflicting locks. +func newLockTableKeyScanner( + reader Reader, txn *roachpb.Transaction, str lock.Strength, maxConflicts int64, +) (*lockTableKeyScanner, error) { + var txnID uuid.UUID + if txn != nil { + txnID = txn.ID + } + minConflictStr, err := minConflictLockStrength(str) + if err != nil { + return nil, err + } + iter, err := NewLockTableIterator(reader, LockTableIteratorOptions{ + Prefix: true, + MatchTxnID: txnID, + MatchMinStr: minConflictStr, + }) + if err != nil { + return nil, err + } + s := lockTableKeyScannerPool.Get().(*lockTableKeyScanner) + s.iter = iter + s.txnID = txnID + s.maxConflicts = maxConflicts + return s, nil +} + +func (s *lockTableKeyScanner) close() { + s.iter.Close() + *s = lockTableKeyScanner{ltKeyBuf: s.ltKeyBuf} + lockTableKeyScannerPool.Put(s) +} + +// scan scans the lock table at the provided key for locks held by other +// transactions that conflict with the configured locking strength and for locks +// of any strength that the configured transaction has already acquired. +func (s *lockTableKeyScanner) scan(key roachpb.Key) error { + s.resetScanState() + for ok := s.seek(key); ok; ok = s.getOneAndAdvance() { + } + return s.afterScan() +} + +// resetScanState resets the scanner's state before a scan. +func (s *lockTableKeyScanner) resetScanState() { + s.err = nil + s.conflicts = nil + for i := range s.ownLocks { + s.ownLocks[i] = nil + } + s.ltValue.Reset() + s.firstOwnLock.Reset() +} + +// afterScan returns any error encountered during the scan. +func (s *lockTableKeyScanner) afterScan() error { + if s.err != nil { + return s.err + } + if len(s.conflicts) != 0 { + return &kvpb.LockConflictError{Locks: s.conflicts} + } + return nil +} + +// seek seeks the iterator to the first lock table key associated with the +// provided key. Returns true if the scanner should continue scanning, false +// if not. +func (s *lockTableKeyScanner) seek(key roachpb.Key) bool { + var ltKey roachpb.Key + ltKey, s.ltKeyBuf = keys.LockTableSingleKey(key, s.ltKeyBuf) + valid, err := s.iter.SeekEngineKeyGE(EngineKey{Key: ltKey}) + if err != nil { + s.err = err + } + return valid +} + +// getOneAndAdvance consumes the current lock table key and value and advances +// the iterator. Returns true if the scanner should continue scanning, false if +// not. +func (s *lockTableKeyScanner) getOneAndAdvance() bool { + ltKey, ok := s.getLockTableKey() + if !ok { + return false + } + ltValue, ok := s.getLockTableValue() + if !ok { + return false + } + if !s.consumeLockTableKeyValue(ltKey, ltValue) { + return false + } + return s.advance() +} + +// advance advances the iterator to the next lock table key. +func (s *lockTableKeyScanner) advance() bool { + valid, err := s.iter.NextEngineKey() + if err != nil { + s.err = err + } + return valid +} + +// getLockTableKey decodes the current lock table key. +func (s *lockTableKeyScanner) getLockTableKey() (LockTableKey, bool) { + ltEngKey, err := s.iter.UnsafeEngineKey() + if err != nil { + s.err = err + return LockTableKey{}, false + } + ltKey, err := ltEngKey.ToLockTableKey() + if err != nil { + s.err = err + return LockTableKey{}, false + } + return ltKey, true +} + +// getLockTableValue decodes the current lock table values. +func (s *lockTableKeyScanner) getLockTableValue() (*enginepb.MVCCMetadata, bool) { + err := s.iter.ValueProto(&s.ltValue) + if err != nil { + s.err = err + return nil, false + } + return &s.ltValue, true +} + +// consumeLockTableKeyValue consumes the current lock table key and value, which +// is either a conflicting lock or a lock held by the scanning transaction. +func (s *lockTableKeyScanner) consumeLockTableKeyValue( + ltKey LockTableKey, ltValue *enginepb.MVCCMetadata, +) bool { + if ltValue.Txn == nil { + s.err = errors.AssertionFailedf("unexpectedly found non-transactional lock: %v", ltValue) + return false + } + if ltKey.TxnUUID != ltValue.Txn.ID { + s.err = errors.AssertionFailedf("lock table key (%+v) and value (%+v) txn ID mismatch", ltKey, ltValue) + return false + } + if ltKey.TxnUUID == s.txnID { + return s.consumeOwnLock(ltKey, ltValue) + } + return s.consumeConflictingLock(ltKey, ltValue) +} + +// consumeOwnLock consumes a lock held by the scanning transaction. +func (s *lockTableKeyScanner) consumeOwnLock( + ltKey LockTableKey, ltValue *enginepb.MVCCMetadata, +) bool { + var ltValueCopy *enginepb.MVCCMetadata + if s.firstOwnLock.Txn == nil { + // This is the first lock held by the transaction that we've seen, so + // we can avoid the heap allocation. + ltValueCopy = &s.firstOwnLock + } else { + ltValueCopy = new(enginepb.MVCCMetadata) + } + // NOTE: this will alias internal pointer fields of ltValueCopy with those + // in ltValue, but this will not lead to issues when ltValue is updated by + // the next call to getLockTableValue, because its internal fields will be + // reset by protoutil.Unmarshal before unmarshalling. + *ltValueCopy = *ltValue + s.ownLocks[replicatedLockStrengthToIndexMap[ltKey.Strength]] = ltValueCopy + return true +} + +// consumeConflictingLock consumes a conflicting lock. +func (s *lockTableKeyScanner) consumeConflictingLock( + ltKey LockTableKey, ltValue *enginepb.MVCCMetadata, +) bool { + conflict := roachpb.MakeLock(ltValue.Txn, ltKey.Key.Clone(), ltKey.Strength) + s.conflicts = append(s.conflicts, conflict) + if s.maxConflicts != 0 && s.maxConflicts == int64(len(s.conflicts)) { + return false + } + return true +} + +// foundOwn returns the lock table value for the provided strength if the +// transaction has already acquired a lock of that strength. Returns nil if not. +func (s *lockTableKeyScanner) foundOwn(str lock.Strength) *enginepb.MVCCMetadata { + return s.ownLocks[replicatedLockStrengthToIndexMap[str]] +} diff --git a/pkg/storage/mvcc.go b/pkg/storage/mvcc.go index 1467c60d05ca..5b436a36aeb3 100644 --- a/pkg/storage/mvcc.go +++ b/pkg/storage/mvcc.go @@ -1479,17 +1479,25 @@ func (b *putBuffer) putInlineMeta( var trueValue = true -// putIntentMeta puts an intent at the given key with the provided value. -func (b *putBuffer) putIntentMeta( - writer Writer, key MVCCKey, meta *enginepb.MVCCMetadata, alreadyExists bool, +// putLockMeta puts a lock at the given key with the provided strength and +// value. +func (b *putBuffer) putLockMeta( + writer Writer, key MVCCKey, str lock.Strength, meta *enginepb.MVCCMetadata, alreadyExists bool, ) (keyBytes, valBytes int64, err error) { - if meta.Txn != nil && meta.Timestamp.ToTimestamp() != meta.Txn.WriteTimestamp { - // The timestamps are supposed to be in sync. If they weren't, it wouldn't - // be clear for readers which one to use for what. - return 0, 0, errors.AssertionFailedf( - "meta.Timestamp != meta.Txn.WriteTimestamp: %s != %s", meta.Timestamp, meta.Txn.WriteTimestamp) + if str == lock.Intent { + if meta.Timestamp.ToTimestamp() != meta.Txn.WriteTimestamp { + // The timestamps are supposed to be in sync. If they weren't, it wouldn't + // be clear for readers which one to use for what. + return 0, 0, errors.AssertionFailedf( + "meta.Timestamp != meta.Txn.WriteTimestamp: %s != %s", meta.Timestamp, meta.Txn.WriteTimestamp) + } + } else { + if !meta.Timestamp.ToTimestamp().IsEmpty() { + return 0, 0, errors.AssertionFailedf( + "meta.Timestamp not zero for lock with strength %s", str.String()) + } } - lockTableKey := b.lockTableKey(key.Key, lock.Intent, meta.Txn.ID) + lockTableKey := b.lockTableKey(key.Key, str, meta.Txn.ID) if alreadyExists { // Absence represents false. meta.TxnDidNotUpdateMeta = nil @@ -1506,19 +1514,26 @@ func (b *putBuffer) putIntentMeta( return int64(key.EncodedSize()), int64(len(bytes)), nil } -// clearIntentMeta clears an intent at the given key. txnDidNotUpdateMeta allows -// for performance optimization when set to true, and has semantics defined in -// MVCCMetadata.TxnDidNotUpdateMeta (it can be conservatively set to false). +// clearLockMeta clears a lock at the given key and strength. +// +// txnDidNotUpdateMeta allows for performance optimization when set to true, and +// has semantics defined in MVCCMetadata.TxnDidNotUpdateMeta (it can be +// conservatively set to false). // // TODO(sumeer): after the full transition to separated locks, measure the cost -// of a putIntentMeta implementation, where there is an existing intent, that -// does a pair. If there isn't a performance decrease, we -// can stop tracking txnDidNotUpdateMeta and still optimize clearIntentMeta by -// always doing single-clear. -func (b *putBuffer) clearIntentMeta( - writer Writer, key MVCCKey, txnDidNotUpdateMeta bool, txnUUID uuid.UUID, opts ClearOptions, +// of a putLockMeta implementation, where there is an existing intent, that does +// a pair. If there isn't a performance decrease, we can +// stop tracking txnDidNotUpdateMeta and still optimize clearLockMeta by always +// doing single-clear. +func (b *putBuffer) clearLockMeta( + writer Writer, + key MVCCKey, + str lock.Strength, + txnDidNotUpdateMeta bool, + txnUUID uuid.UUID, + opts ClearOptions, ) (keyBytes, valBytes int64, err error) { - lockTableKey := b.lockTableKey(key.Key, lock.Intent, txnUUID) + lockTableKey := b.lockTableKey(key.Key, str, txnUUID) if txnDidNotUpdateMeta { err = writer.SingleClearEngineKey(lockTableKey) } else { @@ -2251,8 +2266,8 @@ func mvccPutInternal( // represents a non-manufactured meta, i.e., there is an intent. alreadyExists := ok && buf.meta.Txn != nil // Write the intent metadata key. - metaKeySize, metaValSize, err = buf.putIntentMeta( - writer, metaKey, newMeta, alreadyExists) + metaKeySize, metaValSize, err = buf.putLockMeta( + writer, metaKey, lock.Intent, newMeta, alreadyExists) if err != nil { return false, err } @@ -4959,11 +4974,11 @@ func mvccResolveWriteIntent( // overwriting a newer epoch (see comments above). The pusher's job isn't // to do anything to update the intent but to move the timestamp forward, // even if it can. - metaKeySize, metaValSize, err = buf.putIntentMeta( - writer, metaKey, newMeta, true /* alreadyExists */) + metaKeySize, metaValSize, err = buf.putLockMeta( + writer, metaKey, lock.Intent, newMeta, true /* alreadyExists */) } else { - metaKeySize, metaValSize, err = buf.clearIntentMeta( - writer, metaKey, canSingleDelHelper.onCommitIntent(), meta.Txn.ID, ClearOptions{ + metaKeySize, metaValSize, err = buf.clearLockMeta( + writer, metaKey, lock.Intent, canSingleDelHelper.onCommitIntent(), meta.Txn.ID, ClearOptions{ ValueSizeKnown: true, ValueSize: uint32(origMetaValSize), }) @@ -5074,8 +5089,8 @@ func mvccResolveWriteIntent( if !ok { // If there is no other version, we should just clean up the key entirely. - _, _, err := buf.clearIntentMeta( - writer, metaKey, canSingleDelHelper.onAbortIntent(), meta.Txn.ID, ClearOptions{ + _, _, err := buf.clearLockMeta( + writer, metaKey, lock.Intent, canSingleDelHelper.onAbortIntent(), meta.Txn.ID, ClearOptions{ ValueSizeKnown: true, ValueSize: uint32(origMetaValSize), }) @@ -5096,8 +5111,8 @@ func mvccResolveWriteIntent( KeyBytes: MVCCVersionTimestampSize, ValBytes: int64(nextValueLen), } - metaKeySize, metaValSize, err := buf.clearIntentMeta( - writer, metaKey, canSingleDelHelper.onAbortIntent(), meta.Txn.ID, ClearOptions{ + metaKeySize, metaValSize, err := buf.clearLockMeta( + writer, metaKey, lock.Intent, canSingleDelHelper.onAbortIntent(), meta.Txn.ID, ClearOptions{ ValueSizeKnown: true, ValueSize: uint32(origMetaValSize), }) @@ -5291,6 +5306,164 @@ func MVCCResolveWriteIntentRange( return numKeys, numBytes, nil, 0, nil } +// MVCCCheckForAcquireLock scans the replicated lock table to determine whether +// a lock acquisition at the specified key and strength by the specified +// transaction would succeed. If the lock table scan finds one or more existing +// locks on the key that conflict with the acquisition then a LockConflictError +// is returned. Otherwise, nil is returned. Unlike MVCCAcquireLock, this method +// does not actually acquire the lock (i.e. write to the lock table). +func MVCCCheckForAcquireLock( + ctx context.Context, + reader Reader, + txn *roachpb.Transaction, + str lock.Strength, + key roachpb.Key, + maxConflicts int64, +) error { + if err := validateLockAcquisition(txn, str); err != nil { + return err + } + ltScanner, err := newLockTableKeyScanner(reader, txn, str, maxConflicts) + if err != nil { + return err + } + defer ltScanner.close() + return ltScanner.scan(key) +} + +// MVCCAcquireLock attempts to acquire a lock at the specified key and strength +// by the specified transaction. It first scans the replicated lock table to +// determine whether any conflicting locks are held by other transactions. If +// so, a LockConflictError is returned. Otherwise, the lock is written to the +// lock table and nil is returned. +func MVCCAcquireLock( + ctx context.Context, + rw ReadWriter, + txn *roachpb.Transaction, + str lock.Strength, + key roachpb.Key, + ms *enginepb.MVCCStats, + maxConflicts int64, +) error { + if err := validateLockAcquisition(txn, str); err != nil { + return err + } + ltScanner, err := newLockTableKeyScanner(rw, txn, str, maxConflicts) + if err != nil { + return err + } + defer ltScanner.close() + err = ltScanner.scan(key) + if err != nil { + return err + } + + // Iterate over the replicated lock strengths, from strongest to weakest, + // stopping at the lock strength that we'd like to acquire. If the loop + // terminates, rolledBack will reference the desired lock strength. + var rolledBack bool + for _, iterStr := range strongerOrEqualStrengths(str) { + rolledBack = false + foundLock := ltScanner.foundOwn(iterStr) + if foundLock == nil { + // Proceed to check weaker strengths... + continue + } + + if foundLock.Txn.Epoch > txn.Epoch { + // Acquiring at old epoch. + return errors.Errorf( + "locking request with epoch %d came after lock "+ + "had already been acquired at epoch %d in txn %s", + txn.Epoch, foundLock.Txn.Epoch, txn.ID) + } else if foundLock.Txn.Epoch < txn.Epoch { + // Acquiring at new epoch. + rolledBack = true + } else if foundLock.Txn.Sequence > txn.Sequence { + // Acquiring at same epoch and old sequence number. + return errors.Errorf( + "cannot acquire lock with strength %s at seq number %d, "+ + "already held at higher seq number %d", + str.String(), txn.Sequence, foundLock.Txn.Sequence) + } else if enginepb.TxnSeqIsIgnored(foundLock.Txn.Sequence, txn.IgnoredSeqNums) { + // Acquiring at same epoch and new sequence number after + // previous sequence number was rolled back. + // + // TODO(nvanbenschoten): If this is a stronger strength than + // we're trying to acquire, then it would be an option to + // release this lock/intent at the same time as we acquire the + // new, weaker lock at higher, non-rolled back sequence number. + // This is what we do for unreplicated locks in the lock table. + // + // We don't currently do this for replicated locks because lock + // acquisition may be holding weaker latches than are needed to + // release locks at the stronger strength. This could lead to a race + // where concurrent work that conflicts with the existing lock but + // not the latches held by this acquisition discovers the lock and + // reports it to the lock table. The in-memory lock table could then + // get out of sync with the replicated lock table. + if iterStr != lock.Intent { + rolledBack = true + } else { + // If the existing lock is an intent, additionally check the + // intent history to verify that all of the intent writes in + // the intent history are also rolled back. If not, then we + // can still avoid reacquisition. + inHistoryNotRolledBack := false + for _, e := range foundLock.IntentHistory { + if !enginepb.TxnSeqIsIgnored(e.Sequence, txn.IgnoredSeqNums) { + inHistoryNotRolledBack = true + break + } + } + rolledBack = !inHistoryNotRolledBack + } + } + + if !rolledBack { + // Lock held at desired or stronger strength. No need to reacquire. + // This is both a performance optimization and a necessary check for + // correctness. If we were to reacquire the lock at a newer sequence + // number and clobber the existing lock with its older sequence + // number, our newer sequence number could then be rolled back and + // we would forget that the lock held at the older sequence number + // had been and still should be held. + log.VEventf(ctx, 3, "skipping lock acquisition for txn %s on key %s "+ + "with strength %s; found existing lock with strength %s and sequence %d", + txn, key.String(), str.String(), iterStr.String(), foundLock.Txn.Sequence) + return nil + } + + // Proceed to check weaker strengths... + } + + // Write the lock. + buf := newPutBuffer() + defer buf.release() + + newMeta := &buf.newMeta + newMeta.Txn = &txn.TxnMeta + keyBytes, valBytes, err := buf.putLockMeta(rw, MakeMVCCMetadataKey(key), str, newMeta, rolledBack) + if err != nil { + return err + } + + // TODO(nvanbenschoten): handle MVCCStats update after addressing #109645. + _, _, _ = ms, keyBytes, valBytes + + return nil +} + +func validateLockAcquisition(txn *roachpb.Transaction, str lock.Strength) error { + if txn == nil { + return errors.Errorf("txn must be non-nil to acquire lock") + } + if !(str == lock.Shared || str == lock.Exclusive) { + return errors.Errorf("invalid lock strength to acquire lock: %s", str.String()) + } + return nil +} + // MVCCGarbageCollect creates an iterator on the ReadWriter. In parallel // it iterates through the keys listed for garbage collection by the // keys slice. The iterator is seeked in turn to each listed diff --git a/pkg/storage/mvcc_history_test.go b/pkg/storage/mvcc_history_test.go index de394ba89980..5d01ab813067 100644 --- a/pkg/storage/mvcc_history_test.go +++ b/pkg/storage/mvcc_history_test.go @@ -75,17 +75,19 @@ var ( // // txn_begin t= [ts=[,]] [globalUncertaintyLimit=[,]] // txn_remove t= -// txn_restart t= +// txn_restart t= [epoch=] // txn_update t= t2= -// txn_step t= [n=] +// txn_step t= [n=] [seq=] // txn_advance t= ts=[,] // txn_status t= status= // txn_ignore_seqs t= seqs=[-[,-...]] // -// resolve_intent t= k= [status=] [clockWhilePending=[,]] [targetBytes=] -// resolve_intent_range t= k= end= [status=] [maxKeys=] [targetBytes=] -// check_intent k= [none] -// add_lock t= k= +// resolve_intent t= k= [status=] [clockWhilePending=[,]] [targetBytes=] +// resolve_intent_range t= k= end= [status=] [maxKeys=] [targetBytes=] +// check_intent k= [none] +// add_unreplicated_lock t= k= +// check_for_acquire_lock t= k= str= +// acquire_lock t= k= str= // // cput [t=] [ts=[,]] [localTs=[,]] [resolve [status=]] [ambiguousReplay] k= v= [raw] [cond=] // del [t=] [ts=[,]] [localTs=[,]] [resolve [status=]] [ambiguousReplay] k= @@ -349,7 +351,60 @@ func TestMVCCHistories(t *testing.T) { } } } + return nil + } + + // reportLockTable outputs the contents of the lock table. + reportLockTable := func(e *evalCtx, buf *redact.StringBuilder) error { + // Replicated locks. + ltStart := keys.LocalRangeLockTablePrefix + ltEnd := keys.LocalRangeLockTablePrefix.PrefixEnd() + iter, err := engine.NewEngineIterator(storage.IterOptions{UpperBound: ltEnd}) + if err != nil { + return err + } + defer iter.Close() + var meta enginepb.MVCCMetadata + for valid, err := iter.SeekEngineKeyGE(storage.EngineKey{Key: ltStart}); ; valid, err = iter.NextEngineKey() { + if err != nil { + return err + } else if !valid { + break + } + eKey, err := iter.EngineKey() + if err != nil { + return err + } + ltKey, err := eKey.ToLockTableKey() + if err != nil { + return errors.Wrapf(err, "decoding LockTable key: %v", eKey) + } + // Unmarshal. + v, err := iter.UnsafeValue() + if err != nil { + return err + } + if err := protoutil.Unmarshal(v, &meta); err != nil { + return errors.Wrapf(err, "unmarshaling mvcc meta: %v", ltKey) + } + buf.Printf("lock (%s): %v/%s -> %+v\n", + lock.Replicated, ltKey.Key, ltKey.Strength, &meta) + } + + // Unreplicated locks. + if len(e.unreplLocks) > 0 { + var ks []string + for k := range e.unreplLocks { + ks = append(ks, k) + } + sort.Strings(ks) + for _, k := range ks { + txn := e.unreplLocks[k] + buf.Printf("lock (%s): %v/%s -> %+v\n", + lock.Unreplicated, k, lock.Exclusive, txn) + } + } return nil } @@ -470,19 +525,15 @@ func TestMVCCHistories(t *testing.T) { } } if printLocks { - var ks []string - for k := range e.locks { - ks = append(ks, k) - } - sort.Strings(ks) - buf.Printf("lock-table: {") - for i, k := range ks { - if i > 0 { - buf.Printf(", ") + err = reportLockTable(e, &buf) + if err != nil { + if foundErr == nil { + // Handle the error below. + foundErr = err + } else { + buf.Printf("error reading locks: (%T:) %v\n", err, err) } - buf.Printf("%s:%s", k, e.locks[k].ID) } - buf.Printf("}\n") } } @@ -720,10 +771,12 @@ var commands = map[string]cmd{ "txn_step": {typTxnUpdate, cmdTxnStep}, "txn_update": {typTxnUpdate, cmdTxnUpdate}, - "resolve_intent": {typDataUpdate, cmdResolveIntent}, - "resolve_intent_range": {typDataUpdate, cmdResolveIntentRange}, - "check_intent": {typReadOnly, cmdCheckIntent}, - "add_lock": {typLocksUpdate, cmdAddLock}, + "resolve_intent": {typDataUpdate, cmdResolveIntent}, + "resolve_intent_range": {typDataUpdate, cmdResolveIntentRange}, + "check_intent": {typReadOnly, cmdCheckIntent}, + "add_unreplicated_lock": {typLocksUpdate, cmdAddUnreplicatedLock}, + "check_for_acquire_lock": {typReadOnly, cmdCheckForAcquireLock}, + "acquire_lock": {typLocksUpdate, cmdAcquireLock}, "clear": {typDataUpdate, cmdClear}, "clear_range": {typDataUpdate, cmdClearRange}, @@ -834,6 +887,11 @@ func cmdTxnRestart(e *evalCtx) error { up := roachpb.NormalUserPriority tp := enginepb.MinTxnPriority txn.Restart(up, tp, ts) + if e.hasArg("epoch") { + var epoch int + e.scanArg("epoch", &epoch) + txn.Epoch = enginepb.TxnEpoch(epoch) + } e.results.txn = txn return nil } @@ -994,13 +1052,31 @@ func cmdCheckIntent(e *evalCtx) error { }) } -func cmdAddLock(e *evalCtx) error { +func cmdAddUnreplicatedLock(e *evalCtx) error { txn := e.getTxn(mandatory) key := e.getKey() - e.locks[string(key)] = txn + e.unreplLocks[string(key)] = &txn.TxnMeta return nil } +func cmdCheckForAcquireLock(e *evalCtx) error { + return e.withReader(func(r storage.Reader) error { + txn := e.getTxn(optional) + key := e.getKey() + str := e.getStrength() + return storage.MVCCCheckForAcquireLock(e.ctx, r, txn, str, key, 0) + }) +} + +func cmdAcquireLock(e *evalCtx) error { + return e.withWriter("acquire_lock", func(rw storage.ReadWriter) error { + txn := e.getTxn(optional) + key := e.getKey() + str := e.getStrength() + return storage.MVCCAcquireLock(e.ctx, rw, txn, str, key, e.ms, 0) + }) +} + func cmdClear(e *evalCtx) error { key := e.getKey() ts := e.getTs(nil) @@ -2218,7 +2294,7 @@ type evalCtx struct { td *datadriven.TestData txns map[string]*roachpb.Transaction txnCounter uint32 - locks map[string]*roachpb.Transaction + unreplLocks map[string]*enginepb.TxnMeta ms *enginepb.MVCCStats sstWriter *storage.SSTWriter sstFile *storage.MemObject @@ -2227,11 +2303,11 @@ type evalCtx struct { func newEvalCtx(ctx context.Context, engine storage.Engine) *evalCtx { return &evalCtx{ - ctx: ctx, - st: cluster.MakeTestingClusterSettings(), - engine: engine, - txns: make(map[string]*roachpb.Transaction), - locks: make(map[string]*roachpb.Transaction), + ctx: ctx, + st: cluster.MakeTestingClusterSettings(), + engine: engine, + txns: make(map[string]*roachpb.Transaction), + unreplLocks: make(map[string]*enginepb.TxnMeta), } } @@ -2445,6 +2521,25 @@ func (e *evalCtx) getKeyRange() (sk, ek roachpb.Key) { return sk, ek } +func (e *evalCtx) getStrength() lock.Strength { + e.t.Helper() + var strS string + e.scanArg("str", &strS) + switch strS { + case "none": + return lock.None + case "shared": + return lock.Shared + case "exclusive": + return lock.Exclusive + case "intent": + return lock.Intent + default: + e.Fatalf("unknown lock strength: %s", strS) + return 0 + } +} + func (e *evalCtx) getTenantCodec() keys.SQLCodec { if e.hasArg("tenant-prefix") { var tenantID int @@ -2526,20 +2621,20 @@ func (e *evalCtx) lookupTxn(txnName string) (*roachpb.Transaction, error) { func (e *evalCtx) newLockTableView( txn *roachpb.Transaction, ts hlc.Timestamp, ) storage.LockTableView { - return &mockLockTableView{locks: e.locks, txn: txn, ts: ts} + return &mockLockTableView{unreplLocks: e.unreplLocks, txn: txn, ts: ts} } // mockLockTableView is a mock implementation of LockTableView. type mockLockTableView struct { - locks map[string]*roachpb.Transaction - txn *roachpb.Transaction - ts hlc.Timestamp + unreplLocks map[string]*enginepb.TxnMeta + txn *roachpb.Transaction + ts hlc.Timestamp } func (lt *mockLockTableView) IsKeyLockedByConflictingTxn( k roachpb.Key, s lock.Strength, ) (bool, *enginepb.TxnMeta, error) { - holder, ok := lt.locks[string(k)] + holder, ok := lt.unreplLocks[string(k)] if !ok { return false, nil, nil } @@ -2549,7 +2644,7 @@ func (lt *mockLockTableView) IsKeyLockedByConflictingTxn( if s == lock.None && lt.ts.Less(holder.WriteTimestamp) { return false, nil, nil } - return true, &holder.TxnMeta, nil + return true, holder, nil } func (e *evalCtx) visitWrappedIters(fn func(it storage.SimpleMVCCIterator) (done bool)) { diff --git a/pkg/storage/testdata/mvcc_histories/replicated_locks b/pkg/storage/testdata/mvcc_histories/replicated_locks new file mode 100644 index 000000000000..ba00d2fd09a6 --- /dev/null +++ b/pkg/storage/testdata/mvcc_histories/replicated_locks @@ -0,0 +1,318 @@ +run ok +txn_begin t=A ts=10,0 +txn_begin t=B ts=11,0 +---- +>> at end: +txn: "B" meta={id=00000002 key=/Min iso=Serializable pri=0.00000000 epo=0 ts=11.000000000,0 min=0,0 seq=0} lock=true stat=PENDING rts=11.000000000,0 wto=false gul=0,0 + +run ok +with t=A + check_for_acquire_lock k=k1 str=shared + check_for_acquire_lock k=k2 str=shared + check_for_acquire_lock k=k3 str=exclusive + acquire_lock k=k1 str=shared + acquire_lock k=k2 str=shared + acquire_lock k=k3 str=exclusive +---- +>> at end: +lock (Replicated): "k1"/Shared -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=0 ts=10.000000000,0 min=0,0 seq=0} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=true +lock (Replicated): "k2"/Shared -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=0 ts=10.000000000,0 min=0,0 seq=0} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=true +lock (Replicated): "k3"/Exclusive -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=0 ts=10.000000000,0 min=0,0 seq=0} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=true + +# Reacquire with weaker, equal, and stronger strengths. All should succeed, but +# only the stronger strength should actually write a new lock key. + +run ok +with t=A + acquire_lock k=k2 str=shared + acquire_lock k=k2 str=exclusive + acquire_lock k=k3 str=shared + acquire_lock k=k3 str=exclusive +---- +>> at end: +lock (Replicated): "k1"/Shared -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=0 ts=10.000000000,0 min=0,0 seq=0} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=true +lock (Replicated): "k2"/Exclusive -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=0 ts=10.000000000,0 min=0,0 seq=0} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=true +lock (Replicated): "k2"/Shared -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=0 ts=10.000000000,0 min=0,0 seq=0} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=true +lock (Replicated): "k3"/Exclusive -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=0 ts=10.000000000,0 min=0,0 seq=0} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=true + +# Reacquire with weaker, equal, and stronger strengths in new epoch. All should +# succeed, but only the stronger strength acquisitions (in the new epoch) should +# actually (re)write lock keys. + +run ok +with t=A + txn_restart + acquire_lock k=k1 str=shared + acquire_lock k=k2 str=shared + acquire_lock k=k2 str=exclusive + acquire_lock k=k3 str=exclusive + acquire_lock k=k3 str=shared +---- +>> at end: +txn: "A" meta={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=0} lock=true stat=PENDING rts=10.000000000,0 wto=false gul=0,0 +lock (Replicated): "k1"/Shared -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=0} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=false +lock (Replicated): "k2"/Exclusive -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=0} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=false +lock (Replicated): "k2"/Shared -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=0} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=false +lock (Replicated): "k3"/Exclusive -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=0} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=false + +# Reacquisition of the same locks in the same epoch with later sequences should +# be no-ops. + +run ok +with t=A + txn_step + acquire_lock k=k1 str=shared + acquire_lock k=k2 str=shared + acquire_lock k=k2 str=exclusive + acquire_lock k=k3 str=exclusive + acquire_lock k=k3 str=shared +---- +>> at end: +txn: "A" meta={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=1} lock=true stat=PENDING rts=10.000000000,0 wto=false gul=0,0 +lock (Replicated): "k1"/Shared -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=0} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=false +lock (Replicated): "k2"/Exclusive -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=0} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=false +lock (Replicated): "k2"/Shared -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=0} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=false +lock (Replicated): "k3"/Exclusive -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=0} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=false + +# Reacquisition of the same locks in the same epoch with later sequences after +# the earlier sequence has been rolled back should rewrite the locks with the +# newer sequence. + +run ok +with t=A + txn_ignore_seqs seqs=0-0 + acquire_lock k=k1 str=shared + acquire_lock k=k2 str=shared + acquire_lock k=k2 str=exclusive + acquire_lock k=k3 str=exclusive + acquire_lock k=k3 str=shared +---- +>> at end: +txn: "A" meta={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=1} lock=true stat=PENDING rts=10.000000000,0 wto=false gul=0,0 isn=1 +lock (Replicated): "k1"/Shared -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=1} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=false +lock (Replicated): "k2"/Exclusive -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=1} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=false +lock (Replicated): "k2"/Shared -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=1} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=false +lock (Replicated): "k3"/Exclusive -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=1} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=false + +# Txn B can only acquire a shared lock on k1. + +run ok +with t=B + check_for_acquire_lock k=k1 str=shared + acquire_lock k=k1 str=shared +---- +>> at end: +lock (Replicated): "k1"/Shared -> txn={id=00000002 key=/Min iso=Serializable pri=0.00000000 epo=0 ts=11.000000000,0 min=0,0 seq=0} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=true +lock (Replicated): "k1"/Shared -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=1} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=false +lock (Replicated): "k2"/Exclusive -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=1} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=false +lock (Replicated): "k2"/Shared -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=1} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=false +lock (Replicated): "k3"/Exclusive -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=1} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=false + +run error +check_for_acquire_lock t=B k=k1 str=exclusive +---- +error: (*kvpb.LockConflictError:) conflicting locks on "k1" + +run error +acquire_lock t=B k=k1 str=exclusive +---- +>> at end: +lock (Replicated): "k1"/Shared -> txn={id=00000002 key=/Min iso=Serializable pri=0.00000000 epo=0 ts=11.000000000,0 min=0,0 seq=0} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=true +lock (Replicated): "k1"/Shared -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=1} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=false +lock (Replicated): "k2"/Exclusive -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=1} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=false +lock (Replicated): "k2"/Shared -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=1} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=false +lock (Replicated): "k3"/Exclusive -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=1} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=false +error: (*kvpb.LockConflictError:) conflicting locks on "k1" + +run error +check_for_acquire_lock t=B k=k2 str=shared +---- +error: (*kvpb.LockConflictError:) conflicting locks on "k2" + +run error +acquire_lock t=B k=k2 str=shared +---- +>> at end: +lock (Replicated): "k1"/Shared -> txn={id=00000002 key=/Min iso=Serializable pri=0.00000000 epo=0 ts=11.000000000,0 min=0,0 seq=0} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=true +lock (Replicated): "k1"/Shared -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=1} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=false +lock (Replicated): "k2"/Exclusive -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=1} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=false +lock (Replicated): "k2"/Shared -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=1} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=false +lock (Replicated): "k3"/Exclusive -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=1} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=false +error: (*kvpb.LockConflictError:) conflicting locks on "k2" + +run error +check_for_acquire_lock t=B k=k2 str=exclusive +---- +error: (*kvpb.LockConflictError:) conflicting locks on "k2", "k2" + +run error +acquire_lock t=B k=k2 str=exclusive +---- +>> at end: +lock (Replicated): "k1"/Shared -> txn={id=00000002 key=/Min iso=Serializable pri=0.00000000 epo=0 ts=11.000000000,0 min=0,0 seq=0} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=true +lock (Replicated): "k1"/Shared -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=1} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=false +lock (Replicated): "k2"/Exclusive -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=1} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=false +lock (Replicated): "k2"/Shared -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=1} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=false +lock (Replicated): "k3"/Exclusive -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=1} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=false +error: (*kvpb.LockConflictError:) conflicting locks on "k2", "k2" + +run error +check_for_acquire_lock t=B k=k3 str=shared +---- +error: (*kvpb.LockConflictError:) conflicting locks on "k3" + +run error +acquire_lock t=B k=k3 str=shared +---- +>> at end: +lock (Replicated): "k1"/Shared -> txn={id=00000002 key=/Min iso=Serializable pri=0.00000000 epo=0 ts=11.000000000,0 min=0,0 seq=0} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=true +lock (Replicated): "k1"/Shared -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=1} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=false +lock (Replicated): "k2"/Exclusive -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=1} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=false +lock (Replicated): "k2"/Shared -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=1} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=false +lock (Replicated): "k3"/Exclusive -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=1} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=false +error: (*kvpb.LockConflictError:) conflicting locks on "k3" + +run error +check_for_acquire_lock t=B k=k3 str=exclusive +---- +error: (*kvpb.LockConflictError:) conflicting locks on "k3" + +run error +acquire_lock t=B k=k3 str=exclusive +---- +>> at end: +lock (Replicated): "k1"/Shared -> txn={id=00000002 key=/Min iso=Serializable pri=0.00000000 epo=0 ts=11.000000000,0 min=0,0 seq=0} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=true +lock (Replicated): "k1"/Shared -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=1} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=false +lock (Replicated): "k2"/Exclusive -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=1} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=false +lock (Replicated): "k2"/Shared -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=1} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=false +lock (Replicated): "k3"/Exclusive -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=1} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=false +error: (*kvpb.LockConflictError:) conflicting locks on "k3" + +# Now that there are two shared locks on key k1, txn A can no longer upgrade its lock. + +run error +check_for_acquire_lock t=A k=k1 str=exclusive +---- +error: (*kvpb.LockConflictError:) conflicting locks on "k1" + +run error +acquire_lock t=A k=k1 str=exclusive +---- +>> at end: +lock (Replicated): "k1"/Shared -> txn={id=00000002 key=/Min iso=Serializable pri=0.00000000 epo=0 ts=11.000000000,0 min=0,0 seq=0} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=true +lock (Replicated): "k1"/Shared -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=1} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=false +lock (Replicated): "k2"/Exclusive -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=1} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=false +lock (Replicated): "k2"/Shared -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=1} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=false +lock (Replicated): "k3"/Exclusive -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=1} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=false +error: (*kvpb.LockConflictError:) conflicting locks on "k1" + +# Intents are treated similarly to Exclusive locks. + +run ok +put t=A k=k4 v=v4 +---- +>> at end: +meta: "k4"/0,0 -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=1} ts=10.000000000,0 del=false klen=12 vlen=7 mergeTs= txnDidNotUpdateMeta=true +data: "k4"/10.000000000,0 -> /BYTES/v4 + +run ok +with t=A + check_for_acquire_lock k=k4 str=shared + check_for_acquire_lock k=k4 str=exclusive + acquire_lock k=k4 str=shared + acquire_lock k=k4 str=exclusive +---- +>> at end: +lock (Replicated): "k1"/Shared -> txn={id=00000002 key=/Min iso=Serializable pri=0.00000000 epo=0 ts=11.000000000,0 min=0,0 seq=0} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=true +lock (Replicated): "k1"/Shared -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=1} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=false +lock (Replicated): "k2"/Exclusive -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=1} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=false +lock (Replicated): "k2"/Shared -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=1} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=false +lock (Replicated): "k3"/Exclusive -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=1} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=false +lock (Replicated): "k4"/Intent -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=1} ts=10.000000000,0 del=false klen=12 vlen=7 mergeTs= txnDidNotUpdateMeta=true + +run error +check_for_acquire_lock t=B k=k4 str=shared +---- +error: (*kvpb.LockConflictError:) conflicting locks on "k4" + +run error +acquire_lock t=B k=k4 str=shared +---- +>> at end: +lock (Replicated): "k1"/Shared -> txn={id=00000002 key=/Min iso=Serializable pri=0.00000000 epo=0 ts=11.000000000,0 min=0,0 seq=0} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=true +lock (Replicated): "k1"/Shared -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=1} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=false +lock (Replicated): "k2"/Exclusive -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=1} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=false +lock (Replicated): "k2"/Shared -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=1} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=false +lock (Replicated): "k3"/Exclusive -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=1} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=false +lock (Replicated): "k4"/Intent -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=1} ts=10.000000000,0 del=false klen=12 vlen=7 mergeTs= txnDidNotUpdateMeta=true +error: (*kvpb.LockConflictError:) conflicting locks on "k4" + +# The intent history is considered when determining whether a reacquisition is +# needed on the same key as a previous intent write. + +run ok +with t=A + txn_step + put k=k4 v=v4_prime +---- +>> at end: +txn: "A" meta={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=2} lock=true stat=PENDING rts=10.000000000,0 wto=false gul=0,0 isn=1 +meta: "k4"/0,0 -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=2} ts=10.000000000,0 del=false klen=12 vlen=13 ih={{1 /BYTES/v4}} mergeTs= txnDidNotUpdateMeta=false +data: "k4"/10.000000000,0 -> /BYTES/v4_prime + +run ok +with t=A + txn_step + acquire_lock k=k4 str=shared +---- +>> at end: +txn: "A" meta={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=3} lock=true stat=PENDING rts=10.000000000,0 wto=false gul=0,0 isn=1 +lock (Replicated): "k1"/Shared -> txn={id=00000002 key=/Min iso=Serializable pri=0.00000000 epo=0 ts=11.000000000,0 min=0,0 seq=0} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=true +lock (Replicated): "k1"/Shared -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=1} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=false +lock (Replicated): "k2"/Exclusive -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=1} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=false +lock (Replicated): "k2"/Shared -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=1} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=false +lock (Replicated): "k3"/Exclusive -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=1} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=false +lock (Replicated): "k4"/Intent -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=2} ts=10.000000000,0 del=false klen=12 vlen=13 ih={{1 /BYTES/v4}} mergeTs= txnDidNotUpdateMeta=false + +run +with t=A + txn_ignore_seqs seqs=2-2 + acquire_lock k=k4 str=shared +---- +>> at end: +txn: "A" meta={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=3} lock=true stat=PENDING rts=10.000000000,0 wto=false gul=0,0 isn=1 +lock (Replicated): "k1"/Shared -> txn={id=00000002 key=/Min iso=Serializable pri=0.00000000 epo=0 ts=11.000000000,0 min=0,0 seq=0} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=true +lock (Replicated): "k1"/Shared -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=1} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=false +lock (Replicated): "k2"/Exclusive -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=1} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=false +lock (Replicated): "k2"/Shared -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=1} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=false +lock (Replicated): "k3"/Exclusive -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=1} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=false +lock (Replicated): "k4"/Intent -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=2} ts=10.000000000,0 del=false klen=12 vlen=13 ih={{1 /BYTES/v4}} mergeTs= txnDidNotUpdateMeta=false + +run +with t=A + txn_ignore_seqs seqs=1-2 + acquire_lock k=k4 str=shared +---- +>> at end: +txn: "A" meta={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=3} lock=true stat=PENDING rts=10.000000000,0 wto=false gul=0,0 isn=1 +lock (Replicated): "k1"/Shared -> txn={id=00000002 key=/Min iso=Serializable pri=0.00000000 epo=0 ts=11.000000000,0 min=0,0 seq=0} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=true +lock (Replicated): "k1"/Shared -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=1} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=false +lock (Replicated): "k2"/Exclusive -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=1} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=false +lock (Replicated): "k2"/Shared -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=1} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=false +lock (Replicated): "k3"/Exclusive -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=1} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=false +lock (Replicated): "k4"/Intent -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=2} ts=10.000000000,0 del=false klen=12 vlen=13 ih={{1 /BYTES/v4}} mergeTs= txnDidNotUpdateMeta=false +lock (Replicated): "k4"/Shared -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=3} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=true + +# Replicated locks are ignored by non-locking scans by any transaction. Note +# that we terminate scans at key "k4" to ignore the intent that we just wrote, +# which is not ignored by non-locking scans. + +run ok +with k=k1 end=k4 + scan t=A + scan t=B + scan notxn +---- +scan: "k1"-"k4" -> +scan: "k1"-"k4" -> +scan: "k1"-"k4" -> diff --git a/pkg/storage/testdata/mvcc_histories/replicated_locks_errors b/pkg/storage/testdata/mvcc_histories/replicated_locks_errors new file mode 100644 index 000000000000..a6122bacdd5f --- /dev/null +++ b/pkg/storage/testdata/mvcc_histories/replicated_locks_errors @@ -0,0 +1,81 @@ +# Test invalid lock acquisition inputs. + +run error +check_for_acquire_lock notxn k=k1 str=shared +---- +error: (*withstack.withStack:) txn must be non-nil to acquire lock + +run error +acquire_lock notxn k=k1 str=shared +---- +>> at end: +error: (*withstack.withStack:) txn must be non-nil to acquire lock + +run ok +txn_begin t=A ts=10,0 +---- +>> at end: +txn: "A" meta={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=0 ts=10.000000000,0 min=0,0 seq=0} lock=true stat=PENDING rts=10.000000000,0 wto=false gul=0,0 + +run error +check_for_acquire_lock t=A k=k1 str=none +---- +error: (*withstack.withStack:) invalid lock strength to acquire lock: None + +run error +check_for_acquire_lock t=A k=k1 str=intent +---- +error: (*withstack.withStack:) invalid lock strength to acquire lock: Intent + +run error +acquire_lock t=A k=k1 str=none +---- +>> at end: +error: (*withstack.withStack:) invalid lock strength to acquire lock: None + +run error +acquire_lock t=A k=k1 str=intent +---- +>> at end: +error: (*withstack.withStack:) invalid lock strength to acquire lock: Intent + + +# Test stale lock acquisition. + +run ok +with t=A + txn_step seq=10 + acquire_lock t=A k=k1 str=shared +---- +>> at end: +txn: "A" meta={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=0 ts=10.000000000,0 min=0,0 seq=10} lock=true stat=PENDING rts=10.000000000,0 wto=false gul=0,0 +lock (Replicated): "k1"/Shared -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=0 ts=10.000000000,0 min=0,0 seq=10} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=true + +run error +with t=A + txn_step seq=5 + acquire_lock t=A k=k1 str=shared +---- +>> at end: +txn: "A" meta={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=0 ts=10.000000000,0 min=0,0 seq=5} lock=true stat=PENDING rts=10.000000000,0 wto=false gul=0,0 +lock (Replicated): "k1"/Shared -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=0 ts=10.000000000,0 min=0,0 seq=10} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=true +error: (*withstack.withStack:) cannot acquire lock with strength Shared at seq number 5, already held at higher seq number 10 + +run ok +with t=A + txn_restart epoch=2 + acquire_lock t=A k=k1 str=shared +---- +>> at end: +txn: "A" meta={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=2 ts=10.000000000,0 min=0,0 seq=0} lock=true stat=PENDING rts=10.000000000,0 wto=false gul=0,0 +lock (Replicated): "k1"/Shared -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=2 ts=10.000000000,0 min=0,0 seq=0} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=false + +run error +with t=A + txn_restart epoch=1 + acquire_lock t=A k=k1 str=shared +---- +>> at end: +txn: "A" meta={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=1 ts=10.000000000,0 min=0,0 seq=0} lock=true stat=PENDING rts=10.000000000,0 wto=false gul=0,0 +lock (Replicated): "k1"/Shared -> txn={id=00000001 key=/Min iso=Serializable pri=0.00000000 epo=2 ts=10.000000000,0 min=0,0 seq=0} ts=0,0 del=false klen=0 vlen=0 mergeTs= txnDidNotUpdateMeta=false +error: (*withstack.withStack:) locking request with epoch 1 came after lock had already been acquired at epoch 2 in txn 00000001-0000-0000-0000-000000000000 diff --git a/pkg/storage/testdata/mvcc_histories/skip_locked b/pkg/storage/testdata/mvcc_histories/skip_locked index 23b05eb3ade6..60ddf7b1af7a 100644 --- a/pkg/storage/testdata/mvcc_histories/skip_locked +++ b/pkg/storage/testdata/mvcc_histories/skip_locked @@ -28,7 +28,7 @@ put k=k2 v=v3 ts=13,0 t=B put k=k3 v=v4 ts=14,0 t=C put k=k4 v=v5 ts=15,0 put k=k5 v=v6 ts=17,0 -add_lock k=k4 t=E +add_unreplicated_lock k=k4 t=E ---- >> at end: data: "k1"/11.000000000,0 -> /BYTES/v1 @@ -39,7 +39,9 @@ meta: "k3"/0,0 -> txn={id=00000003 key=/Min iso=Serializable pri=0.00000000 epo= data: "k3"/14.000000000,0 -> /BYTES/v4 data: "k4"/15.000000000,0 -> /BYTES/v5 data: "k5"/17.000000000,0 -> /BYTES/v6 -lock-table: {k4:00000005-0000-0000-0000-000000000000} +lock (Replicated): "k2"/Intent -> txn={id=00000002 key=/Min iso=Serializable pri=0.00000000 epo=0 ts=13.000000000,0 min=0,0 seq=0} ts=13.000000000,0 del=false klen=12 vlen=7 mergeTs= txnDidNotUpdateMeta=true +lock (Replicated): "k3"/Intent -> txn={id=00000003 key=/Min iso=Serializable pri=0.00000000 epo=0 ts=14.000000000,0 min=0,0 seq=0} ts=14.000000000,0 del=false klen=12 vlen=7 mergeTs= txnDidNotUpdateMeta=true +lock (Unreplicated): k4/Exclusive -> id=00000005 key=/Min iso=Serializable pri=0.00000000 epo=0 ts=16.000000000,0 min=0,0 seq=0 # Test cases: #