Skip to content

Commit 8a47295

Browse files
gaoran10lhotari
authored andcommitted
[fix][broker] Catch exception for entry payload interceptor processor (#23683)
(cherry picked from commit 24c337f)
1 parent f16cf67 commit 8a47295

File tree

2 files changed

+58
-2
lines changed

2 files changed

+58
-2
lines changed

managed-ledger/src/main/java/org/apache/bookkeeper/mledger/impl/OpAddEntry.java

Lines changed: 9 additions & 2 deletions
Original file line numberDiff line numberDiff line change
@@ -138,8 +138,15 @@ public void initiate() {
138138
lastInitTime = System.nanoTime();
139139
if (ml.getManagedLedgerInterceptor() != null) {
140140
long originalDataLen = data.readableBytes();
141-
payloadProcessorHandle = ml.getManagedLedgerInterceptor().processPayloadBeforeLedgerWrite(this,
142-
duplicateBuffer);
141+
try {
142+
payloadProcessorHandle = ml.getManagedLedgerInterceptor()
143+
.processPayloadBeforeLedgerWrite(this, duplicateBuffer);
144+
} catch (Exception e) {
145+
ReferenceCountUtil.safeRelease(duplicateBuffer);
146+
log.error("[{}] Error processing payload before ledger write", ml.getName(), e);
147+
this.failed(new ManagedLedgerException.ManagedLedgerInterceptException(e));
148+
return;
149+
}
143150
if (payloadProcessorHandle != null) {
144151
duplicateBuffer = payloadProcessorHandle.getProcessedPayload();
145152
// If data len of entry changes, correct "dataLength" and "currentLedgerSize".

pulsar-broker/src/test/java/org/apache/pulsar/broker/intercept/MangedLedgerInterceptorImplTest.java

Lines changed: 49 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -18,8 +18,12 @@
1818
*/
1919
package org.apache.pulsar.broker.intercept;
2020

21+
import static org.testng.Assert.fail;
22+
2123
import io.netty.buffer.ByteBuf;
2224
import io.netty.buffer.Unpooled;
25+
import java.util.ArrayList;
26+
import java.util.Collections;
2327
import java.util.concurrent.CompletableFuture;
2428
import java.util.concurrent.CountDownLatch;
2529
import java.util.function.Predicate;
@@ -436,4 +440,49 @@ public boolean test(@Nullable Entry entry) {
436440
}
437441
}
438442

443+
@Test(timeOut = 3000)
444+
public void testManagedLedgerPayloadInputProcessorFailure() throws Exception {
445+
var config = new ManagedLedgerConfig();
446+
final String failureMsg = "failed to process input payload";
447+
config.setManagedLedgerInterceptor(new ManagedLedgerInterceptorImpl(
448+
Collections.emptySet(), Set.of(new ManagedLedgerPayloadProcessor() {
449+
@Override
450+
public Processor inputProcessor() {
451+
return new Processor() {
452+
@Override
453+
public ByteBuf process(Object contextObj, ByteBuf inputPayload) {
454+
throw new RuntimeException(failureMsg);
455+
}
456+
457+
@Override
458+
public void release(ByteBuf processedPayload) {
459+
// no-op
460+
fail("the release method can't be reached");
461+
}
462+
};
463+
}
464+
})));
465+
466+
var ledger = factory.open("testManagedLedgerPayloadProcessorFailure", config);
467+
var countDownLatch = new CountDownLatch(1);
468+
var expectedException = new ArrayList<Exception>();
469+
ledger.asyncAddEntry("test".getBytes(), 1, 1, new AsyncCallbacks.AddEntryCallback() {
470+
@Override
471+
public void addComplete(Position position, ByteBuf entryData, Object ctx) {
472+
entryData.release();
473+
countDownLatch.countDown();
474+
}
475+
476+
@Override
477+
public void addFailed(ManagedLedgerException exception, Object ctx) {
478+
// expected
479+
expectedException.add(exception);
480+
countDownLatch.countDown();
481+
}
482+
}, null);
483+
countDownLatch.await();
484+
assertEquals(expectedException.size(), 1);
485+
assertEquals(expectedException.get(0).getCause().getMessage(), failureMsg);
486+
}
487+
439488
}

0 commit comments

Comments
 (0)