|
37 | 37 | import com.rabbitmq.stream.impl.Client.Response;
|
38 | 38 | import com.rabbitmq.stream.impl.Client.StreamParametersBuilder;
|
39 | 39 | import com.rabbitmq.stream.impl.ServerFrameHandler.FrameHandlerInfo;
|
| 40 | +import com.rabbitmq.stream.impl.TestUtils.BrokerVersionAtLeast; |
40 | 41 | import io.netty.buffer.ByteBufAllocator;
|
41 | 42 | import io.netty.buffer.UnpooledByteBufAllocator;
|
42 | 43 | import java.io.ByteArrayOutputStream;
|
@@ -259,7 +260,7 @@ void publishConsumeComplexMessage(
|
259 | 260 | Client.ChunkListener chunkListener =
|
260 | 261 | (client, correlationId, offset, messageCount, dataSize) -> client.credit(correlationId, 1);
|
261 | 262 | Client.MessageListener messageListener =
|
262 |
| - (correlationId, offset, chunkTimestamp, message) -> { |
| 263 | + (correlationId, offset, chunkTimestamp, committedOffset, message) -> { |
263 | 264 | messages.add(message);
|
264 | 265 | latch.countDown();
|
265 | 266 | };
|
@@ -395,7 +396,7 @@ void publishConsumeWithSimpleCodec() throws Exception {
|
395 | 396 | (client, subscriptionId, offset, messageCount1, dataSize) ->
|
396 | 397 | client.credit(subscriptionId, 1))
|
397 | 398 | .messageListener(
|
398 |
| - (subscriptionId, offset, chunkTimestamp, message) -> { |
| 399 | + (subscriptionId, offset, chunkTimestamp, committedOffset, message) -> { |
399 | 400 | messageBodies.add(new String(message.getBodyAsBinary()));
|
400 | 401 | consumeLatch.countDown();
|
401 | 402 | }));
|
@@ -448,7 +449,7 @@ void batchPublishing() throws Exception {
|
448 | 449 | (client, subscriptionId, offset, messageCount1, dataSize) ->
|
449 | 450 | client.credit(subscriptionId, 1))
|
450 | 451 | .messageListener(
|
451 |
| - (subscriptionId, offset, chunkTimestamp, message) -> { |
| 452 | + (subscriptionId, offset, chunkTimestamp, committedOffset, message) -> { |
452 | 453 | ByteBuffer bb = ByteBuffer.wrap(message.getBodyAsBinary());
|
453 | 454 | sizes.add(message.getBodyAsBinary().length);
|
454 | 455 | sequences.add(bb.getInt());
|
@@ -479,7 +480,7 @@ void consume() throws Exception {
|
479 | 480 |
|
480 | 481 | AtomicLong chunkTimestamp = new AtomicLong();
|
481 | 482 | Client.MessageListener messageListener =
|
482 |
| - (corr, offset, chkTimestamp, message) -> { |
| 483 | + (corr, offset, chkTimestamp, committedOffset, message) -> { |
483 | 484 | chunkTimestamp.set(chkTimestamp);
|
484 | 485 | latch.countDown();
|
485 | 486 | };
|
@@ -515,7 +516,7 @@ void publishAndConsume(boolean directBuffer) throws Exception {
|
515 | 516 | };
|
516 | 517 |
|
517 | 518 | Client.MessageListener messageListener =
|
518 |
| - (corr, offset, chunkTimestamp, data) -> consumedLatch.countDown(); |
| 519 | + (corr, offset, chunkTimestamp, committedOffset, data) -> consumedLatch.countDown(); |
519 | 520 |
|
520 | 521 | Client client =
|
521 | 522 | cf.get(
|
@@ -659,7 +660,7 @@ void declareAmqpStreamQueueAndUseItAsStream(TestInfo info) throws Exception {
|
659 | 660 | (client1, subscriptionId, offset, messageCount1, dataSize) ->
|
660 | 661 | client1.credit(subscriptionId, 1))
|
661 | 662 | .messageListener(
|
662 |
| - (subscriptionId, offset, chunkTimestamp, message) -> |
| 663 | + (subscriptionId, offset, chunkTimestamp, committedOffset, message) -> |
663 | 664 | consumedLatch.countDown()));
|
664 | 665 | ConnectionFactory connectionFactory = new ConnectionFactory();
|
665 | 666 | try (Connection amqpConnection = connectionFactory.newConnection()) {
|
@@ -830,11 +831,50 @@ void closingPublisherWhilePublishingShouldNotCloseConnection(String publisherRef
|
830 | 831 | }
|
831 | 832 |
|
832 | 833 | @Test
|
833 |
| - @Disabled |
| 834 | + @BrokerVersionAtLeast("3.11.0") |
834 | 835 | void exchangeCommandVersions() {
|
835 | 836 | Client client = cf.get();
|
836 | 837 | List<FrameHandlerInfo> infos = client.exchangeCommandVersions();
|
837 | 838 | assertThat(infos.stream().filter(info -> info.getKey() == Constants.COMMAND_DECLARE_PUBLISHER))
|
838 | 839 | .isNotEmpty();
|
839 | 840 | }
|
| 841 | + |
| 842 | + @Test |
| 843 | + @BrokerVersionAtLeast("3.11.0") |
| 844 | + void deliverVersion2LastCommittedOffsetShouldBeSet() throws Exception { |
| 845 | + int publishCount = 20_000; |
| 846 | + byte correlationId = 42; |
| 847 | + TestUtils.publishAndWaitForConfirms(cf, publishCount, stream); |
| 848 | + |
| 849 | + CountDownLatch latch = new CountDownLatch(publishCount); |
| 850 | + |
| 851 | + Client.ChunkListener chunkListener = |
| 852 | + (client, corr, offset, messageCountInChunk, dataSize) -> { |
| 853 | + client.credit(correlationId, 1); |
| 854 | + }; |
| 855 | + |
| 856 | + AtomicLong committedOffset = new AtomicLong(); |
| 857 | + Client.MessageListener messageListener = |
| 858 | + (corr, offset, chkTimestamp, committedOfft, message) -> { |
| 859 | + committedOffset.set(committedOfft); |
| 860 | + latch.countDown(); |
| 861 | + }; |
| 862 | + |
| 863 | + Client client = |
| 864 | + cf.get( |
| 865 | + new Client.ClientParameters() |
| 866 | + .chunkListener(chunkListener) |
| 867 | + .messageListener(messageListener)); |
| 868 | + |
| 869 | + client.exchangeCommandVersions(); |
| 870 | + |
| 871 | + Response response = |
| 872 | + client.subscribe(correlationId, stream, OffsetSpecification.first(), credit); |
| 873 | + assertThat(response.getResponseCode()).isEqualTo(Constants.RESPONSE_CODE_OK); |
| 874 | + assertThat(response.isOk()).isTrue(); |
| 875 | + |
| 876 | + assertThat(latch.await(10, SECONDS)).isTrue(); |
| 877 | + assertThat(committedOffset.get()).isPositive(); |
| 878 | + client.close(); |
| 879 | + } |
840 | 880 | }
|
0 commit comments