Skip to content
This repository has been archived by the owner on Apr 30, 2019. It is now read-only.

Commit

Permalink
Fixed simultaneus reads on same ledger/entry with v2 protocol
Browse files Browse the repository at this point in the history
  • Loading branch information
merlimat committed Jun 22, 2016
1 parent b2cc158 commit ce7b0f7
Show file tree
Hide file tree
Showing 2 changed files with 112 additions and 15 deletions.
Original file line number Diff line number Diff line change
Expand Up @@ -55,6 +55,8 @@
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;

import com.google.common.collect.LinkedListMultimap;
import com.google.common.collect.ListMultimap;
import com.google.protobuf.ByteString;
import com.google.protobuf.ExtensionRegistry;

Expand Down Expand Up @@ -108,6 +110,11 @@ public class PerChannelBookieClient extends ChannelInboundHandlerAdapter {

private final ConcurrentHashMap<CompletionKey, CompletionValue> completionObjects = new ConcurrentHashMap<CompletionKey, CompletionValue>();

// Map that hold duplicated read requests. The idea is to only use this map (synchronized) when there is a duplicate
// read request for the same ledgerId/entryId
private final ListMultimap<CompletionKey, CompletionValue> completionObjectsV2Conflicts = LinkedListMultimap
.create();

private final StatsLogger statsLogger;
private final OpStatsLogger readEntryOpLogger;
private final OpStatsLogger readTimeoutOpLogger;
Expand Down Expand Up @@ -442,12 +449,14 @@ public void readEntryAndFenceLedger(final long ledgerId, byte[] masterKey,
}

final CompletionKey completionKey = completion;
if (completionObjects.putIfAbsent(completionKey,
new ReadCompletion(readEntryOpLogger, cb, ctx, ledgerId, entryId,
scheduleTimeout(completionKey, readEntryTimeout))) != null) {
// We cannot have more than 1 pending read on the same ledger/entry in the v2 protocol
cb.readEntryComplete(BKException.Code.BookieHandleNotAvailableException, ledgerId, entryId, null, ctx);
return;
ReadCompletion readCompletion = new ReadCompletion(readEntryOpLogger, cb,
ctx, ledgerId, entryId, scheduleTimeout(completion, readEntryTimeout));
CompletionValue existingValue = completionObjects.putIfAbsent(completion, readCompletion);
if (existingValue != null) {
// There's a pending read request on same ledger/entry. Use the multimap to track all of them
synchronized (this) {
completionObjectsV2Conflicts.put(completionKey, readCompletion);
}
}

final Channel c = channel;
Expand Down Expand Up @@ -511,13 +520,14 @@ public void readEntry(final long ledgerId, final long entryId, ReadEntryCallback

final Object readRequest = request;
final CompletionKey completionKey = completion;
CompletionValue existingValue = completionObjects.putIfAbsent(completion, new ReadCompletion(readEntryOpLogger, cb,
ctx, ledgerId, entryId, scheduleTimeout(completion, readEntryTimeout)));
ReadCompletion readCompletion = new ReadCompletion(readEntryOpLogger, cb,
ctx, ledgerId, entryId, scheduleTimeout(completion, readEntryTimeout));
CompletionValue existingValue = completionObjects.putIfAbsent(completion, readCompletion);
if (existingValue != null) {
// There's a pending read request on same ledger/entry. This is not supported in V2 protocol
LOG.warn("Failing concurrent request to read at ledger: {} entry: {}", ledgerId, entryId);
cb.readEntryComplete(BKException.Code.UnexpectedConditionException, ledgerId, entryId, null, ctx);
return;
// There's a pending read request on same ledger/entry. Use the multimap to track all of them
synchronized (this) {
completionObjectsV2Conflicts.put(completionKey, readCompletion);
}
}

final Channel c = channel;
Expand Down Expand Up @@ -616,7 +626,17 @@ void errorOutReadKey(final CompletionKey key) {
void errorOutReadKey(final CompletionKey key, final int rc) {
LOG.debug("Removing completion key: {}", key);
ReadCompletion completion = (ReadCompletion) completionObjects.remove(key);
if (completion == null) {
// If there's no completion object here, try in the multimap
synchronized (this) {
if (completionObjectsV2Conflicts.containsKey(key)) {
completion = (ReadCompletion) completionObjectsV2Conflicts.get(key).get(0);
completionObjectsV2Conflicts.remove(key, completion);
}
}
}

// If it's still null, give up
if (null == completion) {
return;
}
Expand Down Expand Up @@ -793,7 +813,17 @@ private void readV2Response(final BookieProtocol.Response response) {
final StatusCode status = getStatusCodeFromErrorCode(response.errorCode);

V2CompletionKey key = V2CompletionKey.get(this, ledgerId, entryId, operationType);
final CompletionValue completionValue = completionObjects.remove(key);
CompletionValue completionValue = completionObjects.remove(key);
if (completionValue == null) {
// If there's no completion object here, try in the multimap
synchronized (this) {
if (completionObjectsV2Conflicts.containsKey(key)) {
completionValue = completionObjectsV2Conflicts.get(key).get(0);
completionObjectsV2Conflicts.remove(key, completionValue);
}
}
}

key.recycle();

if (null == completionValue) {
Expand All @@ -804,19 +834,20 @@ private void readV2Response(final BookieProtocol.Response response) {
}
} else {
long orderingKey = completionValue.ledgerId;
final CompletionValue finalCompletionValue = completionValue;

executor.submitOrdered(orderingKey, new SafeRunnable() {
@Override
public void safeRun() {
switch (operationType) {
case ADD_ENTRY: {
handleAddResponse(status, ledgerId, entryId, completionValue);
handleAddResponse(status, ledgerId, entryId, finalCompletionValue);
response.recycle();
break;
}
case READ_ENTRY: {
BookieProtocol.ReadResponse readResponse = (BookieProtocol.ReadResponse) response;
handleReadResponse(status, readResponse.getLedgerId(), readResponse.getEntryId(), readResponse.data, completionValue);
handleReadResponse(status, readResponse.getLedgerId(), readResponse.getEntryId(), readResponse.data, finalCompletionValue);
readResponse.recycle();
break;
}
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -21,13 +21,15 @@
*
*/

import java.util.Enumeration;
import java.util.concurrent.CountDownLatch;
import java.util.concurrent.TimeUnit;
import java.util.concurrent.atomic.AtomicInteger;
import java.util.concurrent.atomic.AtomicBoolean;

import org.apache.bookkeeper.conf.ClientConfiguration;
import org.apache.bookkeeper.client.AsyncCallback.AddCallback;
import org.apache.bookkeeper.client.AsyncCallback.ReadCallback;
import org.apache.bookkeeper.client.BookKeeper.DigestType;
import org.apache.bookkeeper.test.BaseTestCase;
import org.apache.zookeeper.ZooKeeper;
Expand Down Expand Up @@ -240,4 +242,68 @@ public void testIsClosed() throws Exception {

bkc.close();
}

/**
* Tests that issuing multiple reads for the same entry at the same time works as expected.
*
* @throws Exception
*/
@Test(timeout = 20000)
public void testDoubleRead() throws Exception {
LedgerHandle lh = bkc.createLedger(digestType, "".getBytes());

lh.addEntry("test".getBytes());

// Read the same entry more times asynchronously
final int N = 10;
final CountDownLatch latch = new CountDownLatch(N);
for (int i = 0; i < N; i++) {
lh.asyncReadEntries(0, 0, new ReadCallback() {
public void readComplete(int rc, LedgerHandle lh,
Enumeration<LedgerEntry> seq, Object ctx) {
if (rc == BKException.Code.OK) {
latch.countDown();
} else {
fail("Read fail");
}
}
}, null);
}

latch.await();
}

/**
* Tests that issuing multiple reads for the same entry at the same time works as expected.
*
* @throws Exception
*/
@Test(timeout = 20000)
public void testDoubleReadWithV2Protocol() throws Exception {
ClientConfiguration conf = new ClientConfiguration(baseClientConf);
conf.setUseV2WireProtocol(true);
BookKeeperTestClient bkc = new BookKeeperTestClient(conf);
LedgerHandle lh = bkc.createLedger(digestType, "".getBytes());

lh.addEntry("test".getBytes());

// Read the same entry more times asynchronously
final int N = 10;
final CountDownLatch latch = new CountDownLatch(N);
for (int i = 0; i < N; i++) {
lh.asyncReadEntries(0, 0, new ReadCallback() {
public void readComplete(int rc, LedgerHandle lh,
Enumeration<LedgerEntry> seq, Object ctx) {
if (rc == BKException.Code.OK) {
latch.countDown();
} else {
fail("Read fail");
}
}
}, null);
}

latch.await();
bkc.close();
}
}

0 comments on commit ce7b0f7

Please sign in to comment.