2424import static org .mockito .Mockito .when ;
2525import static org .testng .Assert .assertEquals ;
2626import static org .testng .Assert .assertTrue ;
27+ import java .io .Closeable ;
2728import java .net .InetSocketAddress ;
29+ import java .time .Duration ;
2830import java .util .List ;
2931import java .util .UUID ;
3032import java .util .concurrent .CompletableFuture ;
3638import org .apache .pulsar .client .admin .PulsarAdminException ;
3739import org .apache .pulsar .client .api .ClientBuilder ;
3840import org .apache .pulsar .client .api .Consumer ;
41+ import org .apache .pulsar .client .api .MessageId ;
3942import org .apache .pulsar .client .api .Producer ;
4043import org .apache .pulsar .client .api .ProducerConsumerBase ;
44+ import org .apache .pulsar .client .api .PulsarClient ;
4145import org .apache .pulsar .client .api .PulsarClientException ;
4246import org .apache .pulsar .client .impl .LookupService ;
4347import org .apache .pulsar .client .impl .LookupTopicResult ;
4448import org .apache .pulsar .client .impl .PulsarClientImpl ;
4549import org .apache .pulsar .common .naming .NamespaceName ;
50+ import org .apache .pulsar .common .naming .TopicName ;
4651import org .apache .pulsar .common .partition .PartitionedTopicMetadata ;
4752import org .apache .pulsar .common .policies .data .AutoTopicCreationOverride ;
4853import org .apache .pulsar .common .policies .data .TopicType ;
49- import org .testng .annotations .AfterMethod ;
50- import org .testng .annotations .BeforeMethod ;
54+ import org .testng .annotations .AfterClass ;
55+ import org .testng .annotations .BeforeClass ;
5156import org .testng .annotations .Test ;
5257
5358@ Test (groups = "broker-admin" )
5459@ Slf4j
5560public class TopicAutoCreationTest extends ProducerConsumerBase {
5661
5762 @ Override
58- @ BeforeMethod
63+ @ BeforeClass
5964 protected void setup () throws Exception {
6065 conf .setAllowAutoTopicCreationType (TopicType .PARTITIONED );
6166 conf .setAllowAutoTopicCreation (true );
@@ -71,7 +76,7 @@ protected void customizeNewPulsarClientBuilder(ClientBuilder clientBuilder) {
7176 }
7277
7378 @ Override
74- @ AfterMethod (alwaysRun = true )
79+ @ AfterClass (alwaysRun = true )
7580 protected void cleanup () throws Exception {
7681 super .internalCleanup ();
7782 }
@@ -87,9 +92,11 @@ public void testPartitionedTopicAutoCreation() throws PulsarAdminException, Puls
8792 .create ();
8893
8994 List <String > partitionedTopics = admin .topics ().getPartitionedTopicList (namespaceName );
95+ assertTrue (partitionedTopics .contains (topic ));
9096 List <String > topics = admin .topics ().getList (namespaceName );
91- assertEquals (partitionedTopics .size (), 1 );
92- assertEquals (topics .size (), 3 );
97+ for (int i = 0 ; i < conf .getDefaultNumPartitions (); i ++) {
98+ assertTrue (topics .contains (topic + TopicName .PARTITIONED_TOPIC_SUFFIX + i ));
99+ }
93100
94101 producer .close ();
95102 for (String t : topics ) {
@@ -248,4 +255,48 @@ public void testClientWithAutoCreationGotNotFoundException() throws PulsarAdminE
248255 admin .namespaces ().deleteNamespace (namespace , true );
249256 }
250257
258+ @ Test
259+ public void testPartitionsNotCreatedAfterDeletion () throws Exception {
260+ @ Cleanup final var client = PulsarClient .builder ().serviceUrl (pulsar .getBrokerServiceUrl ()).build ();
261+ final var topicName = TopicName .get ("my-property/my-ns/testPartitionsNotCreatedAfterDeletion" );
262+ final var topic = topicName .toString ();
263+ final var interval = Duration .ofSeconds (1 );
264+ final ThrowableConsumer <ThrowableSupplier <Closeable >> verifier = creator -> {
265+ admin .topics ().createPartitionedTopic (topic , 1 );
266+ boolean needCleanup = false ;
267+ try (final var ignored = creator .get ()) {
268+ admin .topics ().terminatePartitionedTopic (topic );
269+ admin .topics ().deletePartitionedTopic (topic , true );
270+ Thread .sleep (interval .toMillis () + 500 ); // wait until the auto update partitions task has run
271+
272+ final var topics = admin .topics ().getList (topicName .getNamespace ()).stream ()
273+ .filter (__ -> __ .contains (topicName .getLocalName ())).toList ();
274+ // Without https://github.com/apache/pulsar/pull/24118, the producer or consumer on partition 0 could be
275+ // automatically created.
276+ if (!topics .isEmpty ()) {
277+ assertEquals (topics , List .of (topicName .getPartition (0 ).toString ()));
278+ needCleanup = true ;
279+ }
280+ }
281+ if (needCleanup ) {
282+ admin .topics ().delete (topicName .getPartition (0 ).toString ());
283+ }
284+ };
285+ verifier .accept (() -> client .newProducer ().topic (topic )
286+ .autoUpdatePartitionsInterval (interval .toSecondsPart (), TimeUnit .SECONDS ).create ());
287+ verifier .accept (() -> client .newConsumer ().topic (topic ).subscriptionName ("sub" )
288+ .autoUpdatePartitionsInterval (interval .toSecondsPart (), TimeUnit .SECONDS ).subscribe ());
289+ verifier .accept (() -> client .newReader ().topic (topic ).startMessageId (MessageId .earliest )
290+ .autoUpdatePartitionsInterval (interval .toSecondsPart (), TimeUnit .SECONDS ).create ());
291+ }
292+
293+ private interface ThrowableConsumer <T > {
294+
295+ void accept (T value ) throws Exception ;
296+ }
297+
298+ public interface ThrowableSupplier <T > {
299+
300+ T get () throws Exception ;
301+ }
251302}
0 commit comments