poorbarcode commented on code in PR #16758:
URL: https://github.com/apache/pulsar/pull/16758#discussion_r958340396
##########
pulsar-transaction/coordinator/src/main/java/org/apache/pulsar/transaction/coordinator/impl/TxnLogBufferedWriter.java:
##########
@@ -374,46 +373,42 @@ private void doFlush(){
/**
* Release resources and cancel pending tasks.
*/
- @Override
- public void close() {
+ public CompletableFuture<Void> close() {
// If batch feature is disabled, there is nothing to close, so set the
stat only.
if (!batchEnabled) {
STATE_UPDATER.compareAndSet(this, State.OPEN, State.CLOSED);
- return;
+ return CompletableFuture.completedFuture(null);
}
// If other thread already called "close()", so do nothing.
if (!STATE_UPDATER.compareAndSet(this, State.OPEN, State.CLOSING)){
- return;
+ return CompletableFuture.completedFuture(null);
}
+ CompletableFuture closeFuture = new CompletableFuture();
// Cancel pending tasks and release resources.
singleThreadExecutorForWrite.execute(() -> {
try {
if (state == State.CLOSED) {
+ closeFuture.complete(null);
return;
}
// If some requests are flushed, BK will trigger these
callbacks, and the remaining requests in should
// fail.
- failureCallbackByContextAndRecycle(flushContext,
BUFFERED_WRITER_CLOSED_EXCEPTION);
+ failureCallbackByContextAndRecycle(flushContext,
+ new
ManagedLedgerException.ManagedLedgerFencedException(
+ new Exception("Transaction log buffered write has
closed")
+ ));
// Cancel the timing task.
- if (timeout == null) {
- log.error("Cancel timeout-task that schedule at fixed rate
trig flush failure. The field-timeout"
- + " is null. managedLedger: " +
managedLedger.getName());
- } else if (timeout.isCancelled()) {
- // TODO How decisions the timer-task has been finished ?
- STATE_UPDATER.set(this, State.CLOSED);
- } else {
- if (this.timeout.cancel()) {
- STATE_UPDATER.set(this, State.CLOSED);
- } else {
- // Cancel task failure, The state will stay at CLOSING.
- log.error("Cancel timeout-task that schedule at fixed
rate trig flush failure. The state will"
- + " stay at CLOSING. managedLedger: " +
managedLedger.getName());
- }
+ if (!timeout.isCancelled()){
+ this.timeout.cancel();
}
+ STATE_UPDATER.set(this, State.CLOSED);
+ closeFuture.complete(null);
} catch (Exception e){
log.error("Close Txn log buffered writer fail", e);
+ closeFuture.completeExceptionally(e);
Review Comment:
Already remove this log line
##########
pulsar-transaction/coordinator/src/main/java/org/apache/pulsar/transaction/coordinator/impl/TxnLogBufferedWriter.java:
##########
@@ -374,46 +373,42 @@ private void doFlush(){
/**
* Release resources and cancel pending tasks.
*/
- @Override
- public void close() {
+ public CompletableFuture<Void> close() {
// If batch feature is disabled, there is nothing to close, so set the
stat only.
if (!batchEnabled) {
STATE_UPDATER.compareAndSet(this, State.OPEN, State.CLOSED);
- return;
+ return CompletableFuture.completedFuture(null);
}
// If other thread already called "close()", so do nothing.
if (!STATE_UPDATER.compareAndSet(this, State.OPEN, State.CLOSING)){
- return;
+ return CompletableFuture.completedFuture(null);
}
+ CompletableFuture closeFuture = new CompletableFuture();
// Cancel pending tasks and release resources.
singleThreadExecutorForWrite.execute(() -> {
try {
if (state == State.CLOSED) {
+ closeFuture.complete(null);
Review Comment:
You are right, already delete this check
--
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
To unsubscribe, e-mail: [email protected]
For queries about this service, please contact Infrastructure at:
[email protected]