congbobo184 commented on code in PR #15592:
URL: https://github.com/apache/pulsar/pull/15592#discussion_r873285996
##########
pulsar-broker/src/main/java/org/apache/pulsar/broker/transaction/buffer/impl/TopicTransactionBuffer.java:
##########
@@ -384,30 +376,32 @@ public void addFailed(ManagedLedgerException exception,
Object ctx) {
}
private void handleLowWaterMark(TxnID txnID, long lowWaterMark) {
- if (!ongoingTxns.isEmpty()) {
- TxnID firstTxn = ongoingTxns.firstKey();
- if (firstTxn.getMostSigBits() == txnID.getMostSigBits() &&
lowWaterMark >= firstTxn.getLeastSigBits()) {
- ByteBuf abortMarker = Markers.newTxnAbortMarker(-1L,
- firstTxn.getMostSigBits(), firstTxn.getLeastSigBits());
- try {
- topic.getManagedLedger().asyncAddEntry(abortMarker, new
AsyncCallbacks.AddEntryCallback() {
- @Override
- public void addComplete(Position position, ByteBuf
entryData, Object ctx) {
- synchronized (TopicTransactionBuffer.this) {
- aborts.put(firstTxn, (PositionImpl) position);
- updateMaxReadPosition(firstTxn);
- }
- }
-
- @Override
- public void addFailed(ManagedLedgerException
exception, Object ctx) {
- log.error("Failed to abort low water mark for txn
{}", txnID, exception);
- }
- }, null);
- } finally {
- abortMarker.release();
+ lowWaterMarks.compute(txnID.getMostSigBits(), (tcId, oldLowWaterMark)
-> {
+ if (oldLowWaterMark == null || oldLowWaterMark < lowWaterMark) {
+ return lowWaterMark;
+ } else {
+ return oldLowWaterMark;
+ }
+ });
+ if (handleLowWaterMark.tryAcquire()) {
+ if (!ongoingTxns.isEmpty()) {
+ TxnID firstTxn = ongoingTxns.firstKey();
+ long tCId = firstTxn.getMostSigBits();
+ if (lowWaterMarks.get(tCId) != null &&
firstTxn.getLeastSigBits() <= lowWaterMarks.get(tCId)) {
+ abortTxn(firstTxn, lowWaterMarks.get(tCId))
+ .thenRun(() -> {
+ log.warn("Successed to abort low water mark
for txn {}", firstTxn);
Review Comment:
log topic name, lowWaterMark
##########
pulsar-broker/src/main/java/org/apache/pulsar/broker/transaction/buffer/impl/TopicTransactionBuffer.java:
##########
@@ -384,30 +376,32 @@ public void addFailed(ManagedLedgerException exception,
Object ctx) {
}
private void handleLowWaterMark(TxnID txnID, long lowWaterMark) {
- if (!ongoingTxns.isEmpty()) {
- TxnID firstTxn = ongoingTxns.firstKey();
- if (firstTxn.getMostSigBits() == txnID.getMostSigBits() &&
lowWaterMark >= firstTxn.getLeastSigBits()) {
- ByteBuf abortMarker = Markers.newTxnAbortMarker(-1L,
- firstTxn.getMostSigBits(), firstTxn.getLeastSigBits());
- try {
- topic.getManagedLedger().asyncAddEntry(abortMarker, new
AsyncCallbacks.AddEntryCallback() {
- @Override
- public void addComplete(Position position, ByteBuf
entryData, Object ctx) {
- synchronized (TopicTransactionBuffer.this) {
- aborts.put(firstTxn, (PositionImpl) position);
- updateMaxReadPosition(firstTxn);
- }
- }
-
- @Override
- public void addFailed(ManagedLedgerException
exception, Object ctx) {
- log.error("Failed to abort low water mark for txn
{}", txnID, exception);
- }
- }, null);
- } finally {
- abortMarker.release();
+ lowWaterMarks.compute(txnID.getMostSigBits(), (tcId, oldLowWaterMark)
-> {
+ if (oldLowWaterMark == null || oldLowWaterMark < lowWaterMark) {
+ return lowWaterMark;
+ } else {
+ return oldLowWaterMark;
+ }
+ });
+ if (handleLowWaterMark.tryAcquire()) {
+ if (!ongoingTxns.isEmpty()) {
+ TxnID firstTxn = ongoingTxns.firstKey();
+ long tCId = firstTxn.getMostSigBits();
+ if (lowWaterMarks.get(tCId) != null &&
firstTxn.getLeastSigBits() <= lowWaterMarks.get(tCId)) {
Review Comment:
```suggestion
Long lowWaterMarkOfFirstTxnId = lowWaterMarks.get(tCId);
if (lowWaterMarkOfFirstTxnId != null &&
firstTxn.getLeastSigBits() <= lowWaterMarkOfFirstTxnId) {
```
##########
pulsar-broker/src/main/java/org/apache/pulsar/broker/transaction/buffer/impl/TopicTransactionBuffer.java:
##########
@@ -384,30 +376,32 @@ public void addFailed(ManagedLedgerException exception,
Object ctx) {
}
private void handleLowWaterMark(TxnID txnID, long lowWaterMark) {
- if (!ongoingTxns.isEmpty()) {
- TxnID firstTxn = ongoingTxns.firstKey();
- if (firstTxn.getMostSigBits() == txnID.getMostSigBits() &&
lowWaterMark >= firstTxn.getLeastSigBits()) {
- ByteBuf abortMarker = Markers.newTxnAbortMarker(-1L,
- firstTxn.getMostSigBits(), firstTxn.getLeastSigBits());
- try {
- topic.getManagedLedger().asyncAddEntry(abortMarker, new
AsyncCallbacks.AddEntryCallback() {
- @Override
- public void addComplete(Position position, ByteBuf
entryData, Object ctx) {
- synchronized (TopicTransactionBuffer.this) {
- aborts.put(firstTxn, (PositionImpl) position);
- updateMaxReadPosition(firstTxn);
- }
- }
-
- @Override
- public void addFailed(ManagedLedgerException
exception, Object ctx) {
- log.error("Failed to abort low water mark for txn
{}", txnID, exception);
- }
- }, null);
- } finally {
- abortMarker.release();
+ lowWaterMarks.compute(txnID.getMostSigBits(), (tcId, oldLowWaterMark)
-> {
+ if (oldLowWaterMark == null || oldLowWaterMark < lowWaterMark) {
+ return lowWaterMark;
+ } else {
+ return oldLowWaterMark;
+ }
+ });
+ if (handleLowWaterMark.tryAcquire()) {
+ if (!ongoingTxns.isEmpty()) {
+ TxnID firstTxn = ongoingTxns.firstKey();
+ long tCId = firstTxn.getMostSigBits();
+ if (lowWaterMarks.get(tCId) != null &&
firstTxn.getLeastSigBits() <= lowWaterMarks.get(tCId)) {
+ abortTxn(firstTxn, lowWaterMarks.get(tCId))
+ .thenRun(() -> {
+ log.warn("Successed to abort low water mark
for txn {}", firstTxn);
+ handleLowWaterMark.release();
+ })
+ .exceptionally(ex -> {
+ log.warn("Failed to abort low water mark for
txn {}", firstTxn, ex);
Review Comment:
log topic name, lowWaterMark
##########
pulsar-broker/src/main/java/org/apache/pulsar/broker/transaction/pendingack/impl/PendingAckHandleImpl.java:
##########
@@ -595,20 +604,33 @@ public CompletableFuture<Void> abortTxn(TxnID txnId,
Consumer consumer, long low
}
private void handleLowWaterMark(TxnID txnID, long lowWaterMark) {
- if (individualAckOfTransaction != null &&
!individualAckOfTransaction.isEmpty()) {
- TxnID firstTxn = individualAckOfTransaction.firstKey();
-
- if (firstTxn.getMostSigBits() == txnID.getMostSigBits()
- && firstTxn.getLeastSigBits() <= lowWaterMark) {
- abortTxn(firstTxn, null, lowWaterMark).thenRun(() -> {
- log.warn("[{}] Transaction pending ack handle low water
mark success! txnId : [{}], "
- + "lowWaterMark : [{}]", topicName, txnID,
lowWaterMark);
- }).exceptionally(e -> {
- log.warn("[{}] Transaction pending ack handle low water
mark fail! txnId : [{}], "
- + "lowWaterMark : [{}]", topicName, txnID,
lowWaterMark);
- return null;
- });
+ lowWaterMarks.compute(txnID.getMostSigBits(), (tcId, oldLowWaterMark)
-> {
+ if (oldLowWaterMark == null || oldLowWaterMark < lowWaterMark) {
+ return lowWaterMark;
+ } else {
+ return oldLowWaterMark;
+ }
+ });
+
+ if (handleLowWaterMark.tryAcquire()) {
+ if (individualAckOfTransaction != null &&
!individualAckOfTransaction.isEmpty()) {
+ TxnID firstTxn = individualAckOfTransaction.firstKey();
+ long tCId = firstTxn.getMostSigBits();
+ if (lowWaterMarks.get(tCId) != null &&
firstTxn.getLeastSigBits() <= lowWaterMarks.get(tCId)) {
Review Comment:
```suggestion
Long lowWaterMarkOfFirstTxnId = lowWaterMarks.get(tCId);
if (lowWaterMarkOfFirstTxnId != null && firstTxn.getLeastSigBits()
<= lowWaterMarkOfFirstTxnId) {
```
##########
pulsar-broker/src/main/java/org/apache/pulsar/broker/transaction/buffer/impl/TopicTransactionBuffer.java:
##########
@@ -384,30 +376,32 @@ public void addFailed(ManagedLedgerException exception,
Object ctx) {
}
private void handleLowWaterMark(TxnID txnID, long lowWaterMark) {
- if (!ongoingTxns.isEmpty()) {
- TxnID firstTxn = ongoingTxns.firstKey();
- if (firstTxn.getMostSigBits() == txnID.getMostSigBits() &&
lowWaterMark >= firstTxn.getLeastSigBits()) {
- ByteBuf abortMarker = Markers.newTxnAbortMarker(-1L,
- firstTxn.getMostSigBits(), firstTxn.getLeastSigBits());
- try {
- topic.getManagedLedger().asyncAddEntry(abortMarker, new
AsyncCallbacks.AddEntryCallback() {
- @Override
- public void addComplete(Position position, ByteBuf
entryData, Object ctx) {
- synchronized (TopicTransactionBuffer.this) {
- aborts.put(firstTxn, (PositionImpl) position);
- updateMaxReadPosition(firstTxn);
- }
- }
-
- @Override
- public void addFailed(ManagedLedgerException
exception, Object ctx) {
- log.error("Failed to abort low water mark for txn
{}", txnID, exception);
- }
- }, null);
- } finally {
- abortMarker.release();
+ lowWaterMarks.compute(txnID.getMostSigBits(), (tcId, oldLowWaterMark)
-> {
+ if (oldLowWaterMark == null || oldLowWaterMark < lowWaterMark) {
+ return lowWaterMark;
+ } else {
+ return oldLowWaterMark;
+ }
+ });
+ if (handleLowWaterMark.tryAcquire()) {
+ if (!ongoingTxns.isEmpty()) {
+ TxnID firstTxn = ongoingTxns.firstKey();
+ long tCId = firstTxn.getMostSigBits();
+ if (lowWaterMarks.get(tCId) != null &&
firstTxn.getLeastSigBits() <= lowWaterMarks.get(tCId)) {
+ abortTxn(firstTxn, lowWaterMarks.get(tCId))
Review Comment:
```suggestion
abortTxn(firstTxn, lowWaterMarkOfFirstTxnId)
```
##########
pulsar-broker/src/test/java/org/apache/pulsar/broker/transaction/buffer/TransactionLowWaterMarkTest.java:
##########
@@ -325,4 +327,127 @@ public void testTBLowWaterMarkEndToEnd() throws Exception
{
// no-op
}
}
+
+ @Test
+ public void testLowWaterMarkForDifferentTC() throws Exception {
Review Comment:
why we need txn4 and txn5, I think when txn3 has committed the txn1 and txn2
will be write abort marker. and also we need to test the TopicTransactionBuffer
recover and the will recover to a correct state
##########
pulsar-broker/src/main/java/org/apache/pulsar/broker/transaction/pendingack/impl/PendingAckHandleImpl.java:
##########
@@ -595,20 +604,33 @@ public CompletableFuture<Void> abortTxn(TxnID txnId,
Consumer consumer, long low
}
private void handleLowWaterMark(TxnID txnID, long lowWaterMark) {
- if (individualAckOfTransaction != null &&
!individualAckOfTransaction.isEmpty()) {
- TxnID firstTxn = individualAckOfTransaction.firstKey();
-
- if (firstTxn.getMostSigBits() == txnID.getMostSigBits()
- && firstTxn.getLeastSigBits() <= lowWaterMark) {
- abortTxn(firstTxn, null, lowWaterMark).thenRun(() -> {
- log.warn("[{}] Transaction pending ack handle low water
mark success! txnId : [{}], "
- + "lowWaterMark : [{}]", topicName, txnID,
lowWaterMark);
- }).exceptionally(e -> {
- log.warn("[{}] Transaction pending ack handle low water
mark fail! txnId : [{}], "
- + "lowWaterMark : [{}]", topicName, txnID,
lowWaterMark);
- return null;
- });
+ lowWaterMarks.compute(txnID.getMostSigBits(), (tcId, oldLowWaterMark)
-> {
+ if (oldLowWaterMark == null || oldLowWaterMark < lowWaterMark) {
+ return lowWaterMark;
+ } else {
+ return oldLowWaterMark;
+ }
+ });
+
+ if (handleLowWaterMark.tryAcquire()) {
+ if (individualAckOfTransaction != null &&
!individualAckOfTransaction.isEmpty()) {
+ TxnID firstTxn = individualAckOfTransaction.firstKey();
+ long tCId = firstTxn.getMostSigBits();
+ if (lowWaterMarks.get(tCId) != null &&
firstTxn.getLeastSigBits() <= lowWaterMarks.get(tCId)) {
+ abortTxn(firstTxn, null,
lowWaterMarks.get(tCId)).thenRun(() -> {
Review Comment:
```suggestion
abortTxn(firstTxn, null,
lowWaterMarkOfFirstTxnId).thenRun(() -> {
```
--
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]