Skip to content

Commit d85a5e2

Browse files
authored
[Feature] Introduce continuous offset for pulsar (#9039)
Fixes #9038 ### Motivation As described in [PIP-70](https://github.com/apache/pulsar/wiki/PIP-70%3A-Introduce-lightweight-broker-entry-metadata). One of the use case for Broker entry metadata is providing continuous message sequence-Id for messages in one topic-partition which is useful for Protocol Hanlder like KOP. This PR enable Pulsar to support continuous offset for message based on Broker entry metadata. ### Modifications Introduce a new field for broker entry metadta named `offset`; Introduce a new interceptor type `ManagedLedgerInterceptor` which intercept entry in `ManagedLedger`; Each partition will be assigned a `ManagedLedgerInterceptor` when `ManagedLedger` created; Each Entry will be intercept for adding a monotone increasing offset in Broker entry metadata and the offet is added by batchSize of entry; Support find position by a given offset.
1 parent 11b9359 commit d85a5e2

File tree

21 files changed

+897
-37
lines changed

21 files changed

+897
-37
lines changed

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

Lines changed: 74 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -32,6 +32,7 @@
3232
import org.apache.bookkeeper.mledger.AsyncCallbacks.OffloadCallback;
3333
import org.apache.bookkeeper.mledger.AsyncCallbacks.OpenCursorCallback;
3434
import org.apache.bookkeeper.mledger.AsyncCallbacks.TerminateCallback;
35+
import org.apache.bookkeeper.mledger.interceptor.ManagedLedgerInterceptor;
3536
import org.apache.pulsar.common.api.proto.PulsarApi.CommandSubscribe.InitialPosition;
3637

3738
/**
@@ -74,6 +75,18 @@ public interface ManagedLedger {
7475
*/
7576
Position addEntry(byte[] data) throws InterruptedException, ManagedLedgerException;
7677

78+
/**
79+
* Append a new entry to the end of a managed ledger.
80+
*
81+
* @param data
82+
* data entry to be persisted
83+
* @param numberOfMessages
84+
* numberOfMessages of entry
85+
* @return the Position at which the entry has been inserted
86+
* @throws ManagedLedgerException
87+
*/
88+
Position addEntry(byte[] data, int numberOfMessages) throws InterruptedException, ManagedLedgerException;
89+
7790
/**
7891
* Append a new entry asynchronously.
7992
*
@@ -102,6 +115,22 @@ public interface ManagedLedger {
102115
*/
103116
Position addEntry(byte[] data, int offset, int length) throws InterruptedException, ManagedLedgerException;
104117

118+
/**
119+
* Append a new entry to the end of a managed ledger.
120+
*
121+
* @param data
122+
* data entry to be persisted
123+
* @param numberOfMessages
124+
* numberOfMessages of entry
125+
* @param offset
126+
* offset in the data array
127+
* @param length
128+
* number of bytes
129+
* @return the Position at which the entry has been inserted
130+
* @throws ManagedLedgerException
131+
*/
132+
Position addEntry(byte[] data, int numberOfMessages, int offset, int length) throws InterruptedException, ManagedLedgerException;
133+
105134
/**
106135
* Append a new entry asynchronously.
107136
*
@@ -119,6 +148,26 @@ public interface ManagedLedger {
119148
*/
120149
void asyncAddEntry(byte[] data, int offset, int length, AddEntryCallback callback, Object ctx);
121150

151+
/**
152+
* Append a new entry asynchronously.
153+
*
154+
* @see #addEntry(byte[])
155+
* @param data
156+
* data entry to be persisted
157+
* @param numberOfMessages
158+
* numberOfMessages of entry
159+
* @param offset
160+
* offset in the data array
161+
* @param length
162+
* number of bytes
163+
* @param callback
164+
* callback object
165+
* @param ctx
166+
* opaque context
167+
*/
168+
void asyncAddEntry(byte[] data, int numberOfMessages, int offset, int length, AddEntryCallback callback, Object ctx);
169+
170+
122171
/**
123172
* Append a new entry asynchronously.
124173
*
@@ -132,6 +181,21 @@ public interface ManagedLedger {
132181
*/
133182
void asyncAddEntry(ByteBuf buffer, AddEntryCallback callback, Object ctx);
134183

184+
/**
185+
* Append a new entry asynchronously.
186+
*
187+
* @see #addEntry(byte[])
188+
* @param buffer
189+
* buffer with the data entry
190+
* @param numberOfMessages
191+
* numberOfMessages for data entry
192+
* @param callback
193+
* callback object
194+
* @param ctx
195+
* opaque context
196+
*/
197+
void asyncAddEntry(ByteBuf buffer, int numberOfMessages, AddEntryCallback callback, Object ctx);
198+
135199
/**
136200
* Open a ManagedCursor in this ManagedLedger.
137201
*
@@ -520,4 +584,14 @@ void asyncSetProperties(Map<String, String> properties, final AsyncCallbacks.Upd
520584
* Roll current ledger if it is full
521585
*/
522586
void rollCurrentLedgerIfFull();
587+
588+
/**
589+
* Find position by sequenceId.
590+
* */
591+
CompletableFuture<Position> asyncFindPosition(com.google.common.base.Predicate<Entry> predicate);
592+
593+
/**
594+
* Get the ManagedLedgerInterceptor for ManagedLedger.
595+
* */
596+
ManagedLedgerInterceptor getManagedLedgerInterceptor();
523597
}

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

Lines changed: 9 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -33,6 +33,7 @@
3333
import org.apache.bookkeeper.common.annotation.InterfaceStability;
3434
import org.apache.bookkeeper.mledger.impl.NullLedgerOffloader;
3535

36+
import org.apache.bookkeeper.mledger.interceptor.ManagedLedgerInterceptor;
3637
import org.apache.pulsar.common.util.collections.ConcurrentOpenLongPairRangeSet;
3738

3839
/**
@@ -75,6 +76,7 @@ public class ManagedLedgerConfig {
7576
private LedgerOffloader ledgerOffloader = NullLedgerOffloader.INSTANCE;
7677
private int newEntriesCheckDelayInMillis = 10;
7778
private Clock clock = Clock.systemUTC();
79+
private ManagedLedgerInterceptor managedLedgerInterceptor;
7880

7981
public boolean isCreateIfMissing() {
8082
return createIfMissing;
@@ -637,4 +639,11 @@ public void setNewEntriesCheckDelayInMillis(int newEntriesCheckDelayInMillis) {
637639
this.newEntriesCheckDelayInMillis = newEntriesCheckDelayInMillis;
638640
}
639641

642+
public ManagedLedgerInterceptor getManagedLedgerInterceptor() {
643+
return managedLedgerInterceptor;
644+
}
645+
646+
public void setManagedLedgerInterceptor(ManagedLedgerInterceptor managedLedgerInterceptor) {
647+
this.managedLedgerInterceptor = managedLedgerInterceptor;
648+
}
640649
}

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

Lines changed: 6 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -157,6 +157,12 @@ public CursorNotFoundException(String msg) {
157157
}
158158
}
159159

160+
public static class ManagedLedgerInterceptException extends ManagedLedgerException {
161+
public ManagedLedgerInterceptException(String msg) {
162+
super(msg);
163+
}
164+
}
165+
160166
@Override
161167
public synchronized Throwable fillInStackTrace() {
162168
// Disable stack traces to be filled in

0 commit comments

Comments
 (0)