Skip to content
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] [broker] Close temporary open ledger in BookkeeperBucketSnapshotStorage #20111

Merged
Changes from all commits
Commits
File filter

Filter by extension

Filter by extension

Conversations
Failed to load comments.
Loading
Jump to
Jump to file
Failed to load files.
Loading
Diff view
Diff view
Original file line number Diff line number Diff line change
Expand Up @@ -66,22 +66,41 @@ public CompletableFuture<Long> createBucketSnapshot(SnapshotMetadata snapshotMet

@Override
public CompletableFuture<SnapshotMetadata> getBucketSnapshotMetadata(long bucketId) {
return openLedger(bucketId).thenCompose(
ledgerHandle -> getLedgerEntry(ledgerHandle, 0, 0).
thenApply(entryEnumeration -> parseSnapshotMetadataEntry(entryEnumeration.nextElement())));
return openLedger(bucketId).thenCompose(ledgerHandle -> {
CompletableFuture<SnapshotMetadata> snapshotFuture =
getLedgerEntry(ledgerHandle, 0, 0)
.thenApply(entryEnumeration -> parseSnapshotMetadataEntry(entryEnumeration.nextElement()));

snapshotFuture.whenComplete((__, e) -> closeLedger(ledgerHandle));

return snapshotFuture;
});
}

@Override
public CompletableFuture<List<SnapshotSegment>> getBucketSnapshotSegment(long bucketId, long firstSegmentEntryId,
long lastSegmentEntryId) {
return openLedger(bucketId).thenCompose(
ledgerHandle -> getLedgerEntry(ledgerHandle, firstSegmentEntryId,
lastSegmentEntryId).thenApply(this::parseSnapshotSegmentEntries));
return openLedger(bucketId).thenCompose(ledgerHandle -> {
CompletableFuture<List<SnapshotSegment>> parseFuture =
getLedgerEntry(ledgerHandle, firstSegmentEntryId, lastSegmentEntryId)
.thenApply(this::parseSnapshotSegmentEntries);

parseFuture.whenComplete((__, e) -> closeLedger(ledgerHandle));

return parseFuture;
});
}

@Override
public CompletableFuture<Long> getBucketSnapshotLength(long bucketId) {
return openLedger(bucketId).thenApply(LedgerHandle::getLength);
return openLedger(bucketId).thenCompose(ledgerHandle -> {
CompletableFuture<Long> lengthFuture =
CompletableFuture.completedFuture(ledgerHandle.getLength());

lengthFuture.whenComplete((__, e) -> closeLedger(ledgerHandle));

return lengthFuture;
});
}

@Override
Expand Down