|
96 | 96 | import java.util.Optional;
|
97 | 97 | import java.util.Properties;
|
98 | 98 | import java.util.Set;
|
| 99 | +import java.util.concurrent.CompletableFuture; |
99 | 100 | import java.util.concurrent.ConcurrentLinkedQueue;
|
100 | 101 | import java.util.concurrent.LinkedBlockingQueue;
|
| 102 | +import java.util.concurrent.atomic.AtomicInteger; |
101 | 103 | import java.util.stream.Collectors;
|
102 | 104 | import java.util.stream.Stream;
|
103 | 105 |
|
@@ -548,6 +550,89 @@ public void testAcknowledgeOnCloseWithPendingCommitSync() {
|
548 | 550 | completedAcknowledgements.clear();
|
549 | 551 | }
|
550 | 552 |
|
| 553 | + @Test |
| 554 | + public void testResultHandlerOnCommitAsync() { |
| 555 | + buildRequestManager(); |
| 556 | + // Enabling the config so that background event is sent when the acknowledgement response is received. |
| 557 | + shareConsumeRequestManager.setAcknowledgementCommitCallbackRegistered(true); |
| 558 | + |
| 559 | + Acknowledgements acknowledgements = Acknowledgements.empty(); |
| 560 | + acknowledgements.add(1L, AcknowledgeType.ACCEPT); |
| 561 | + acknowledgements.add(2L, AcknowledgeType.ACCEPT); |
| 562 | + acknowledgements.add(3L, AcknowledgeType.REJECT); |
| 563 | + |
| 564 | + ShareConsumeRequestManager.ResultHandler resultHandler = shareConsumeRequestManager.buildResultHandler(null, Optional.empty()); |
| 565 | + |
| 566 | + // Passing null acknowledgements should mean we do not send the background event at all. |
| 567 | + resultHandler.complete(tip0, null, true); |
| 568 | + assertEquals(0, completedAcknowledgements.size()); |
| 569 | + |
| 570 | + // Setting isCommitAsync to false should still not send any background event |
| 571 | + // as we have initialized remainingResults to null. |
| 572 | + resultHandler.complete(tip0, acknowledgements, false); |
| 573 | + assertEquals(0, completedAcknowledgements.size()); |
| 574 | + |
| 575 | + // Sending non-null acknowledgements means we do send the background event |
| 576 | + resultHandler.complete(tip0, acknowledgements, true); |
| 577 | + assertEquals(3, completedAcknowledgements.get(0).get(tip0).size()); |
| 578 | + } |
| 579 | + |
| 580 | + @Test |
| 581 | + public void testResultHandlerOnCommitSync() { |
| 582 | + buildRequestManager(); |
| 583 | + // Enabling the config so that background event is sent when the acknowledgement response is received. |
| 584 | + shareConsumeRequestManager.setAcknowledgementCommitCallbackRegistered(true); |
| 585 | + |
| 586 | + Acknowledgements acknowledgements = Acknowledgements.empty(); |
| 587 | + acknowledgements.add(1L, AcknowledgeType.ACCEPT); |
| 588 | + acknowledgements.add(2L, AcknowledgeType.ACCEPT); |
| 589 | + acknowledgements.add(3L, AcknowledgeType.REJECT); |
| 590 | + |
| 591 | + final CompletableFuture<Map<TopicIdPartition, Acknowledgements>> future = new CompletableFuture<>(); |
| 592 | + |
| 593 | + // Initializing resultCount to 3. |
| 594 | + AtomicInteger resultCount = new AtomicInteger(3); |
| 595 | + |
| 596 | + ShareConsumeRequestManager.ResultHandler resultHandler = shareConsumeRequestManager.buildResultHandler(resultCount, Optional.of(future)); |
| 597 | + |
| 598 | + // We only send the background event after all results have been completed. |
| 599 | + resultHandler.complete(tip0, acknowledgements, false); |
| 600 | + assertEquals(0, completedAcknowledgements.size()); |
| 601 | + assertFalse(future.isDone()); |
| 602 | + |
| 603 | + resultHandler.complete(t2ip0, null, false); |
| 604 | + assertEquals(0, completedAcknowledgements.size()); |
| 605 | + assertFalse(future.isDone()); |
| 606 | + |
| 607 | + // After third response is received, we send the background event. |
| 608 | + resultHandler.complete(tip1, acknowledgements, false); |
| 609 | + assertEquals(1, completedAcknowledgements.size()); |
| 610 | + assertEquals(2, completedAcknowledgements.get(0).size()); |
| 611 | + assertEquals(3, completedAcknowledgements.get(0).get(tip0).size()); |
| 612 | + assertEquals(3, completedAcknowledgements.get(0).get(tip1).size()); |
| 613 | + assertTrue(future.isDone()); |
| 614 | + } |
| 615 | + |
| 616 | + @Test |
| 617 | + public void testResultHandlerCompleteIfEmpty() { |
| 618 | + buildRequestManager(); |
| 619 | + |
| 620 | + final CompletableFuture<Map<TopicIdPartition, Acknowledgements>> future = new CompletableFuture<>(); |
| 621 | + |
| 622 | + // Initializing resultCount to 1. |
| 623 | + AtomicInteger resultCount = new AtomicInteger(1); |
| 624 | + |
| 625 | + ShareConsumeRequestManager.ResultHandler resultHandler = shareConsumeRequestManager.buildResultHandler(resultCount, Optional.of(future)); |
| 626 | + |
| 627 | + resultHandler.completeIfEmpty(); |
| 628 | + assertFalse(future.isDone()); |
| 629 | + |
| 630 | + resultCount.decrementAndGet(); |
| 631 | + |
| 632 | + resultHandler.completeIfEmpty(); |
| 633 | + assertTrue(future.isDone()); |
| 634 | + } |
| 635 | + |
551 | 636 | @Test
|
552 | 637 | public void testBatchingAcknowledgeRequestStates() {
|
553 | 638 | buildRequestManager();
|
@@ -1730,6 +1815,11 @@ private int sendAcknowledgements() {
|
1730 | 1815 | return pollResult.unsentRequests.size();
|
1731 | 1816 | }
|
1732 | 1817 |
|
| 1818 | + public ResultHandler buildResultHandler(final AtomicInteger remainingResults, |
| 1819 | + final Optional<CompletableFuture<Map<TopicIdPartition, Acknowledgements>>> future) { |
| 1820 | + return new ResultHandler(remainingResults, future); |
| 1821 | + } |
| 1822 | + |
1733 | 1823 | public Tuple<AcknowledgeRequestState> requestStates(int nodeId) {
|
1734 | 1824 | return super.requestStates(nodeId);
|
1735 | 1825 | }
|
|
0 commit comments