-
Notifications
You must be signed in to change notification settings - Fork 3.6k
New issue
Have a question about this project? Sign up for a free GitHub account to open an issue and contact its maintainers and the community.
By clicking “Sign up for GitHub”, you agree to our terms of service and privacy statement. We’ll occasionally send you account related emails.
Already on GitHub? Sign in to your account
[improve][ml] Support Bookkeeper batch read #23180
base: master
Are you sure you want to change the base?
Changes from 8 commits
53bf4a2
55f052f
9c09343
3dcb6b9
1c0073d
c58109a
38479f3
6eea5b1
8c4eebf
eaf48a3
0c5d785
b0a47f3
File filter
Filter by extension
Conversations
Jump to
Diff view
Diff view
There are no files selected for viewing
Original file line number | Diff line number | Diff line change | ||||||||||||||||||||||||||||||
---|---|---|---|---|---|---|---|---|---|---|---|---|---|---|---|---|---|---|---|---|---|---|---|---|---|---|---|---|---|---|---|---|
|
@@ -18,21 +18,32 @@ | |||||||||||||||||||||||||||||||
*/ | ||||||||||||||||||||||||||||||||
package org.apache.bookkeeper.mledger.impl.cache; | ||||||||||||||||||||||||||||||||
|
||||||||||||||||||||||||||||||||
import com.google.common.annotations.VisibleForTesting; | ||||||||||||||||||||||||||||||||
import java.util.concurrent.CompletableFuture; | ||||||||||||||||||||||||||||||||
import org.apache.bookkeeper.client.api.LedgerEntries; | ||||||||||||||||||||||||||||||||
import org.apache.bookkeeper.client.api.LedgerMetadata; | ||||||||||||||||||||||||||||||||
import org.apache.bookkeeper.client.api.ReadHandle; | ||||||||||||||||||||||||||||||||
import org.apache.bookkeeper.mledger.ManagedLedger; | ||||||||||||||||||||||||||||||||
import org.apache.bookkeeper.mledger.ManagedLedgerException; | ||||||||||||||||||||||||||||||||
|
||||||||||||||||||||||||||||||||
class ReadEntryUtils { | ||||||||||||||||||||||||||||||||
@VisibleForTesting | ||||||||||||||||||||||||||||||||
public class ReadEntryUtils { | ||||||||||||||||||||||||||||||||
|
||||||||||||||||||||||||||||||||
static CompletableFuture<LedgerEntries> readAsync(ManagedLedger ml, ReadHandle handle, long firstEntry, | ||||||||||||||||||||||||||||||||
long lastEntry) { | ||||||||||||||||||||||||||||||||
@VisibleForTesting | ||||||||||||||||||||||||||||||||
public static CompletableFuture<LedgerEntries> readAsync(ManagedLedger ml, ReadHandle handle, long firstEntry, | ||||||||||||||||||||||||||||||||
long lastEntry, boolean useBookkeeperV2WireProtocol) { | ||||||||||||||||||||||||||||||||
if (ml.getOptionalLedgerInfo(handle.getId()).isEmpty()) { | ||||||||||||||||||||||||||||||||
// The read handle comes from another managed ledger, in this case, we can only compare the entry range with | ||||||||||||||||||||||||||||||||
// the LAC of that read handle. Specifically, it happens when this method is called by a | ||||||||||||||||||||||||||||||||
// ReadOnlyManagedLedgerImpl object. | ||||||||||||||||||||||||||||||||
return handle.readAsync(firstEntry, lastEntry); | ||||||||||||||||||||||||||||||||
int entriesToRead = (int) (lastEntry - firstEntry + 1); | ||||||||||||||||||||||||||||||||
// Batch read is not supported for striped ledgers. | ||||||||||||||||||||||||||||||||
LedgerMetadata m = handle.getLedgerMetadata(); | ||||||||||||||||||||||||||||||||
boolean isStriped = m.getEnsembleSize() != m.getWriteQuorumSize(); | ||||||||||||||||||||||||||||||||
if (!useBatchRead(entriesToRead, useBookkeeperV2WireProtocol, isStriped)) { | ||||||||||||||||||||||||||||||||
return handle.readAsync(firstEntry, lastEntry); | ||||||||||||||||||||||||||||||||
} | ||||||||||||||||||||||||||||||||
return handle.batchReadAsync(firstEntry, entriesToRead, 0); | ||||||||||||||||||||||||||||||||
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. If maxSize = 0, info log will be printed for each call, which will affect performance.
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. If isAutoSkipNonRecoverableData is set to true, is the batch read failure consistent with the behavior here? pulsar/managed-ledger/src/main/java/org/apache/bookkeeper/mledger/impl/OpReadEntry.java Lines 115 to 129 in 66e1a06
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more.
apache/bookkeeper#4485 I created a PR to change the log level to debug |
||||||||||||||||||||||||||||||||
} | ||||||||||||||||||||||||||||||||
// Compare the entry range with the lastConfirmedEntry maintained by the managed ledger because the entry cache | ||||||||||||||||||||||||||||||||
// of `ShadowManagedLedgerImpl` reads entries via `ReadOnlyLedgerHandle`, which never updates `lastAddConfirmed` | ||||||||||||||||||||||||||||||||
|
@@ -51,4 +62,8 @@ static CompletableFuture<LedgerEntries> readAsync(ManagedLedger ml, ReadHandle h | |||||||||||||||||||||||||||||||
} | ||||||||||||||||||||||||||||||||
return handle.readUnconfirmedAsync(firstEntry, lastEntry); | ||||||||||||||||||||||||||||||||
} | ||||||||||||||||||||||||||||||||
|
||||||||||||||||||||||||||||||||
private static boolean useBatchRead(int entriesToRead, boolean useBookkeeperV2WireProtocol, boolean isStriped) { | ||||||||||||||||||||||||||||||||
return entriesToRead > 1 && useBookkeeperV2WireProtocol && !isStriped; | ||||||||||||||||||||||||||||||||
} | ||||||||||||||||||||||||||||||||
} |
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
It needs to be made clear in the document that enabling batch does not necessarily mean batch reading.