Skip to content

Commit 9e16341

Browse files
committed
Add SAC test against cluster
References rabbitmq/rabbitmq-server#13672
1 parent 7fa8779 commit 9e16341

File tree

4 files changed

+234
-20
lines changed

4 files changed

+234
-20
lines changed

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

Lines changed: 3 additions & 3 deletions
Original file line numberDiff line numberDiff line change
@@ -28,8 +28,8 @@
2828

2929
class StreamProducerBuilder implements ProducerBuilder {
3030

31-
static final boolean DEFAULT_DYNAMIC_BATCH = true;
32-
// Boolean.parseBoolean(System.getProperty("rabbitmq.stream.producer.dynamic.batch", "true"));
31+
static final boolean DEFAULT_DYNAMIC_BATCH =
32+
Boolean.parseBoolean(System.getProperty("rabbitmq.stream.producer.dynamic.batch", "true"));
3333

3434
private final StreamEnvironment environment;
3535

@@ -201,7 +201,7 @@ public Producer build() {
201201

202202
if (this.routingConfiguration == null && this.superStream != null) {
203203
throw new IllegalArgumentException(
204-
"A routing configuration must specified when a super stream is set");
204+
"A routing configuration must be specified when a super stream is set");
205205
}
206206

207207
if (this.stream != null) {

src/test/java/com/rabbitmq/stream/Cli.java

Lines changed: 48 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -178,6 +178,30 @@ static List<ConnectionInfo> toConnectionInfoList(String json) {
178178
return GSON.fromJson(json, new TypeToken<List<ConnectionInfo>>() {}.getType());
179179
}
180180

181+
public static List<SubscriptionInfo> listGroupConsumers(String stream, String reference) {
182+
ProcessState process =
183+
rabbitmqStreams(
184+
format(
185+
"list_stream_group_consumers -q --stream %s --reference %s "
186+
+ "--formatter table subscription_id,state",
187+
stream, reference));
188+
189+
List<SubscriptionInfo> itemList = Collections.emptyList();
190+
String content = process.output();
191+
String[] lines = content.split(System.lineSeparator());
192+
if (lines.length > 1) {
193+
itemList = new ArrayList<>(lines.length - 1);
194+
for (int i = 1; i < lines.length; i++) {
195+
String line = lines[i];
196+
String[] fields = line.split("\t");
197+
String id = fields[0];
198+
String state = fields[1].replace("\"", "");
199+
itemList.add(new SubscriptionInfo(Integer.parseInt(id), state));
200+
}
201+
}
202+
return itemList;
203+
}
204+
181205
public static void restartStream(String stream) {
182206
rabbitmqStreams(" restart_stream " + stream);
183207
}
@@ -420,6 +444,30 @@ public String toString() {
420444
}
421445
}
422446

447+
public static final class SubscriptionInfo {
448+
449+
private final int id;
450+
private final String state;
451+
452+
public SubscriptionInfo(int id, String state) {
453+
this.id = id;
454+
this.state = state;
455+
}
456+
457+
public int id() {
458+
return this.id;
459+
}
460+
461+
public String state() {
462+
return this.state;
463+
}
464+
465+
@Override
466+
public String toString() {
467+
return "SubscriptionInfo{id='" + id + '\'' + ", state='" + state + '\'' + '}';
468+
}
469+
}
470+
423471
public static class ProcessState {
424472

425473
private final InputStreamPumpState inputState;

src/test/java/com/rabbitmq/stream/impl/RecoveryClusterTest.java

Lines changed: 181 additions & 16 deletions
Original file line numberDiff line numberDiff line change
@@ -16,18 +16,22 @@
1616

1717
import static com.rabbitmq.stream.impl.Assertions.assertThat;
1818
import static com.rabbitmq.stream.impl.LoadBalancerClusterTest.LOAD_BALANCER_ADDRESS;
19+
import static com.rabbitmq.stream.impl.TestUtils.BrokerVersion.RABBITMQ_4_1_2;
1920
import static com.rabbitmq.stream.impl.TestUtils.newLoggerLevel;
2021
import static com.rabbitmq.stream.impl.TestUtils.sync;
22+
import static com.rabbitmq.stream.impl.TestUtils.waitAtMost;
2123
import static com.rabbitmq.stream.impl.ThreadUtils.threadFactory;
2224
import static com.rabbitmq.stream.impl.Tuples.pair;
2325
import static java.util.stream.Collectors.toList;
2426
import static java.util.stream.IntStream.range;
2527
import static org.assertj.core.api.Assertions.assertThat;
28+
import static org.assertj.core.api.InstanceOfAssertFactories.stream;
2629

2730
import ch.qos.logback.classic.Level;
2831
import com.google.common.collect.Streams;
2932
import com.google.common.util.concurrent.RateLimiter;
3033
import com.rabbitmq.stream.*;
34+
import com.rabbitmq.stream.impl.TestUtils.BrokerVersionAtLeast;
3135
import com.rabbitmq.stream.impl.TestUtils.DisabledIfNotCluster;
3236
import com.rabbitmq.stream.impl.TestUtils.Sync;
3337
import com.rabbitmq.stream.impl.Tuples.Pair;
@@ -41,15 +45,20 @@
4145
import java.util.LinkedHashMap;
4246
import java.util.List;
4347
import java.util.Map;
48+
import java.util.concurrent.Callable;
49+
import java.util.concurrent.ConcurrentHashMap;
4450
import java.util.concurrent.Executors;
4551
import java.util.concurrent.ScheduledExecutorService;
4652
import java.util.concurrent.ThreadFactory;
4753
import java.util.concurrent.atomic.AtomicBoolean;
4854
import java.util.concurrent.atomic.AtomicInteger;
55+
import java.util.concurrent.atomic.AtomicLong;
4956
import java.util.concurrent.atomic.AtomicReference;
57+
import java.util.stream.IntStream;
5058
import org.junit.jupiter.api.*;
5159
import org.junit.jupiter.params.ParameterizedTest;
5260
import org.junit.jupiter.params.provider.CsvSource;
61+
import org.junit.jupiter.params.provider.ValueSource;
5362
import org.slf4j.Logger;
5463
import org.slf4j.LoggerFactory;
5564

@@ -201,15 +210,7 @@ void clusterRestart(boolean useLoadBalancer, boolean forceLeader) throws Interru
201210
syncs = consumers.stream().map(c -> c.waitForNewMessages(100)).collect(toList());
202211
syncs.forEach(s -> assertThat(s).completes());
203212

204-
nodes.forEach(
205-
n -> {
206-
LOGGER.info("Restarting node {}...", n);
207-
Cli.restartNode(n);
208-
LOGGER.info("Restarted node {}.", n);
209-
});
210-
LOGGER.info("Rebalancing...");
211-
Cli.rebalance();
212-
LOGGER.info("Rebalancing over.");
213+
restartCluster();
213214

214215
Thread.sleep(BACK_OFF_DELAY_POLICY.delay(0).toMillis());
215216

@@ -291,8 +292,132 @@ void clusterRestart(boolean useLoadBalancer, boolean forceLeader) throws Interru
291292
}
292293
}
293294

295+
@ParameterizedTest
296+
@ValueSource(booleans = {true, false})
297+
@BrokerVersionAtLeast(RABBITMQ_4_1_2)
298+
void sacWithClusterRestart(boolean superStream) throws Exception {
299+
environment =
300+
environmentBuilder
301+
.uris(URIS)
302+
.netty()
303+
.bootstrapCustomizer(
304+
b -> {
305+
b.option(
306+
ChannelOption.CONNECT_TIMEOUT_MILLIS,
307+
(int) BACK_OFF_DELAY_POLICY.delay(0).toMillis());
308+
})
309+
.environmentBuilder()
310+
.maxConsumersByConnection(1)
311+
.build();
312+
313+
int consumerCount = 3;
314+
AtomicLong lastOffset = new AtomicLong(0);
315+
String app = "app-name";
316+
String s = TestUtils.streamName(testInfo);
317+
ProducerState pState = null;
318+
List<ConsumerState> consumers = Collections.emptyList();
319+
try {
320+
StreamCreator sCreator = environment.streamCreator().stream(s);
321+
if (superStream) {
322+
sCreator = sCreator.superStream().partitions(1).creator();
323+
}
324+
sCreator.create();
325+
326+
pState = new ProducerState(s, true, superStream, environment);
327+
pState.start();
328+
329+
Map<Integer, Boolean> consumerStatus = new ConcurrentHashMap<>();
330+
consumers =
331+
IntStream.range(0, consumerCount)
332+
.mapToObj(
333+
i ->
334+
new ConsumerState(
335+
s,
336+
environment,
337+
b -> {
338+
b.singleActiveConsumer()
339+
.name(app)
340+
.noTrackingStrategy()
341+
.consumerUpdateListener(
342+
ctx -> {
343+
consumerStatus.put(i, ctx.isActive());
344+
return OffsetSpecification.offset(lastOffset.get());
345+
});
346+
if (superStream) {
347+
b.superStream(s);
348+
} else {
349+
b.stream(s);
350+
}
351+
},
352+
(ctx, m) -> lastOffset.set(ctx.offset())))
353+
.collect(toList());
354+
355+
Sync sync = pState.waitForNewMessages(100);
356+
assertThat(sync).completes();
357+
sync = consumers.get(0).waitForNewMessages(100);
358+
assertThat(sync).completes();
359+
360+
String streamArg = superStream ? s + "-0" : s;
361+
362+
Callable<Void> checkConsumers =
363+
() -> {
364+
waitAtMost(
365+
() -> {
366+
List<Cli.SubscriptionInfo> subscriptions = Cli.listGroupConsumers(streamArg, app);
367+
LOGGER.info("Group consumers: {}", subscriptions);
368+
return subscriptions.size() == consumerCount
369+
&& subscriptions.stream()
370+
.filter(sub -> sub.state().startsWith("active"))
371+
.count()
372+
== 1
373+
&& subscriptions.stream()
374+
.filter(sub -> sub.state().startsWith("waiting"))
375+
.count()
376+
== 2;
377+
},
378+
() ->
379+
"Group consumers not in expected state: "
380+
+ Cli.listGroupConsumers(streamArg, app));
381+
return null;
382+
};
383+
384+
checkConsumers.call();
385+
386+
restartCluster();
387+
388+
Thread.sleep(BACK_OFF_DELAY_POLICY.delay(0).toMillis());
389+
390+
sync = pState.waitForNewMessages(100);
391+
assertThat(sync).completes(ASSERTION_TIMEOUT);
392+
int activeIndex =
393+
consumerStatus.entrySet().stream()
394+
.filter(Map.Entry::getValue)
395+
.map(Map.Entry::getKey)
396+
.findFirst()
397+
.orElseThrow(() -> new IllegalStateException("No active consumer found"));
398+
399+
sync = consumers.get(activeIndex).waitForNewMessages(100);
400+
assertThat(sync).completes(ASSERTION_TIMEOUT);
401+
402+
checkConsumers.call();
403+
404+
} finally {
405+
if (pState != null) {
406+
pState.close();
407+
}
408+
consumers.forEach(ConsumerState::close);
409+
if (superStream) {
410+
environment.deleteSuperStream(s);
411+
} else {
412+
environment.deleteStream(s);
413+
}
414+
}
415+
}
416+
294417
private static class ProducerState implements AutoCloseable {
295418

419+
private static final AtomicLong MSG_ID_SEQ = new AtomicLong(0);
420+
296421
private static final byte[] BODY = "hello".getBytes(StandardCharsets.UTF_8);
297422

298423
private final String stream;
@@ -306,9 +431,19 @@ private static class ProducerState implements AutoCloseable {
306431
final AtomicReference<Instant> lastExceptionInstant = new AtomicReference<>();
307432

308433
private ProducerState(String stream, boolean dynamicBatch, Environment environment) {
434+
this(stream, dynamicBatch, false, environment);
435+
}
436+
437+
private ProducerState(
438+
String stream, boolean dynamicBatch, boolean superStream, Environment environment) {
309439
this.stream = stream;
310-
this.producer =
311-
environment.producerBuilder().stream(stream).dynamicBatch(dynamicBatch).build();
440+
ProducerBuilder builder = environment.producerBuilder().dynamicBatch(dynamicBatch);
441+
if (superStream) {
442+
builder.superStream(stream).routing(m -> m.getProperties().getMessageIdAsString());
443+
} else {
444+
builder.stream(stream);
445+
}
446+
this.producer = builder.build();
312447
}
313448

314449
void start() {
@@ -327,7 +462,14 @@ void start() {
327462
try {
328463
this.limiter.acquire(1);
329464
this.producer.send(
330-
producer.messageBuilder().addData(BODY).build(), confirmationHandler);
465+
producer
466+
.messageBuilder()
467+
.properties()
468+
.messageId(MSG_ID_SEQ.getAndIncrement())
469+
.messageBuilder()
470+
.addData(BODY)
471+
.build(),
472+
confirmationHandler);
331473
} catch (Throwable e) {
332474
this.lastException.set(e);
333475
this.lastExceptionInstant.set(Instant.now());
@@ -380,16 +522,27 @@ private static class ConsumerState implements AutoCloseable {
380522
final AtomicReference<Runnable> postHandle = new AtomicReference<>(() -> {});
381523

382524
private ConsumerState(String stream, Environment environment) {
525+
this(stream, environment, b -> b.stream(stream), (ctx, m) -> {});
526+
}
527+
528+
private ConsumerState(
529+
String stream,
530+
Environment environment,
531+
java.util.function.Consumer<ConsumerBuilder> customizer,
532+
MessageHandler delegateHandler) {
383533
this.stream = stream;
384-
this.consumer =
385-
environment.consumerBuilder().stream(stream)
534+
ConsumerBuilder builder =
535+
environment
536+
.consumerBuilder()
386537
.offset(OffsetSpecification.first())
387538
.messageHandler(
388539
(ctx, m) -> {
540+
delegateHandler.handle(ctx, m);
389541
receivedCount.incrementAndGet();
390542
postHandle.get().run();
391-
})
392-
.build();
543+
});
544+
customizer.accept(builder);
545+
this.consumer = builder.build();
393546
}
394547

395548
Sync waitForNewMessages(int messageCount) {
@@ -414,4 +567,16 @@ public void close() {
414567
this.consumer.close();
415568
}
416569
}
570+
571+
private static void restartCluster() {
572+
nodes.forEach(
573+
n -> {
574+
LOGGER.info("Restarting node {}...", n);
575+
Cli.restartNode(n);
576+
LOGGER.info("Restarted node {}.", n);
577+
});
578+
LOGGER.info("Rebalancing...");
579+
Cli.rebalance();
580+
LOGGER.info("Rebalancing over.");
581+
}
417582
}

src/test/java/com/rabbitmq/stream/impl/TestUtils.java

Lines changed: 2 additions & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -1064,7 +1064,8 @@ public enum BrokerVersion {
10641064
RABBITMQ_3_11_11("3.11.11"),
10651065
RABBITMQ_3_11_14("3.11.14"),
10661066
RABBITMQ_3_13_0("3.13.0"),
1067-
RABBITMQ_4_0_0("4.0.0");
1067+
RABBITMQ_4_0_0("4.0.0"),
1068+
RABBITMQ_4_1_2("4.1.2");
10681069

10691070
final String value;
10701071

0 commit comments

Comments
 (0)