Skip to content

Commit

Permalink
[fix][broker] Catch exception for entry payload interceptor processor (
Browse files Browse the repository at this point in the history
  • Loading branch information
gaoran10 authored Dec 8, 2024
1 parent fa8aa9e commit 24c337f
Show file tree
Hide file tree
Showing 2 changed files with 58 additions and 2 deletions.
Original file line number Diff line number Diff line change
Expand Up @@ -139,8 +139,15 @@ public void initiate() {
lastInitTime = System.nanoTime();
if (ml.getManagedLedgerInterceptor() != null) {
long originalDataLen = data.readableBytes();
payloadProcessorHandle = ml.getManagedLedgerInterceptor()
.processPayloadBeforeLedgerWrite(this.getCtx(), duplicateBuffer);
try {
payloadProcessorHandle = ml.getManagedLedgerInterceptor()
.processPayloadBeforeLedgerWrite(this.getCtx(), duplicateBuffer);
} catch (Exception e) {
ReferenceCountUtil.safeRelease(duplicateBuffer);
log.error("[{}] Error processing payload before ledger write", ml.getName(), e);
this.failed(new ManagedLedgerException.ManagedLedgerInterceptException(e));
return;
}
if (payloadProcessorHandle != null) {
duplicateBuffer = payloadProcessorHandle.getProcessedPayload();
// If data len of entry changes, correct "dataLength" and "currentLedgerSize".
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -21,9 +21,13 @@
import static org.testng.Assert.assertEquals;
import static org.testng.Assert.assertNotEquals;
import static org.testng.Assert.assertNotNull;
import static org.testng.Assert.fail;

import io.netty.buffer.ByteBuf;
import io.netty.buffer.Unpooled;
import java.nio.charset.StandardCharsets;
import java.util.ArrayList;
import java.util.Collections;
import java.util.HashSet;
import java.util.List;
import java.util.Set;
Expand Down Expand Up @@ -431,4 +435,49 @@ public boolean test(@Nullable Entry entry) {
}
}

@Test(timeOut = 3000)
public void testManagedLedgerPayloadInputProcessorFailure() throws Exception {
var config = new ManagedLedgerConfig();
final String failureMsg = "failed to process input payload";
config.setManagedLedgerInterceptor(new ManagedLedgerInterceptorImpl(
Collections.emptySet(), Set.of(new ManagedLedgerPayloadProcessor() {
@Override
public Processor inputProcessor() {
return new Processor() {
@Override
public ByteBuf process(Object contextObj, ByteBuf inputPayload) {
throw new RuntimeException(failureMsg);
}

@Override
public void release(ByteBuf processedPayload) {
// no-op
fail("the release method can't be reached");
}
};
}
})));

var ledger = factory.open("testManagedLedgerPayloadProcessorFailure", config);
var countDownLatch = new CountDownLatch(1);
var expectedException = new ArrayList<Exception>();
ledger.asyncAddEntry("test".getBytes(), 1, 1, new AsyncCallbacks.AddEntryCallback() {
@Override
public void addComplete(Position position, ByteBuf entryData, Object ctx) {
entryData.release();
countDownLatch.countDown();
}

@Override
public void addFailed(ManagedLedgerException exception, Object ctx) {
// expected
expectedException.add(exception);
countDownLatch.countDown();
}
}, null);
countDownLatch.await();
assertEquals(expectedException.size(), 1);
assertEquals(expectedException.get(0).getCause().getMessage(), failureMsg);
}

}

0 comments on commit 24c337f

Please sign in to comment.