Skip to content

Commit 4d7650f

Browse files
committed
Add support for super stream + SAC in performance tool
References #46 Conflicts: pom.xml
1 parent 2075971 commit 4d7650f

File tree

7 files changed

+356
-81
lines changed

7 files changed

+356
-81
lines changed

pom.xml

+12-7
Original file line numberDiff line numberDiff line change
@@ -186,6 +186,12 @@
186186
<optional>true</optional>
187187
</dependency>
188188

189+
<dependency>
190+
<groupId>com.rabbitmq</groupId>
191+
<artifactId>amqp-client</artifactId>
192+
<version>${amqp-client.version}</version>
193+
<optional>true</optional>
194+
</dependency>
189195
<!-- end of dependencies for performance tool -->
190196

191197
<dependency>
@@ -245,13 +251,6 @@
245251
<scope>test</scope>
246252
</dependency>
247253

248-
<dependency>
249-
<groupId>com.rabbitmq</groupId>
250-
<artifactId>amqp-client</artifactId>
251-
<version>${amqp-client.version}</version>
252-
<scope>test</scope>
253-
</dependency>
254-
255254
<dependency>
256255
<groupId>org.eclipse.paho</groupId>
257256
<artifactId>org.eclipse.paho.client.mqttv3</artifactId>
@@ -625,6 +624,12 @@
625624
<version>${guava.version}</version>
626625
</dependency>
627626

627+
<dependency>
628+
<groupId>com.rabbitmq</groupId>
629+
<artifactId>amqp-client</artifactId>
630+
<version>${amqp-client.version}</version>
631+
</dependency>
632+
628633
</dependencies>
629634
<build>
630635
<finalName>${finalName}</finalName>

src/main/java/com/rabbitmq/stream/impl/SuperStreamProducer.java

+1-1
Original file line numberDiff line numberDiff line change
@@ -1,4 +1,4 @@
1-
// Copyright (c) 2021 VMware, Inc. or its affiliates. All rights reserved.
1+
// Copyright (c) 2021-2022 VMware, Inc. or its affiliates. All rights reserved.
22
//
33
// This software, the RabbitMQ Stream Java client library, is dual-licensed under the
44
// Mozilla Public License 2.0 ("MPL"), and the Apache License version 2 ("ASL").

src/main/java/com/rabbitmq/stream/perf/StreamPerfTest.java

+130-33
Original file line numberDiff line numberDiff line change
@@ -19,6 +19,8 @@
1919
import static java.lang.String.format;
2020

2121
import com.google.common.util.concurrent.RateLimiter;
22+
import com.rabbitmq.client.Channel;
23+
import com.rabbitmq.client.Connection;
2224
import com.rabbitmq.stream.Address;
2325
import com.rabbitmq.stream.AddressResolver;
2426
import com.rabbitmq.stream.ByteCapacity;
@@ -31,6 +33,7 @@
3133
import com.rabbitmq.stream.Environment;
3234
import com.rabbitmq.stream.EnvironmentBuilder;
3335
import com.rabbitmq.stream.EnvironmentBuilder.TlsConfiguration;
36+
import com.rabbitmq.stream.MessageBuilder;
3437
import com.rabbitmq.stream.OffsetSpecification;
3538
import com.rabbitmq.stream.Producer;
3639
import com.rabbitmq.stream.ProducerBuilder;
@@ -76,6 +79,7 @@
7679
import java.util.concurrent.TimeUnit;
7780
import java.util.concurrent.atomic.AtomicInteger;
7881
import java.util.concurrent.atomic.AtomicLong;
82+
import java.util.concurrent.atomic.AtomicReference;
7983
import java.util.function.BiFunction;
8084
import java.util.function.Function;
8185
import java.util.function.Supplier;
@@ -366,6 +370,30 @@ public void setMaxSegmentSize(ByteCapacity in) {
366370
defaultValue = "false")
367371
private boolean confirmLatency;
368372

373+
@CommandLine.Option(
374+
names = {"--super-streams", "-sst"},
375+
description = "use super streams",
376+
defaultValue = "false")
377+
private boolean superStreams;
378+
379+
@CommandLine.Option(
380+
names = {"--super-streams-partitions", "-ssp"},
381+
description = "number of partitions for the super streams",
382+
defaultValue = "3",
383+
converter = Utils.PositiveIntegerTypeConverter.class)
384+
private int superStreamsPartitions;
385+
386+
@CommandLine.Option(
387+
names = {"--single-active-consumer", "-sac"},
388+
description = "use single active consumer",
389+
defaultValue = "false")
390+
private boolean singleActiveConsumer;
391+
392+
@CommandLine.Option(
393+
names = {"--amqp-uri", "-au"},
394+
description = "AMQP URI to use to create super stream topology")
395+
private String amqpUri;
396+
369397
private MetricsCollector metricsCollector;
370398
private PerformanceMetrics performanceMetrics;
371399
private List<Monitoring> monitorings;
@@ -622,33 +650,32 @@ public Integer call() throws Exception {
622650

623651
streams = Utils.streams(this.streamCount, this.streams);
624652

625-
for (String stream : streams) {
626-
StreamCreator streamCreator =
627-
environment.streamCreator().stream(stream)
628-
.maxLengthBytes(this.maxLengthBytes)
629-
.maxSegmentSizeBytes(this.maxSegmentSize)
630-
.leaderLocator(this.leaderLocator);
631-
632-
if (this.maxAge != null) {
633-
streamCreator.maxAge(this.maxAge);
653+
AtomicReference<Channel> amqpChannel = new AtomicReference<>();
654+
Connection amqpConnection;
655+
if (this.superStreams) {
656+
amqpConnection = Utils.amqpConnection(this.amqpUri, uris, tls, this.sniServerNames);
657+
if (this.deleteStreams) {
658+
// we keep it open for deletion, so adding a close step
659+
shutdownService.wrap(
660+
closeStep("Closing AMQP connection for super streams", () -> amqpConnection.close()));
634661
}
662+
amqpChannel.set(amqpConnection.createChannel());
663+
} else {
664+
amqpConnection = null;
665+
}
635666

636-
try {
637-
streamCreator.create();
638-
} catch (StreamException e) {
639-
if (e.getCode() == Constants.RESPONSE_CODE_PRECONDITION_FAILED) {
640-
String message =
641-
String.format(
642-
"Warning: stream '%s' already exists, but with different properties than "
643-
+ "max-length-bytes=%s, stream-max-segment-size-bytes=%s, queue-leader-locator=%s",
644-
stream, this.maxLengthBytes, this.maxSegmentSize, this.leaderLocator);
645-
if (this.maxAge != null) {
646-
message += String.format(", max-age=%s", this.maxAge);
647-
}
648-
this.out.println(message);
649-
} else {
650-
throw e;
667+
for (String stream : streams) {
668+
if (this.superStreams) {
669+
List<String> partitions =
670+
Utils.superStreamPartitions(stream, this.superStreamsPartitions);
671+
for (String partition : partitions) {
672+
createStream(environment, partition);
651673
}
674+
675+
Utils.declareSuperStreamExchangeAndBindings(amqpChannel.get(), stream, partitions);
676+
677+
} else {
678+
createStream(environment, stream);
652679
}
653680
}
654681

@@ -658,15 +685,30 @@ public Integer call() throws Exception {
658685
"Deleting stream(s)",
659686
() -> {
660687
for (String stream : streams) {
661-
LOGGER.debug("Deleting {}", stream);
662-
try {
663-
environment.deleteStream(stream);
664-
LOGGER.debug("Deleted {}", stream);
665-
} catch (Exception e) {
666-
LOGGER.warn("Could not delete stream {}: {}", stream, e.getMessage());
688+
if (this.superStreams) {
689+
List<String> partitions =
690+
Utils.superStreamPartitions(stream, this.superStreamsPartitions);
691+
for (String partition : partitions) {
692+
environment.deleteStream(partition);
693+
}
694+
Utils.deleteSuperStreamExchange(amqpChannel.get(), stream);
695+
696+
} else {
697+
LOGGER.debug("Deleting {}", stream);
698+
try {
699+
environment.deleteStream(stream);
700+
LOGGER.debug("Deleted {}", stream);
701+
} catch (Exception e) {
702+
LOGGER.warn("Could not delete stream {}: {}", stream, e.getMessage());
703+
}
667704
}
668705
}
669706
}));
707+
} else {
708+
if (this.superStreams) {
709+
// we don't want to delete the super streams at the end, so we close the AMQP connection
710+
amqpConnection.close();
711+
}
670712
}
671713

672714
List<Producer> producers = Collections.synchronizedList(new ArrayList<>(this.producers));
@@ -691,6 +733,16 @@ public Integer call() throws Exception {
691733
producerBuilder.name(producerName).confirmTimeout(Duration.ZERO);
692734
}
693735

736+
java.util.function.Consumer<MessageBuilder> messageBuilderConsumer;
737+
if (this.superStreams) {
738+
producerBuilder.routing(msg -> msg.getProperties().getMessageIdAsString());
739+
AtomicLong messageIdSequence = new AtomicLong(0);
740+
messageBuilderConsumer =
741+
mg -> mg.properties().messageId(messageIdSequence.getAndIncrement());
742+
} else {
743+
messageBuilderConsumer = mg -> {};
744+
}
745+
694746
Producer producer =
695747
producerBuilder
696748
.subEntrySize(this.subEntrySize)
@@ -753,9 +805,10 @@ public Integer call() throws Exception {
753805
long creationTime = System.currentTimeMillis();
754806
byte[] payload = new byte[msgSize];
755807
Utils.writeLong(payload, creationTime);
808+
MessageBuilder messageBuilder = producer.messageBuilder();
809+
messageBuilderConsumer.accept(messageBuilder);
756810
producer.send(
757-
producer.messageBuilder().addData(payload).build(),
758-
confirmationHandler);
811+
messageBuilder.addData(payload).build(), confirmationHandler);
759812
}
760813
} catch (Exception e) {
761814
if (e instanceof InterruptedException
@@ -780,7 +833,21 @@ public Integer call() throws Exception {
780833
AtomicLong messageCount = new AtomicLong(0);
781834
String stream = stream(streams, i);
782835
ConsumerBuilder consumerBuilder = environment.consumerBuilder();
783-
consumerBuilder = consumerBuilder.stream(stream).offset(this.offset);
836+
consumerBuilder = consumerBuilder.offset(this.offset);
837+
838+
if (this.superStreams) {
839+
consumerBuilder.superStream(stream);
840+
} else {
841+
consumerBuilder.stream(stream);
842+
}
843+
844+
if (this.singleActiveConsumer) {
845+
consumerBuilder.singleActiveConsumer();
846+
// single active consumer requires a name
847+
if (this.storeEvery == 0) {
848+
this.storeEvery = 10_000;
849+
}
850+
}
784851

785852
if (this.storeEvery > 0) {
786853
String consumerName = this.consumerNameStrategy.apply(stream, i + 1);
@@ -880,6 +947,36 @@ public Integer call() throws Exception {
880947
return 0;
881948
}
882949

950+
private void createStream(Environment environment, String stream) {
951+
StreamCreator streamCreator =
952+
environment.streamCreator().stream(stream)
953+
.maxLengthBytes(this.maxLengthBytes)
954+
.maxSegmentSizeBytes(this.maxSegmentSize)
955+
.leaderLocator(this.leaderLocator);
956+
957+
if (this.maxAge != null) {
958+
streamCreator.maxAge(this.maxAge);
959+
}
960+
961+
try {
962+
streamCreator.create();
963+
} catch (StreamException e) {
964+
if (e.getCode() == Constants.RESPONSE_CODE_PRECONDITION_FAILED) {
965+
String message =
966+
String.format(
967+
"Warning: stream '%s' already exists, but with different properties than "
968+
+ "max-length-bytes=%s, stream-max-segment-size-bytes=%s, queue-leader-locator=%s",
969+
stream, this.maxLengthBytes, this.maxSegmentSize, this.leaderLocator);
970+
if (this.maxAge != null) {
971+
message += String.format(", max-age=%s", this.maxAge);
972+
}
973+
this.out.println(message);
974+
} else {
975+
throw e;
976+
}
977+
}
978+
}
979+
883980
private void overridePropertiesWithEnvironmentVariables() throws Exception {
884981
Function<String, String> optionToEnvMappings =
885982
OPTION_TO_ENVIRONMENT_VARIABLE

0 commit comments

Comments
 (0)