Skip to content

Commit 4819df3

Browse files
craig[bot]stevendanna
andcommitted
Merge #147065
147065: concurrency: update wait queues in AddDiscoveredLock r=miraradeva a=stevendanna Concurrent readers in the face of a concurrent, retrying writer can result in a situation where AddDiscoveredLock moves the timestamp of a held intent past the read timestamp a waiting reader. The reader should be unblocked in this case but previously wasn't, resulting in a lock table verification assertion failure in the form of: error: non locking reader ... does not conflict with lock holder This is my best theory for what is happening in #146749 based on increased logging. It is difficult to be certain given the required ordering of events is hard to observe with locking. See the comment in the test for a more complete timeline that can lead to the bug. Fixes #146749 Release note: None Co-authored-by: Steven Danna <[email protected]>
2 parents d358a24 + 5b8fdaf commit 4819df3

File tree

3 files changed

+171
-2
lines changed

3 files changed

+171
-2
lines changed

pkg/kv/kvserver/concurrency/lock_table.go

Lines changed: 9 additions & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -3132,6 +3132,7 @@ func (kl *keyLocks) discoveredLock(
31323132
accessStrength lock.Strength,
31333133
notRemovable bool,
31343134
clock *hlc.Clock,
3135+
st *cluster.Settings,
31353136
) error {
31363137
kl.mu.Lock()
31373138
defer kl.mu.Unlock()
@@ -3144,7 +3145,14 @@ func (kl *keyLocks) discoveredLock(
31443145
if kl.isLockedBy(foundLock.Txn.ID) {
31453146
e := kl.heldBy[foundLock.Txn.ID]
31463147
tl = e.Value
3148+
3149+
beforeTs := tl.writeTS()
3150+
31473151
tl.replicatedInfo.acquire(foundLock.Strength, foundLock.Txn.WriteTimestamp)
3152+
3153+
if beforeTs.Less(tl.writeTS()) {
3154+
kl.recomputeWaitQueues(st)
3155+
}
31483156
// TODO(arul): If the discovered lock indicates a newer epoch than what's
31493157
// being tracked, should we clear out unreplicatedLockInfo here?
31503158
} else {
@@ -4352,7 +4360,7 @@ func (t *lockTableImpl) AddDiscoveredLock(
43524360
g.notRemovableLock = l
43534361
notRemovableLock = true
43544362
}
4355-
err = l.discoveredLock(foundLock, g, str, notRemovableLock, g.lt.clock)
4363+
err = l.discoveredLock(foundLock, g, str, notRemovableLock, g.lt.clock, g.lt.settings)
43564364
// Can't release tree.mu until call l.discoveredLock() since someone may
43574365
// find an empty lock and remove it from the tree.
43584366
t.locks.mu.Unlock()

pkg/kv/kvserver/concurrency/lock_table_test.go

Lines changed: 8 additions & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -312,6 +312,7 @@ func TestLockTableBasic(t *testing.T) {
312312
if d.HasArg("skip-locked") {
313313
waitPolicy = lock.WaitPolicy_SkipLocked
314314
}
315+
updateRetainedTxn := !d.HasArg("no-update-retained-txn")
315316
var maxLockWaitQueueLength int
316317
if d.HasArg("max-lock-wait-queue-length") {
317318
d.ScanArgs(t, "max-lock-wait-queue-length", &maxLockWaitQueueLength)
@@ -331,11 +332,17 @@ func TestLockTableBasic(t *testing.T) {
331332
if txnMeta != nil {
332333
// Update the transaction's timestamp, if necessary. The transaction
333334
// may have needed to move its timestamp for any number of reasons.
334-
txnMeta.WriteTimestamp = ts
335+
if updateRetainedTxn {
336+
txnMeta.WriteTimestamp = ts
337+
}
335338
ba.Txn = &roachpb.Transaction{
336339
TxnMeta: *txnMeta,
337340
ReadTimestamp: ts,
338341
}
342+
if !updateRetainedTxn {
343+
ba.Txn.WriteTimestamp = ts
344+
}
345+
339346
req.Txn = ba.Txn
340347
}
341348
requestsByName[reqName] = req

pkg/kv/kvserver/concurrency/testdata/lock_table/add_discovered

Lines changed: 154 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -141,3 +141,157 @@ new: state=doneWaiting
141141
dequeue r=req2
142142
----
143143
num=0
144+
145+
# -------------------------------------------------------------
146+
# This is a regression test for #146749
147+
#
148+
# It attempts to reproduce the following series of events on a single key.
149+
#
150+
# s | w1@ts10 | w2@ts12 | r1@ts10 |r2@ts12 | r3@ts11
151+
# --+--------------------------+--------------------------+----------------------------+-----------------------------+-----------------------------
152+
# 1 | Seq | | | |
153+
# 2 | AcquireLock(intent@ts10) | | | |
154+
# 3 | FinishReq | | | |
155+
# 4 | | SeqReq | SeqReq | |
156+
# 5 | | AcquireLock(intent@ts12) | | |
157+
# 6 | | FinishReq | | |
158+
# 8 | | | | SeqReq |
159+
# 9 | | | AddDiscovered(intent@ts10) | |
160+
# 10| | | FinishReq | |
161+
# 11| | | | | SeqReq(waits on intent@10)
162+
# 12| | | | AddDiscovered(intent@ts12) |
163+
# 13| | | | FinishReq
164+
#
165+
# Note that w1 and w2 are from the same txn.
166+
#
167+
# The invariant that the test cares about is that r3 that starts waiting at step 11 is unblocked by the lock table
168+
# update at step 12.
169+
#
170+
# Note that in the test as written, the followin are how the reqests align
171+
# to the above diagram:
172+
#
173+
# w1 = req1
174+
# w2 = req3
175+
# r1 = req2
176+
# r2 = req4
177+
# r3 = req5
178+
#
179+
# -------------------------------------------------------------
180+
181+
new-lock-table maxlocks=10000
182+
----
183+
184+
# txn1 is used for the writer (req1 and req3)
185+
new-txn txn=txn1 ts=10 epoch=0
186+
----
187+
188+
# txn2-4 are our readers.
189+
new-txn txn=txn2 ts=10 epoch=0
190+
----
191+
192+
new-txn txn=txn3 ts=12 epoch=0
193+
----
194+
195+
new-txn txn=txn4 ts=11 epoch=0
196+
----
197+
198+
# w1
199+
new-request r=req1 txn=txn1 ts=10 spans=intent@
200+
----
201+
202+
scan r=req1
203+
----
204+
start-waiting: false
205+
206+
acquire r=req1 k=a durability=r strength=intent
207+
----
208+
num=0
209+
210+
dequeue r=req1
211+
----
212+
num=0
213+
214+
# r1
215+
new-request r=req2 txn=txn2 ts=10 spans=none@a
216+
----
217+
218+
# w2
219+
# Note that we don't allow the retained txn to be updated because that will erroneously impact
220+
# the fist call to add-discovered-lock.
221+
new-request r=req3 txn=txn1 ts=12 spans=intent@a no-update-retained-txn
222+
----
223+
224+
scan r=req3
225+
----
226+
start-waiting: false
227+
228+
scan r=req2
229+
----
230+
start-waiting: false
231+
232+
acquire r=req3 k=a durability=r strength=intent
233+
----
234+
num=0
235+
236+
dequeue r=req3
237+
----
238+
num=0
239+
240+
# r2
241+
new-request r=req4 txn=txn3 ts=12 spans=none@a
242+
----
243+
244+
scan r=req4
245+
----
246+
start-waiting: false
247+
248+
# The first reader now discovers the intent at ts10.
249+
add-discovered r=req2 k=a txn=txn1
250+
----
251+
num=1
252+
lock: "a"
253+
holder: txn: 00000000-0000-0000-0000-000000000001 epoch: 0, iso: Serializable, ts: 10.000000000,0, info: repl [Intent]
254+
255+
dequeue r=req2
256+
----
257+
num=1
258+
lock: "a"
259+
holder: txn: 00000000-0000-0000-0000-000000000001 epoch: 0, iso: Serializable, ts: 10.000000000,0, info: repl [Intent]
260+
261+
# Third reader that waits on intents.
262+
new-request r=req5 txn=txn4 ts=11 spans=none@a
263+
----
264+
265+
scan r=req5
266+
----
267+
start-waiting: true
268+
269+
print
270+
----
271+
num=1
272+
lock: "a"
273+
holder: txn: 00000000-0000-0000-0000-000000000001 epoch: 0, iso: Serializable, ts: 10.000000000,0, info: repl [Intent]
274+
waiting readers:
275+
req: 5, txn: 00000000-0000-0000-0000-000000000004
276+
277+
# Now update txn1's retained record to ts12.
278+
update-txn-not-observed txn=txn1 ts=12 epoch=1
279+
----
280+
281+
add-discovered r=req4 k=a txn=txn1
282+
----
283+
num=1
284+
lock: "a"
285+
holder: txn: 00000000-0000-0000-0000-000000000001 epoch: 0, iso: Serializable, ts: 12.000000000,0, info: repl [Intent]
286+
287+
dequeue r=req4
288+
----
289+
num=1
290+
lock: "a"
291+
holder: txn: 00000000-0000-0000-0000-000000000001 epoch: 0, iso: Serializable, ts: 12.000000000,0, info: repl [Intent]
292+
293+
dequeue r=req5
294+
----
295+
num=1
296+
lock: "a"
297+
holder: txn: 00000000-0000-0000-0000-000000000001 epoch: 0, iso: Serializable, ts: 12.000000000,0, info: repl [Intent]

0 commit comments

Comments
 (0)