Skip to content

Commit 16ea0d8

Browse files
[fix] [broker][branch-3.0] Expire messages according to ledger close time to avoid client clock skew (#21940) (#22211)
Co-authored-by: feynmanlin <[email protected]>
1 parent dc035f5 commit 16ea0d8

File tree

2 files changed

+58
-2
lines changed

2 files changed

+58
-2
lines changed

pulsar-broker/src/main/java/org/apache/pulsar/broker/service/persistent/PersistentMessageExpiryMonitor.java

+34-1
Original file line numberDiff line numberDiff line change
@@ -21,6 +21,7 @@
2121
import com.google.common.annotations.VisibleForTesting;
2222
import java.util.Objects;
2323
import java.util.Optional;
24+
import java.util.SortedMap;
2425
import java.util.concurrent.atomic.AtomicIntegerFieldUpdater;
2526
import java.util.concurrent.atomic.LongAdder;
2627
import org.apache.bookkeeper.mledger.AsyncCallbacks.FindEntryCallback;
@@ -30,8 +31,10 @@
3031
import org.apache.bookkeeper.mledger.ManagedLedgerException.LedgerNotExistException;
3132
import org.apache.bookkeeper.mledger.ManagedLedgerException.NonRecoverableLedgerException;
3233
import org.apache.bookkeeper.mledger.Position;
34+
import org.apache.bookkeeper.mledger.impl.ManagedCursorImpl;
3335
import org.apache.bookkeeper.mledger.impl.ManagedLedgerImpl;
3436
import org.apache.bookkeeper.mledger.impl.PositionImpl;
37+
import org.apache.bookkeeper.mledger.proto.MLDataFormats;
3538
import org.apache.pulsar.client.impl.MessageImpl;
3639
import org.apache.pulsar.common.api.proto.CommandSubscribe.SubType;
3740
import org.apache.pulsar.common.protocol.Commands;
@@ -78,7 +81,9 @@ public boolean expireMessages(int messageTTLInSeconds) {
7881
if (expirationCheckInProgressUpdater.compareAndSet(this, FALSE, TRUE)) {
7982
log.info("[{}][{}] Starting message expiry check, ttl= {} seconds", topicName, subName,
8083
messageTTLInSeconds);
81-
84+
// First filter the entire Ledger reached TTL based on the Ledger closing time to avoid client clock skew
85+
checkExpiryByLedgerClosureTime(cursor, messageTTLInSeconds);
86+
// Some part of entries in active Ledger may have reached TTL, so we need to continue searching.
8287
cursor.asyncFindNewestMatching(ManagedCursor.FindPositionConstraint.SearchActiveEntries, entry -> {
8388
try {
8489
long entryTimestamp = Commands.getEntryTimestamp(entry.getDataBuffer());
@@ -99,6 +104,34 @@ public boolean expireMessages(int messageTTLInSeconds) {
99104
return false;
100105
}
101106
}
107+
private void checkExpiryByLedgerClosureTime(ManagedCursor cursor, int messageTTLInSeconds) {
108+
if (messageTTLInSeconds <= 0) {
109+
return;
110+
}
111+
if (cursor instanceof ManagedCursorImpl managedCursor) {
112+
ManagedLedgerImpl managedLedger = (ManagedLedgerImpl) managedCursor.getManagedLedger();
113+
Position deletedPosition = managedCursor.getMarkDeletedPosition();
114+
SortedMap<Long, MLDataFormats.ManagedLedgerInfo.LedgerInfo> ledgerInfoSortedMap =
115+
managedLedger.getLedgersInfo().subMap(deletedPosition.getLedgerId(), true,
116+
managedLedger.getLedgersInfo().lastKey(), true);
117+
MLDataFormats.ManagedLedgerInfo.LedgerInfo info = null;
118+
for (MLDataFormats.ManagedLedgerInfo.LedgerInfo ledgerInfo : ledgerInfoSortedMap.values()) {
119+
if (!ledgerInfo.hasTimestamp() || !MessageImpl.isEntryExpired(messageTTLInSeconds,
120+
ledgerInfo.getTimestamp())) {
121+
break;
122+
}
123+
info = ledgerInfo;
124+
}
125+
if (info != null && info.getLedgerId() > -1) {
126+
PositionImpl position = PositionImpl.get(info.getLedgerId(), info.getEntries() - 1);
127+
if (((PositionImpl) managedLedger.getLastConfirmedEntry()).compareTo(position) < 0) {
128+
findEntryComplete(managedLedger.getLastConfirmedEntry(), null);
129+
} else {
130+
findEntryComplete(position, null);
131+
}
132+
}
133+
}
134+
}
102135

103136
public boolean expireMessages(Position messagePosition) {
104137
// If it's beyond last position of this topic, do nothing.

pulsar-broker/src/test/java/org/apache/pulsar/broker/service/PersistentMessageFinderTest.java

+24-1
Original file line numberDiff line numberDiff line change
@@ -80,8 +80,11 @@
8080
public class PersistentMessageFinderTest extends MockedBookKeeperTestCase {
8181

8282
public static byte[] createMessageWrittenToLedger(String msg) {
83+
return createMessageWrittenToLedger(msg, System.currentTimeMillis());
84+
}
85+
public static byte[] createMessageWrittenToLedger(String msg, long messageTimestamp) {
8386
MessageMetadata messageMetadata = new MessageMetadata()
84-
.setPublishTime(System.currentTimeMillis())
87+
.setPublishTime(messageTimestamp)
8588
.setProducerName("createMessageWrittenToLedger")
8689
.setSequenceId(1);
8790
ByteBuf data = UnpooledByteBufAllocator.DEFAULT.heapBuffer().writeBytes(msg.getBytes());
@@ -428,6 +431,26 @@ void testMessageExpiryWithTimestampNonRecoverableException() throws Exception {
428431

429432
}
430433

434+
@Test
435+
public void testIncorrectClientClock() throws Exception {
436+
final String ledgerAndCursorName = "testIncorrectClientClock";
437+
int maxTTLSeconds = 1;
438+
ManagedLedgerConfig config = new ManagedLedgerConfig();
439+
config.setMaxEntriesPerLedger(1);
440+
ManagedLedgerImpl ledger = (ManagedLedgerImpl) factory.open(ledgerAndCursorName, config);
441+
ManagedCursorImpl c1 = (ManagedCursorImpl) ledger.openCursor(ledgerAndCursorName);
442+
// set client clock to 10 days later
443+
long incorrectPublishTimestamp = System.currentTimeMillis() + TimeUnit.DAYS.toMillis(10);
444+
for (int i = 0; i < 10; i++) {
445+
ledger.addEntry(createMessageWrittenToLedger("msg" + i, incorrectPublishTimestamp));
446+
}
447+
assertEquals(ledger.getLedgersInfoAsList().size(), 10);
448+
PersistentMessageExpiryMonitor monitor = new PersistentMessageExpiryMonitor("topicname", c1.getName(), c1, null);
449+
Thread.sleep(TimeUnit.SECONDS.toMillis(maxTTLSeconds));
450+
monitor.expireMessages(maxTTLSeconds);
451+
assertEquals(c1.getNumberOfEntriesInBacklog(true), 0);
452+
}
453+
431454
@Test
432455
void testMessageExpiryWithPosition() throws Exception {
433456
final String ledgerAndCursorName = "testPersistentMessageExpiryWithPositionNonRecoverableLedgers";

0 commit comments

Comments
 (0)