diff --git a/.gitignore b/.gitignore index cd00c44200059..80d760cd29df7 100644 --- a/.gitignore +++ b/.gitignore @@ -97,3 +97,5 @@ test-reports/ # Gradle Enterprise .mvn/.gradle-enterprise/ +# Gradle Develocity +.mvn/.develocity/ diff --git a/.mvn/gradle-enterprise-custom-user-data.groovy b/.mvn/develocity-custom-user-data.groovy similarity index 100% rename from .mvn/gradle-enterprise-custom-user-data.groovy rename to .mvn/develocity-custom-user-data.groovy diff --git a/.mvn/gradle-enterprise.xml b/.mvn/develocity.xml similarity index 66% rename from .mvn/gradle-enterprise.xml rename to .mvn/develocity.xml index 2667402c23cdb..5c0fbb47c7217 100644 --- a/.mvn/gradle-enterprise.xml +++ b/.mvn/develocity.xml @@ -19,22 +19,21 @@ under the License. --> - + + + #{(env['GRADLE_ENTERPRISE_ACCESS_KEY']?.trim() > '' or env['DEVELOCITY_ACCESS_KEY']?.trim() > '') and !(env['GITHUB_HEAD_REF']?.matches('(?i).*(experiment|wip|private).*') or env['GITHUB_REPOSITORY']?.matches('(?i).*(experiment|wip|private).*'))} https://ge.apache.org false - true true true #{isFalse(env['GITHUB_ACTIONS'])} - ALWAYS - true #{{'0.0.0.0'}} @@ -47,4 +46,4 @@ false - + \ No newline at end of file diff --git a/.mvn/extensions.xml b/.mvn/extensions.xml index 872764f899827..4a2117925f163 100644 --- a/.mvn/extensions.xml +++ b/.mvn/extensions.xml @@ -23,12 +23,12 @@ xsi:schemaLocation="http://maven.apache.org/EXTENSIONS/1.0.0 http://maven.apache.org/xsd/core-extensions-1.0.0.xsd"> com.gradle - gradle-enterprise-maven-extension - 1.17.1 + develocity-maven-extension + 1.21.4 com.gradle common-custom-user-data-maven-extension - 1.11.1 + 2.0 diff --git a/bin/pulsar-daemon b/bin/pulsar-daemon index 210162b6a2190..2c05cb5c49dab 100755 --- a/bin/pulsar-daemon +++ b/bin/pulsar-daemon @@ -157,7 +157,7 @@ start () echo starting $command, logging to $logfile echo Note: Set immediateFlush to true in conf/log4j2.yaml will guarantee the logging event is flushing to disk immediately. The default behavior is switched off due to performance considerations. pulsar=$PULSAR_HOME/bin/pulsar - nohup $pulsar $command "$1" > "$out" 2>&1 < /dev/null & + nohup $pulsar $command "$@" > "$out" 2>&1 < /dev/null & echo $! > $pid sleep 1; head $out sleep 2; @@ -216,7 +216,7 @@ stop () case $startStop in (start) - start "$*" + start "$@" ;; (stop) @@ -224,21 +224,20 @@ case $startStop in ;; (restart) - forceStopFlag=$(echo "$*"|grep "\-force") - if [[ "$forceStopFlag" != "" ]] + if [[ "$1" == "-force" ]] then - stop "-force" + stop -force + # remove "-force" from the arguments + shift else stop fi if [ "$?" == 0 ] then - sleep 3 - paramaters="$*" - startParamaters=${paramaters//-force/} - start "$startParamaters" + sleep 3 + start "$@" else - echo "WARNNING : $command failed restart, for $command is not stopped completely." + echo "WARNNING : $command failed restart, for $command is not stopped completely." fi ;; diff --git a/distribution/server/src/assemble/LICENSE.bin.txt b/distribution/server/src/assemble/LICENSE.bin.txt index 0374b97b6836f..1e7a6ba455ccf 100644 --- a/distribution/server/src/assemble/LICENSE.bin.txt +++ b/distribution/server/src/assemble/LICENSE.bin.txt @@ -306,6 +306,7 @@ The Apache Software License, Version 2.0 - io.netty-netty-resolver-dns-native-macos-4.1.108.Final-osx-x86_64.jar - io.netty-netty-transport-4.1.108.Final.jar - io.netty-netty-transport-classes-epoll-4.1.108.Final.jar + - io.netty-netty-transport-native-epoll-4.1.108.Final-linux-aarch_64.jar - io.netty-netty-transport-native-epoll-4.1.108.Final-linux-x86_64.jar - io.netty-netty-transport-native-unix-common-4.1.108.Final.jar - io.netty-netty-transport-native-unix-common-4.1.108.Final-linux-x86_64.jar @@ -500,11 +501,11 @@ The Apache Software License, Version 2.0 - io.etcd-jetcd-core-0.7.7.jar - io.etcd-jetcd-grpc-0.7.7.jar * IPAddress - - com.github.seancfoley-ipaddress-5.3.3.jar + - com.github.seancfoley-ipaddress-5.5.0.jar * RxJava - io.reactivex.rxjava3-rxjava-3.0.1.jar * RoaringBitmap - - org.roaringbitmap-RoaringBitmap-0.9.44.jar + - org.roaringbitmap-RoaringBitmap-1.1.0.jar BSD 3-clause "New" or "Revised" License * Google auth library @@ -537,7 +538,6 @@ Protocol Buffers License CDDL-1.1 -- ../licenses/LICENSE-CDDL-1.1.txt * Java Annotations API - - javax.annotation-javax.annotation-api-1.3.2.jar - com.sun.activation-javax.activation-1.2.0.jar - javax.xml.bind-jaxb-api-2.3.1.jar * Java Servlet API -- javax.servlet-javax.servlet-api-3.1.0.jar diff --git a/distribution/server/src/assemble/bin.xml b/distribution/server/src/assemble/bin.xml index aafb559d67fb2..ab3199079b9eb 100644 --- a/distribution/server/src/assemble/bin.xml +++ b/distribution/server/src/assemble/bin.xml @@ -126,7 +126,7 @@ lib false - compile + runtime false @@ -134,13 +134,15 @@ com.datastax.oss:pulsar-functions-runtime-all - - org.projectlombok:lombok - com.datastax.oss:pulsar-functions-api-examples *:tar.gz + + org.codehaus.mojo:animal-sniffer-annotations + com.google.android:annotations + + net.java.dev.jna:jna diff --git a/distribution/shell/src/assemble/LICENSE.bin.txt b/distribution/shell/src/assemble/LICENSE.bin.txt index ff69efeda4644..c6d6566bb9502 100644 --- a/distribution/shell/src/assemble/LICENSE.bin.txt +++ b/distribution/shell/src/assemble/LICENSE.bin.txt @@ -357,6 +357,7 @@ The Apache Software License, Version 2.0 - netty-resolver-dns-4.1.108.Final.jar - netty-transport-4.1.108.Final.jar - netty-transport-classes-epoll-4.1.108.Final.jar + - netty-transport-native-epoll-4.1.108.Final-linux-aarch_64.jar - netty-transport-native-epoll-4.1.108.Final-linux-x86_64.jar - netty-transport-native-unix-common-4.1.108.Final.jar - netty-transport-native-unix-common-4.1.108.Final-linux-x86_64.jar @@ -379,6 +380,8 @@ The Apache Software License, Version 2.0 - simpleclient_tracer_common-0.16.0.jar - simpleclient_tracer_otel-0.16.0.jar - simpleclient_tracer_otel_agent-0.16.0.jar + * RoaringBitmap + - RoaringBitmap-1.1.0.jar * Log4J - log4j-api-2.18.0.jar - log4j-core-2.18.0.jar diff --git a/managed-ledger/src/main/java/org/apache/bookkeeper/mledger/impl/ManagedCursorImpl.java b/managed-ledger/src/main/java/org/apache/bookkeeper/mledger/impl/ManagedCursorImpl.java index 52d4e803f0bb9..4391fa0cb5fd9 100644 --- a/managed-ledger/src/main/java/org/apache/bookkeeper/mledger/impl/ManagedCursorImpl.java +++ b/managed-ledger/src/main/java/org/apache/bookkeeper/mledger/impl/ManagedCursorImpl.java @@ -1112,6 +1112,13 @@ public long getEstimatedSizeSinceMarkDeletePosition() { return ledger.estimateBacklogFromPosition(markDeletePosition); } + private long getNumberOfEntriesInBacklog() { + if (markDeletePosition.compareTo(ledger.getLastPosition()) >= 0) { + return 0; + } + return getNumberOfEntries(Range.openClosed(markDeletePosition, ledger.getLastPosition())); + } + @Override public long getNumberOfEntriesInBacklog(boolean isPrecise) { if (log.isDebugEnabled()) { @@ -1120,16 +1127,13 @@ public long getNumberOfEntriesInBacklog(boolean isPrecise) { messagesConsumedCounter, markDeletePosition, readPosition); } if (isPrecise) { - if (markDeletePosition.compareTo(ledger.getLastPosition()) >= 0) { - return 0; - } - return getNumberOfEntries(Range.openClosed(markDeletePosition, ledger.getLastPosition())); + return getNumberOfEntriesInBacklog(); } long backlog = ManagedLedgerImpl.ENTRIES_ADDED_COUNTER_UPDATER.get(ledger) - messagesConsumedCounter; if (backlog < 0) { // In some case the counters get incorrect values, fall back to the precise backlog count - backlog = getNumberOfEntries(Range.openClosed(markDeletePosition, ledger.getLastPosition())); + backlog = getNumberOfEntriesInBacklog(); } return backlog; diff --git a/managed-ledger/src/main/java/org/apache/bookkeeper/mledger/impl/ManagedLedgerFactoryImpl.java b/managed-ledger/src/main/java/org/apache/bookkeeper/mledger/impl/ManagedLedgerFactoryImpl.java index 805958531403f..49d0cf847d7c2 100644 --- a/managed-ledger/src/main/java/org/apache/bookkeeper/mledger/impl/ManagedLedgerFactoryImpl.java +++ b/managed-ledger/src/main/java/org/apache/bookkeeper/mledger/impl/ManagedLedgerFactoryImpl.java @@ -36,6 +36,7 @@ import java.util.concurrent.ConcurrentHashMap; import java.util.concurrent.ConcurrentSkipListMap; import java.util.concurrent.CountDownLatch; +import java.util.concurrent.ExecutionException; import java.util.concurrent.Executors; import java.util.concurrent.ScheduledExecutorService; import java.util.concurrent.ScheduledFuture; @@ -535,13 +536,12 @@ public CompletableFuture shutdownAsync() throws ManagedLedgerException { int numLedgers = ledgerNames.size(); log.info("Closing {} ledgers", numLedgers); for (String ledgerName : ledgerNames) { - CompletableFuture future = new CompletableFuture<>(); - futures.add(future); CompletableFuture ledgerFuture = ledgers.remove(ledgerName); if (ledgerFuture == null) { - future.complete(null); continue; } + CompletableFuture future = new CompletableFuture<>(); + futures.add(future); ledgerFuture.whenCompleteAsync((managedLedger, throwable) -> { if (throwable != null || managedLedger == null) { future.complete(null); @@ -606,68 +606,20 @@ public void closeFailed(ManagedLedgerException exception, Object ctx) { })); } })); - entryCacheManager.clear(); - return FutureUtil.waitForAll(futures).thenAccept(__ -> { + return FutureUtil.waitForAll(futures).thenAcceptAsync(__ -> { //wait for tasks in scheduledExecutor executed. - scheduledExecutor.shutdown(); + scheduledExecutor.shutdownNow(); + entryCacheManager.clear(); }); } @Override public void shutdown() throws InterruptedException, ManagedLedgerException { - if (closed) { - throw new ManagedLedgerException.ManagedLedgerFactoryClosedException(); + try { + shutdownAsync().get(); + } catch (ExecutionException e) { + throw getManagedLedgerException(e.getCause()); } - closed = true; - - statsTask.cancel(true); - flushCursorsTask.cancel(true); - cacheEvictionExecutor.shutdownNow(); - - // take a snapshot of ledgers currently in the map to prevent race conditions - List> ledgers = new ArrayList<>(this.ledgers.values()); - int numLedgers = ledgers.size(); - final CountDownLatch latch = new CountDownLatch(numLedgers); - log.info("Closing {} ledgers", numLedgers); - - for (CompletableFuture ledgerFuture : ledgers) { - ManagedLedgerImpl ledger = ledgerFuture.getNow(null); - if (ledger == null) { - latch.countDown(); - continue; - } - - ledger.asyncClose(new AsyncCallbacks.CloseCallback() { - @Override - public void closeComplete(Object ctx) { - latch.countDown(); - } - - @Override - public void closeFailed(ManagedLedgerException exception, Object ctx) { - log.warn("[{}] Got exception when closing managed ledger: {}", ledger.getName(), exception); - latch.countDown(); - } - }, null); - } - - latch.await(); - log.info("{} ledgers closed", numLedgers); - - if (isBookkeeperManaged) { - try { - BookKeeper bookkeeper = bookkeeperFactory.get(); - if (bookkeeper != null) { - bookkeeper.close(); - } - } catch (BKException e) { - throw new ManagedLedgerException(e); - } - } - - scheduledExecutor.shutdownNow(); - - entryCacheManager.clear(); } @Override diff --git a/managed-ledger/src/test/java/org/apache/bookkeeper/mledger/impl/ManagedLedgerTest.java b/managed-ledger/src/test/java/org/apache/bookkeeper/mledger/impl/ManagedLedgerTest.java index 797dbe88a0196..fc47b8c3f72a4 100644 --- a/managed-ledger/src/test/java/org/apache/bookkeeper/mledger/impl/ManagedLedgerTest.java +++ b/managed-ledger/src/test/java/org/apache/bookkeeper/mledger/impl/ManagedLedgerTest.java @@ -200,10 +200,10 @@ private DeleteLedgerInfo makeDelayIfDoLedgerDelete(LedgerHandle ledger, final At bkc.asyncDeleteLedger(ledgerId, originalCb, ctx); } else { deleteLedgerInfo.hasCalled = true; - new Thread(() -> { + cachedExecutor.submit(() -> { Awaitility.await().atMost(Duration.ofSeconds(60)).until(signal::get); bkc.asyncDeleteLedger(ledgerId, cb, ctx); - }).start(); + }); } return null; }).when(spyBookKeeper).asyncDeleteLedger(any(long.class), any(AsyncCallback.DeleteCallback.class), any()); @@ -220,6 +220,7 @@ private DeleteLedgerInfo makeDelayIfDoLedgerDelete(LedgerHandle ledger, final At public void testLedgerInfoMetaCorrectIfAddEntryTimeOut() throws Exception { String mlName = "testLedgerInfoMetaCorrectIfAddEntryTimeOut"; BookKeeper spyBookKeeper = spy(bkc); + @Cleanup("shutdown") ManagedLedgerFactoryImpl factory = new ManagedLedgerFactoryImpl(metadataStore, spyBookKeeper); ManagedLedgerImpl ml = (ManagedLedgerImpl) factory.open(mlName); @@ -3938,6 +3939,7 @@ public void testCancellationOfScheduledTasks() throws Exception { public void testInactiveLedgerRollOver() throws Exception { int inactiveLedgerRollOverTimeMs = 5; ManagedLedgerFactoryConfig factoryConf = new ManagedLedgerFactoryConfig(); + @Cleanup("shutdown") ManagedLedgerFactory factory = new ManagedLedgerFactoryImpl(metadataStore, bkc); ManagedLedgerConfig config = new ManagedLedgerConfig(); config.setInactiveLedgerRollOverTime(inactiveLedgerRollOverTimeMs, TimeUnit.MILLISECONDS); @@ -3969,7 +3971,6 @@ public void testInactiveLedgerRollOver() throws Exception { List ledgers = ledger.getLedgersInfoAsList(); assertEquals(ledgers.size(), totalAddEntries); ledger.close(); - factory.shutdown(); } @Test @@ -4022,6 +4023,7 @@ public void testDontRollOverInactiveLedgersWhenMetadataServiceInvalid() throws E @Test public void testOffloadTaskCancelled() throws Exception { + @Cleanup("shutdown") ManagedLedgerFactory factory = new ManagedLedgerFactoryImpl(metadataStore, bkc); ManagedLedgerConfig config = new ManagedLedgerConfig(); config.setMaxEntriesPerLedger(2); diff --git a/managed-ledger/src/test/java/org/apache/bookkeeper/test/BookKeeperClusterTestCase.java b/managed-ledger/src/test/java/org/apache/bookkeeper/test/BookKeeperClusterTestCase.java index e316083e837fd..a323ecfeb8ea6 100644 --- a/managed-ledger/src/test/java/org/apache/bookkeeper/test/BookKeeperClusterTestCase.java +++ b/managed-ledger/src/test/java/org/apache/bookkeeper/test/BookKeeperClusterTestCase.java @@ -242,7 +242,9 @@ protected void startZKCluster() throws Exception { zkc = zkUtil.getZooKeeperClient(); metadataStore = new FaultInjectionMetadataStore( MetadataStoreExtended.create(zkUtil.getZooKeeperConnectString(), - MetadataStoreConfig.builder().build())); + MetadataStoreConfig.builder() + .metadataStoreName("metastore-" + getClass().getSimpleName()) + .build())); } /** diff --git a/managed-ledger/src/test/java/org/apache/bookkeeper/test/MockedBookKeeperTestCase.java b/managed-ledger/src/test/java/org/apache/bookkeeper/test/MockedBookKeeperTestCase.java index e2101268b09e6..645563eb78c4d 100644 --- a/managed-ledger/src/test/java/org/apache/bookkeeper/test/MockedBookKeeperTestCase.java +++ b/managed-ledger/src/test/java/org/apache/bookkeeper/test/MockedBookKeeperTestCase.java @@ -26,6 +26,7 @@ import lombok.SneakyThrows; import org.apache.bookkeeper.client.PulsarMockBookKeeper; import org.apache.bookkeeper.common.util.OrderedScheduler; +import org.apache.bookkeeper.mledger.ManagedLedgerException; import org.apache.bookkeeper.mledger.ManagedLedgerFactoryConfig; import org.apache.bookkeeper.mledger.impl.ManagedLedgerFactoryImpl; import org.apache.pulsar.metadata.api.MetadataStoreConfig; @@ -70,7 +71,8 @@ public MockedBookKeeperTestCase(int numBookies) { public final void setUp(Method method) throws Exception { LOG.info(">>>>>> starting {}", method); metadataStore = new FaultInjectionMetadataStore( - MetadataStoreExtended.create("memory:local", MetadataStoreConfig.builder().build())); + MetadataStoreExtended.create("memory:local", + MetadataStoreConfig.builder().metadataStoreName("metastore-" + method.getName()).build())); try { // start bookkeeper service @@ -102,7 +104,11 @@ public final void tearDown(Method method) { } try { LOG.info("@@@@@@@@@ stopping " + method); - factory.shutdownAsync().get(10, TimeUnit.SECONDS); + try { + factory.shutdownAsync().get(10, TimeUnit.SECONDS); + } catch (ManagedLedgerException.ManagedLedgerFactoryClosedException e) { + // ignore + } factory = null; stopBookKeeper(); metadataStore.close(); diff --git a/pom.xml b/pom.xml index 5aea0b7f98c53..237fecf85dee9 100644 --- a/pom.xml +++ b/pom.xml @@ -251,7 +251,7 @@ flexible messaging model and an intuitive client API. 0.7.7 2.0 1.10.12 - 5.3.3 + 5.5.0 3.4.3 1.5.2-3 2.0.6 @@ -306,7 +306,7 @@ flexible messaging model and an intuitive client API. 1.3 0.4 9.1.0 - 0.9.44 + 1.1.0 1.6.1 6.4.0 diff --git a/pulsar-broker-common/src/main/java/org/apache/pulsar/bookie/rackawareness/BookieRackAffinityMapping.java b/pulsar-broker-common/src/main/java/org/apache/pulsar/bookie/rackawareness/BookieRackAffinityMapping.java index 983822f22941b..4a5ff746f4039 100644 --- a/pulsar-broker-common/src/main/java/org/apache/pulsar/bookie/rackawareness/BookieRackAffinityMapping.java +++ b/pulsar-broker-common/src/main/java/org/apache/pulsar/bookie/rackawareness/BookieRackAffinityMapping.java @@ -70,7 +70,7 @@ public class BookieRackAffinityMapping extends AbstractDNSToSwitchMapping private BookiesRackConfiguration racksWithHost = new BookiesRackConfiguration(); private Map bookieInfoMap = new HashMap<>(); - public static MetadataStore createMetadataStore(Configuration conf) throws MetadataException { + static MetadataStore getMetadataStore(Configuration conf) throws MetadataException { MetadataStore store; Object storeProperty = conf.getProperty(METADATA_STORE_INSTANCE); if (storeProperty != null) { @@ -116,12 +116,20 @@ public synchronized void setConf(Configuration conf) { super.setConf(conf); MetadataStore store; try { - store = createMetadataStore(conf); - bookieMappingCache = store.getMetadataCache(BookiesRackConfiguration.class); - store.registerListener(this::handleUpdates); - racksWithHost = bookieMappingCache.get(BOOKIE_INFO_ROOT_PATH).get() - .orElseGet(BookiesRackConfiguration::new); - for (Map bookieMapping : racksWithHost.values()) { + store = getMetadataStore(conf); + } catch (MetadataException e) { + throw new RuntimeException(METADATA_STORE_INSTANCE + " failed to init BookieId list"); + } + + bookieMappingCache = store.getMetadataCache(BookiesRackConfiguration.class); + store.registerListener(this::handleUpdates); + + try { + var racksWithHost = bookieMappingCache.get(BOOKIE_INFO_ROOT_PATH) + .thenApply(optRes -> optRes.orElseGet(BookiesRackConfiguration::new)) + .get(); + + for (var bookieMapping : racksWithHost.values()) { for (String address : bookieMapping.keySet()) { bookieAddressListLastTime.add(BookieId.parse(address)); } @@ -131,10 +139,12 @@ public synchronized void setConf(Configuration conf) { } } updateRacksWithHost(racksWithHost); - watchAvailableBookies(); - } catch (InterruptedException | ExecutionException | MetadataException e) { - throw new RuntimeException(METADATA_STORE_INSTANCE + " failed to init BookieId list"); + } catch (ExecutionException | InterruptedException e) { + LOG.error("Failed to update rack info. ", e); + throw new RuntimeException(e); } + + watchAvailableBookies(); } private void watchAvailableBookies() { @@ -145,13 +155,13 @@ private void watchAvailableBookies() { field.setAccessible(true); RegistrationClient registrationClient = (RegistrationClient) field.get(bookieAddressResolver); registrationClient.watchWritableBookies(versioned -> { - try { - racksWithHost = bookieMappingCache.get(BOOKIE_INFO_ROOT_PATH).get() - .orElseGet(BookiesRackConfiguration::new); - updateRacksWithHost(racksWithHost); - } catch (InterruptedException | ExecutionException e) { - LOG.error("Failed to update rack info. ", e); - } + bookieMappingCache.get(BOOKIE_INFO_ROOT_PATH) + .thenApply(optRes -> optRes.orElseGet(BookiesRackConfiguration::new)) + .thenAccept(this::updateRacksWithHost) + .exceptionally(ex -> { + LOG.error("Failed to update rack info. ", ex); + return null; + }); }); } catch (NoSuchFieldException | IllegalAccessException e) { LOG.error("Failed watch available bookies.", e); diff --git a/pulsar-broker-common/src/main/java/org/apache/pulsar/bookie/rackawareness/IsolatedBookieEnsemblePlacementPolicy.java b/pulsar-broker-common/src/main/java/org/apache/pulsar/bookie/rackawareness/IsolatedBookieEnsemblePlacementPolicy.java index 02ddea9487469..8b1ef16cb6dae 100644 --- a/pulsar-broker-common/src/main/java/org/apache/pulsar/bookie/rackawareness/IsolatedBookieEnsemblePlacementPolicy.java +++ b/pulsar-broker-common/src/main/java/org/apache/pulsar/bookie/rackawareness/IsolatedBookieEnsemblePlacementPolicy.java @@ -73,7 +73,7 @@ public RackawareEnsemblePlacementPolicyImpl initialize(ClientConfiguration conf, StatsLogger statsLogger, BookieAddressResolver bookieAddressResolver) { MetadataStore store; try { - store = BookieRackAffinityMapping.createMetadataStore(conf); + store = BookieRackAffinityMapping.getMetadataStore(conf); } catch (MetadataException e) { throw new RuntimeException(METADATA_STORE_INSTANCE + " failed initialized"); } diff --git a/pulsar-broker/src/main/java/org/apache/pulsar/PulsarStandalone.java b/pulsar-broker/src/main/java/org/apache/pulsar/PulsarStandalone.java index ba136e7c91058..360f8caeba6ec 100644 --- a/pulsar-broker/src/main/java/org/apache/pulsar/PulsarStandalone.java +++ b/pulsar-broker/src/main/java/org/apache/pulsar/PulsarStandalone.java @@ -18,6 +18,7 @@ */ package org.apache.pulsar; +import static org.apache.commons.io.FileUtils.cleanDirectory; import static org.apache.pulsar.common.naming.NamespaceName.SYSTEM_NAMESPACE; import static org.apache.pulsar.common.naming.SystemTopicNames.TRANSACTION_COORDINATOR_ASSIGN; import com.beust.jcommander.Parameter; @@ -25,6 +26,7 @@ import com.google.common.collect.Sets; import io.netty.util.internal.PlatformDependent; import java.io.File; +import java.nio.file.Files; import java.nio.file.Path; import java.nio.file.Paths; import java.util.List; @@ -417,18 +419,22 @@ public void close() { try { if (fnWorkerService != null) { fnWorkerService.stop(); + fnWorkerService = null; } if (broker != null) { broker.close(); + broker = null; } if (bkCluster != null) { bkCluster.close(); + bkCluster = null; } if (bkEnsemble != null) { bkEnsemble.stop(); + bkEnsemble = null; } } catch (Exception e) { log.error("Shutdown failed: {}", e.getMessage(), e); @@ -439,7 +445,12 @@ public void close() { void startBookieWithMetadataStore() throws Exception { if (StringUtils.isBlank(metadataStoreUrl)){ log.info("Starting BK with RocksDb metadata store"); - metadataStoreUrl = "rocksdb://" + Paths.get(metadataDir).toAbsolutePath(); + Path metadataDirPath = Paths.get(metadataDir); + metadataStoreUrl = "rocksdb://" + metadataDirPath.toAbsolutePath(); + if (wipeData && Files.exists(metadataDirPath)) { + log.info("Wiping RocksDb metadata store at {}", metadataStoreUrl); + cleanDirectory(metadataDirPath.toFile()); + } } else { log.info("Starting BK with metadata store: {}", metadataStoreUrl); } @@ -493,5 +504,11 @@ private static void processTerminator(int exitCode) { ShutdownUtil.triggerImmediateForcefulShutdown(exitCode); } + public String getBrokerServiceUrl() { + return broker.getBrokerServiceUrl(); + } + public String getWebServiceUrl() { + return broker.getWebServiceAddress(); + } } diff --git a/pulsar-broker/src/main/java/org/apache/pulsar/PulsarStandaloneStarter.java b/pulsar-broker/src/main/java/org/apache/pulsar/PulsarStandaloneStarter.java index 25320964fd62b..63ee992de91ca 100644 --- a/pulsar-broker/src/main/java/org/apache/pulsar/PulsarStandaloneStarter.java +++ b/pulsar-broker/src/main/java/org/apache/pulsar/PulsarStandaloneStarter.java @@ -21,9 +21,12 @@ import static org.apache.commons.lang3.StringUtils.isBlank; import com.beust.jcommander.JCommander; import com.beust.jcommander.Parameter; +import com.google.common.annotations.VisibleForTesting; import com.google.common.base.Strings; import java.io.FileInputStream; import java.util.Arrays; +import lombok.AccessLevel; +import lombok.Setter; import lombok.extern.slf4j.Slf4j; import org.apache.commons.lang3.StringUtils; import org.apache.logging.log4j.LogManager; @@ -38,6 +41,9 @@ public class PulsarStandaloneStarter extends PulsarStandalone { @Parameter(names = {"-g", "--generate-docs"}, description = "Generate docs") private boolean generateDocs = false; + private Thread shutdownThread; + @Setter(AccessLevel.PACKAGE) + private boolean testMode; public PulsarStandaloneStarter(String[] args) throws Exception { @@ -108,26 +114,50 @@ public PulsarStandaloneStarter(String[] args) throws Exception { } } } + } - Runtime.getRuntime().addShutdownHook(new Thread(() -> { + @Override + public synchronized void start() throws Exception { + super.start(); + if (shutdownThread != null) { + throw new IllegalStateException("Shutdown hook already registered"); + } + shutdownThread = new Thread(() -> { try { - if (fnWorkerService != null) { - fnWorkerService.stop(); - } - - if (broker != null) { - broker.close(); - } - - if (bkEnsemble != null) { - bkEnsemble.stop(); - } + doClose(false); } catch (Exception e) { log.error("Shutdown failed: {}", e.getMessage(), e); } finally { - LogManager.shutdown(); + if (!testMode) { + LogManager.shutdown(); + } } - })); + }); + Runtime.getRuntime().addShutdownHook(shutdownThread); + } + + // simulate running the shutdown hook, for testing + @VisibleForTesting + void runShutdownHook() { + if (!testMode) { + throw new IllegalStateException("Not in test mode"); + } + Runtime.getRuntime().removeShutdownHook(shutdownThread); + shutdownThread.run(); + shutdownThread = null; + } + + @Override + public void close() { + doClose(true); + } + + private synchronized void doClose(boolean removeShutdownHook) { + super.close(); + if (shutdownThread != null && removeShutdownHook) { + Runtime.getRuntime().removeShutdownHook(shutdownThread); + shutdownThread = null; + } } private static boolean argsContains(String[] args, String arg) { diff --git a/pulsar-broker/src/main/java/org/apache/pulsar/broker/PulsarService.java b/pulsar-broker/src/main/java/org/apache/pulsar/broker/PulsarService.java index 6d3cd98738762..58018be20b642 100644 --- a/pulsar-broker/src/main/java/org/apache/pulsar/broker/PulsarService.java +++ b/pulsar-broker/src/main/java/org/apache/pulsar/broker/PulsarService.java @@ -1940,6 +1940,11 @@ protected BrokerService newBrokerService(PulsarService pulsar) throws Exception return new BrokerService(pulsar, ioEventLoopGroup); } + @VisibleForTesting + public void setTransactionExecutorProvider(TransactionBufferProvider transactionBufferProvider) { + this.transactionBufferProvider = transactionBufferProvider; + } + public void initConfigMetadataSynchronizerIfNeeded() { mutex.lock(); try { diff --git a/pulsar-broker/src/main/java/org/apache/pulsar/broker/admin/impl/BrokersBase.java b/pulsar-broker/src/main/java/org/apache/pulsar/broker/admin/impl/BrokersBase.java index eeb65590bec8a..9289cb3e039f9 100644 --- a/pulsar-broker/src/main/java/org/apache/pulsar/broker/admin/impl/BrokersBase.java +++ b/pulsar-broker/src/main/java/org/apache/pulsar/broker/admin/impl/BrokersBase.java @@ -88,7 +88,7 @@ public class BrokersBase extends AdminResource { private static final Duration HEALTH_CHECK_READ_TIMEOUT = Duration.ofSeconds(58); private static final TimeoutException HEALTH_CHECK_TIMEOUT_EXCEPTION = FutureUtil.createTimeoutException("Timeout", BrokersBase.class, "healthCheckRecursiveReadNext(...)"); - private volatile long threadDumpLoggedTimestamp; + private static volatile long threadDumpLoggedTimestamp; @GET @Path("/{cluster}") diff --git a/pulsar-broker/src/main/java/org/apache/pulsar/broker/admin/impl/PersistentTopicsBase.java b/pulsar-broker/src/main/java/org/apache/pulsar/broker/admin/impl/PersistentTopicsBase.java index f20897634d685..2a6b89b413371 100644 --- a/pulsar-broker/src/main/java/org/apache/pulsar/broker/admin/impl/PersistentTopicsBase.java +++ b/pulsar-broker/src/main/java/org/apache/pulsar/broker/admin/impl/PersistentTopicsBase.java @@ -557,19 +557,29 @@ protected CompletableFuture internalGetPartitionedMeta boolean checkAllowAutoCreation) { return getPartitionedTopicMetadataAsync(topicName, authoritative, checkAllowAutoCreation) .thenCompose(metadata -> { - CompletableFuture ret; - if (metadata.partitions == 0 && !checkAllowAutoCreation) { + if (metadata.partitions > 1) { + // Some clients does not support partitioned topic. + return internalValidateClientVersionAsync().thenApply(__ -> metadata); + } else if (metadata.partitions == 1) { + return CompletableFuture.completedFuture(metadata); + } else { + // metadata.partitions == 0 // The topic may be a non-partitioned topic, so check if it exists here. // However, when checkAllowAutoCreation is true, the client will create the topic if // it doesn't exist. In this case, `partitions == 0` means the automatically created topic // is a non-partitioned topic so we shouldn't check if the topic exists. - ret = internalCheckTopicExists(topicName); - } else if (metadata.partitions > 1) { - ret = internalValidateClientVersionAsync(); - } else { - ret = CompletableFuture.completedFuture(null); + return pulsar().getBrokerService().isAllowAutoTopicCreationAsync(topicName) + .thenCompose(brokerAllowAutoTopicCreation -> { + if (checkAllowAutoCreation && brokerAllowAutoTopicCreation) { + // Whether it exists or not, auto create a non-partitioned topic by client. + return CompletableFuture.completedFuture(metadata); + } else { + // If it does not exist, response a Not Found error. + // Otherwise, response a non-partitioned metadata. + return internalCheckNonPartitionedTopicExists(topicName).thenApply(__ -> metadata); + } + }); } - return ret.thenApply(__ -> metadata); }); } @@ -714,6 +724,17 @@ public void updatePropertiesFailed(ManagedLedgerException exception, Object ctx) protected CompletableFuture internalCheckTopicExists(TopicName topicName) { return pulsar().getNamespaceService().checkTopicExists(topicName) + .thenAccept(info -> { + boolean exists = info.isExists(); + info.recycle(); + if (!exists) { + throw new RestException(Status.NOT_FOUND, getTopicNotFoundErrorMessage(topicName.toString())); + } + }); + } + + protected CompletableFuture internalCheckNonPartitionedTopicExists(TopicName topicName) { + return pulsar().getNamespaceService().checkNonPartitionedTopicExists(topicName) .thenAccept(exist -> { if (!exist) { throw new RestException(Status.NOT_FOUND, getTopicNotFoundErrorMessage(topicName.toString())); @@ -5531,8 +5552,10 @@ protected CompletableFuture validateShadowTopics(List shadowTopics "Only persistent topic can be set as shadow topic")); } futures.add(pulsar().getNamespaceService().checkTopicExists(shadowTopicName) - .thenAccept(isExists -> { - if (!isExists) { + .thenAccept(info -> { + boolean exists = info.isExists(); + info.recycle(); + if (!exists) { throw new RestException(Status.PRECONDITION_FAILED, "Shadow topic [" + shadowTopic + "] not exists."); } diff --git a/pulsar-broker/src/main/java/org/apache/pulsar/broker/admin/v2/NonPersistentTopics.java b/pulsar-broker/src/main/java/org/apache/pulsar/broker/admin/v2/NonPersistentTopics.java index cae7c651ce791..945ad69477c70 100644 --- a/pulsar-broker/src/main/java/org/apache/pulsar/broker/admin/v2/NonPersistentTopics.java +++ b/pulsar-broker/src/main/java/org/apache/pulsar/broker/admin/v2/NonPersistentTopics.java @@ -98,8 +98,20 @@ public void getPartitionedMetadata( @QueryParam("authoritative") @DefaultValue("false") boolean authoritative, @ApiParam(value = "Is check configuration required to automatically create topic") @QueryParam("checkAllowAutoCreation") @DefaultValue("false") boolean checkAllowAutoCreation) { - super.getPartitionedMetadata(asyncResponse, tenant, namespace, encodedTopic, authoritative, - checkAllowAutoCreation); + validateTopicName(tenant, namespace, encodedTopic); + validateTopicOwnershipAsync(topicName, authoritative).whenComplete((__, ex) -> { + if (ex != null) { + Throwable actEx = FutureUtil.unwrapCompletionException(ex); + if (isNot307And404Exception(actEx)) { + log.error("[{}] Failed to get internal stats for topic {}", clientAppId(), topicName, ex); + } + resumeAsyncResponseExceptionally(asyncResponse, actEx); + } else { + // "super.getPartitionedMetadata" will handle error itself. + super.getPartitionedMetadata(asyncResponse, tenant, namespace, encodedTopic, authoritative, + checkAllowAutoCreation); + } + }); } @GET diff --git a/pulsar-broker/src/main/java/org/apache/pulsar/broker/lookup/TopicLookupBase.java b/pulsar-broker/src/main/java/org/apache/pulsar/broker/lookup/TopicLookupBase.java index 7b2c777414884..9a05c3d992aaf 100644 --- a/pulsar-broker/src/main/java/org/apache/pulsar/broker/lookup/TopicLookupBase.java +++ b/pulsar-broker/src/main/java/org/apache/pulsar/broker/lookup/TopicLookupBase.java @@ -67,16 +67,22 @@ protected CompletableFuture internalLookupTopicAsync(final TopicName .thenCompose(__ -> validateGlobalNamespaceOwnershipAsync(topicName.getNamespaceObject())) .thenCompose(__ -> validateTopicOperationAsync(topicName, TopicOperation.LOOKUP, null)) .thenCompose(__ -> { + // Case-1: Non-persistent topic. // Currently, it's hard to check the non-persistent-non-partitioned topic, because it only exists // in the broker, it doesn't have metadata. If the topic is non-persistent and non-partitioned, - // we'll return the true flag. - CompletableFuture existFuture = (!topicName.isPersistent() && !topicName.isPartitioned()) - ? CompletableFuture.completedFuture(true) - : pulsar().getNamespaceService().checkTopicExists(topicName) - .thenCompose(exists -> exists ? CompletableFuture.completedFuture(true) - : pulsar().getBrokerService().isAllowAutoTopicCreationAsync(topicName)); - - return existFuture; + // we'll return the true flag. So either it is a partitioned topic or not, the result will be true. + if (!topicName.isPersistent()) { + return CompletableFuture.completedFuture(true); + } + // Case-2: Persistent topic. + return pulsar().getNamespaceService().checkTopicExists(topicName).thenCompose(info -> { + boolean exists = info.isExists(); + info.recycle(); + if (exists) { + return CompletableFuture.completedFuture(true); + } + return pulsar().getBrokerService().isAllowAutoTopicCreationAsync(topicName); + }); }) .thenCompose(exist -> { if (!exist) { diff --git a/pulsar-broker/src/main/java/org/apache/pulsar/broker/namespace/NamespaceService.java b/pulsar-broker/src/main/java/org/apache/pulsar/broker/namespace/NamespaceService.java index 626a187f46388..c62e9c52a768e 100644 --- a/pulsar-broker/src/main/java/org/apache/pulsar/broker/namespace/NamespaceService.java +++ b/pulsar-broker/src/main/java/org/apache/pulsar/broker/namespace/NamespaceService.java @@ -47,6 +47,7 @@ import java.util.regex.Pattern; import java.util.stream.Collectors; import javax.annotation.Nullable; +import lombok.extern.slf4j.Slf4j; import org.apache.commons.collections4.CollectionUtils; import org.apache.commons.collections4.ListUtils; import org.apache.commons.lang3.StringUtils; @@ -68,6 +69,7 @@ import org.apache.pulsar.broker.stats.prometheus.metrics.Summary; import org.apache.pulsar.broker.web.PulsarWebResource; import org.apache.pulsar.client.admin.PulsarAdmin; +import org.apache.pulsar.client.admin.PulsarAdminException; import org.apache.pulsar.client.api.ClientBuilder; import org.apache.pulsar.client.api.PulsarClient; import org.apache.pulsar.client.api.PulsarClientException; @@ -117,6 +119,7 @@ * * @see org.apache.pulsar.broker.PulsarService */ +@Slf4j public class NamespaceService implements AutoCloseable { private static final Logger LOG = LoggerFactory.getLogger(NamespaceService.class); @@ -1312,40 +1315,86 @@ public CompletableFuture> getOwnedTopicListForNamespaceBundle(Names }); } - public CompletableFuture checkTopicExists(TopicName topic) { - CompletableFuture future; - // If the topic is persistent and the name includes `-partition-`, find the topic from the managed/ledger. - if (topic.isPersistent() && topic.isPartitioned()) { - future = pulsar.getPulsarResources().getTopicResources().persistentTopicExists(topic); + /*** + * Check topic exists( partitioned or non-partitioned ). + */ + public CompletableFuture checkTopicExists(TopicName topic) { + return pulsar.getBrokerService() + .fetchPartitionedTopicMetadataAsync(TopicName.get(topic.toString())) + .thenCompose(metadata -> { + if (metadata.partitions > 0) { + return CompletableFuture.completedFuture( + TopicExistsInfo.newPartitionedTopicExists(metadata.partitions)); + } + return checkNonPartitionedTopicExists(topic) + .thenApply(b -> b ? TopicExistsInfo.newNonPartitionedTopicExists() + : TopicExistsInfo.newTopicNotExists()); + }); + } + + /*** + * Check non-partitioned topic exists. + */ + public CompletableFuture checkNonPartitionedTopicExists(TopicName topic) { + if (topic.isPersistent()) { + return pulsar.getPulsarResources().getTopicResources().persistentTopicExists(topic); } else { - future = CompletableFuture.completedFuture(false); + return checkNonPersistentNonPartitionedTopicExists(topic.toString()); } + } - return future.thenCompose(found -> { - if (found != null && found) { - return CompletableFuture.completedFuture(true); + /** + * Regarding non-persistent topic, we do not know whether it exists or not. Redirect the request to the ownership + * broker of this topic. HTTP API has implemented the mechanism that redirect to ownership broker, so just call + * HTTP API here. + */ + public CompletableFuture checkNonPersistentNonPartitionedTopicExists(String topic) { + TopicName topicName = TopicName.get(topic); + // "non-partitioned & non-persistent" topics only exist on the owner broker. + return checkTopicOwnership(TopicName.get(topic)).thenCompose(isOwned -> { + // The current broker is the owner. + if (isOwned) { + CompletableFuture> nonPersistentTopicFuture = pulsar.getBrokerService() + .getTopic(topic, false); + if (nonPersistentTopicFuture != null) { + return nonPersistentTopicFuture.thenApply(Optional::isPresent); + } else { + return CompletableFuture.completedFuture(false); + } } - return pulsar.getBrokerService() - .fetchPartitionedTopicMetadataAsync(TopicName.get(topic.getPartitionedTopicName())) - .thenCompose(metadata -> { - if (metadata.partitions > 0) { - return CompletableFuture.completedFuture(true); - } - - if (topic.isPersistent()) { - return pulsar.getPulsarResources().getTopicResources().persistentTopicExists(topic); - } else { - // The non-partitioned non-persistent topic only exist in the broker topics. - CompletableFuture> nonPersistentTopicFuture = - pulsar.getBrokerService().getTopics().get(topic.toString()); - if (nonPersistentTopicFuture == null) { + // Forward to the owner broker. + PulsarClientImpl pulsarClient; + try { + pulsarClient = (PulsarClientImpl) pulsar.getClient(); + } catch (Exception ex) { + // This error will never occur. + log.error("{} Failed to get partition metadata due to create internal admin client fails", topic, ex); + return FutureUtil.failedFuture(ex); + } + LookupOptions lookupOptions = LookupOptions.builder().readOnly(false).authoritative(true).build(); + return getBrokerServiceUrlAsync(TopicName.get(topic), lookupOptions) + .thenCompose(lookupResult -> { + if (!lookupResult.isPresent()) { + log.error("{} Failed to get partition metadata due can not find the owner broker", topic); + return FutureUtil.failedFuture(new ServiceUnitNotReadyException( + "No broker was available to own " + topicName)); + } + return pulsarClient.getLookup(lookupResult.get().getLookupData().getBrokerUrl()) + .getPartitionedTopicMetadata(topicName, false) + .thenApply(metadata -> true) + .exceptionallyCompose(ex -> { + Throwable actEx = FutureUtil.unwrapCompletionException(ex); + if (actEx instanceof PulsarClientException.NotFoundException + || actEx instanceof PulsarClientException.TopicDoesNotExistException + || actEx instanceof PulsarAdminException.NotFoundException) { return CompletableFuture.completedFuture(false); } else { - return nonPersistentTopicFuture.thenApply(Optional::isPresent); + log.error("{} Failed to get partition metadata due to redirecting fails", topic, ex); + return CompletableFuture.failedFuture(ex); } - } - }); + }); + }); }); } diff --git a/pulsar-broker/src/main/java/org/apache/pulsar/broker/namespace/TopicExistsInfo.java b/pulsar-broker/src/main/java/org/apache/pulsar/broker/namespace/TopicExistsInfo.java new file mode 100644 index 0000000000000..1c3f117719e8e --- /dev/null +++ b/pulsar-broker/src/main/java/org/apache/pulsar/broker/namespace/TopicExistsInfo.java @@ -0,0 +1,82 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. The ASF licenses this file + * to you under the Apache License, Version 2.0 (the + * "License"); you may not use this file except in compliance + * with the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, + * software distributed under the License is distributed on an + * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY + * KIND, either express or implied. See the License for the + * specific language governing permissions and limitations + * under the License. + */ +package org.apache.pulsar.broker.namespace; + +import io.netty.util.Recycler; +import lombok.Getter; +import org.apache.pulsar.common.policies.data.TopicType; + +public class TopicExistsInfo { + + private static final Recycler RECYCLER = new Recycler<>() { + @Override + protected TopicExistsInfo newObject(Handle handle) { + return new TopicExistsInfo(handle); + } + }; + + private static TopicExistsInfo nonPartitionedExists = new TopicExistsInfo(true, 0); + + private static TopicExistsInfo notExists = new TopicExistsInfo(false, 0); + + public static TopicExistsInfo newPartitionedTopicExists(Integer partitions){ + TopicExistsInfo info = RECYCLER.get(); + info.exists = true; + info.partitions = partitions.intValue(); + return info; + } + + public static TopicExistsInfo newNonPartitionedTopicExists(){ + return nonPartitionedExists; + } + + public static TopicExistsInfo newTopicNotExists(){ + return notExists; + } + + private final Recycler.Handle handle; + + @Getter + private int partitions; + @Getter + private boolean exists; + + private TopicExistsInfo(Recycler.Handle handle) { + this.handle = handle; + } + + private TopicExistsInfo(boolean exists, int partitions) { + this.handle = null; + this.partitions = partitions; + this.exists = exists; + } + + public void recycle() { + if (this == notExists || this == nonPartitionedExists || this.handle == null) { + return; + } + this.exists = false; + this.partitions = 0; + this.handle.recycle(this); + } + + public TopicType getTopicType() { + return this.partitions > 0 ? TopicType.PARTITIONED : TopicType.NON_PARTITIONED; + } +} diff --git a/pulsar-broker/src/main/java/org/apache/pulsar/broker/service/BrokerService.java b/pulsar-broker/src/main/java/org/apache/pulsar/broker/service/BrokerService.java index 45f9bdd7b0f7e..7ce9898bb38a5 100644 --- a/pulsar-broker/src/main/java/org/apache/pulsar/broker/service/BrokerService.java +++ b/pulsar-broker/src/main/java/org/apache/pulsar/broker/service/BrokerService.java @@ -1044,38 +1044,38 @@ public CompletableFuture> getTopic(final String topic, boolean c return getTopic(TopicName.get(topic), createIfMissing, properties); } + /** + * Retrieves or creates a topic based on the specified parameters. + * 0. If disable PersistentTopics or NonPersistentTopics, it will return a failed future with NotAllowedException. + * 1. If topic future exists in the cache returned directly regardless of whether it fails or timeout. + * 2. If the topic metadata exists, the topic is created regardless of {@code createIfMissing}. + * 3. If the topic metadata not exists, and {@code createIfMissing} is false, + * returns an empty Optional in a CompletableFuture. And this empty future not be added to the map. + * 4. Otherwise, use computeIfAbsent. It returns the existing topic or creates and adds a new topicFuture. + * Any exceptions will remove the topicFuture from the map. + * + * @param topicName The name of the topic, potentially including partition information. + * @param createIfMissing If true, creates the topic if it does not exist. + * @param properties Topic configuration properties used during creation. + * @return CompletableFuture with an Optional of the topic if found or created, otherwise empty. + */ public CompletableFuture> getTopic(final TopicName topicName, boolean createIfMissing, Map properties) { try { - CompletableFuture> topicFuture = topics.get(topicName.toString()); - if (topicFuture != null) { - if (topicFuture.isCompletedExceptionally() - || (topicFuture.isDone() && !topicFuture.getNow(Optional.empty()).isPresent())) { - // Exceptional topics should be recreated. - topics.remove(topicName.toString(), topicFuture); - } else { - // a non-existing topic in the cache shouldn't prevent creating a topic - if (createIfMissing) { - if (topicFuture.isDone() && topicFuture.getNow(Optional.empty()).isPresent()) { - return topicFuture; - } else { - return topicFuture.thenCompose(value -> { - if (!value.isPresent()) { - // retry and create topic - return getTopic(topicName, createIfMissing, properties); - } else { - // in-progress future completed successfully - return CompletableFuture.completedFuture(value); - } - }); - } - } else { - return topicFuture; - } - } + // If topic future exists in the cache returned directly regardless of whether it fails or timeout. + CompletableFuture> tp = topics.get(topicName.toString()); + if (tp != null) { + return tp; } final boolean isPersistentTopic = topicName.getDomain().equals(TopicDomain.persistent); if (isPersistentTopic) { + if (!pulsar.getConfiguration().isEnablePersistentTopics()) { + if (log.isDebugEnabled()) { + log.debug("Broker is unable to load persistent topic {}", topicName); + } + return FutureUtil.failedFuture(new NotAllowedException( + "Broker is unable to load persistent topic")); + } return pulsar.getPulsarResources().getTopicResources().persistentTopicExists(topicName) .thenCompose(exists -> { if (!exists && !createIfMissing) { @@ -1090,44 +1090,48 @@ public CompletableFuture> getTopic(final TopicName topicName, bo throw FutureUtil.wrapToCompletionException(new ServiceUnitNotReadyException(errorInfo)); }).thenCompose(optionalTopicPolicies -> { final TopicPolicies topicPolicies = optionalTopicPolicies.orElse(null); - return topics.computeIfAbsent(topicName.toString(), (tpName) -> { - if (topicName.isPartitioned()) { - final TopicName topicNameEntity = TopicName.get(topicName.getPartitionedTopicName()); - return fetchPartitionedTopicMetadataAsync(topicNameEntity) - .thenCompose((metadata) -> { - // Allow crate non-partitioned persistent topic that name includes - // `partition` - if (metadata.partitions == 0 - || topicName.getPartitionIndex() < metadata.partitions) { - return loadOrCreatePersistentTopic(tpName, createIfMissing, - properties, topicPolicies); - } + if (topicName.isPartitioned()) { + final TopicName topicNameEntity = TopicName.get(topicName.getPartitionedTopicName()); + return fetchPartitionedTopicMetadataAsync(topicNameEntity) + .thenCompose((metadata) -> { + // Allow crate non-partitioned persistent topic that name includes + // `partition` + if (metadata.partitions == 0 + || topicName.getPartitionIndex() < metadata.partitions) { + return topics.computeIfAbsent(topicName.toString(), (tpName) -> + loadOrCreatePersistentTopic(tpName, + createIfMissing, properties, topicPolicies)); + } else { final String errorMsg = String.format("Illegal topic partition name %s with max allowed " + "%d partitions", topicName, metadata.partitions); log.warn(errorMsg); return FutureUtil.failedFuture( new BrokerServiceException.NotAllowedException(errorMsg)); - }); - } - return loadOrCreatePersistentTopic(tpName, createIfMissing, properties, topicPolicies); - }).thenCompose(optionalTopic -> { - if (!optionalTopic.isPresent() && createIfMissing) { - log.warn("[{}] Try to recreate the topic with createIfMissing=true " - + "but the returned topic is empty", topicName); - return getTopic(topicName, createIfMissing, properties); - } - return CompletableFuture.completedFuture(optionalTopic); - }); + } + }); + } else { + return topics.computeIfAbsent(topicName.toString(), (tpName) -> + loadOrCreatePersistentTopic(tpName, createIfMissing, properties, topicPolicies)); + } }); }); } else { - return topics.computeIfAbsent(topicName.toString(), (name) -> { + if (!pulsar.getConfiguration().isEnableNonPersistentTopics()) { + if (log.isDebugEnabled()) { + log.debug("Broker is unable to load non-persistent topic {}", topicName); + } + return FutureUtil.failedFuture(new NotAllowedException( + "Broker is unable to load persistent topic")); + } + if (!topics.containsKey(topicName.toString())) { topicEventsDispatcher.notify(topicName.toString(), TopicEvent.LOAD, EventStage.BEFORE); - if (topicName.isPartitioned()) { - final TopicName partitionedTopicName = TopicName.get(topicName.getPartitionedTopicName()); - return this.fetchPartitionedTopicMetadataAsync(partitionedTopicName).thenCompose((metadata) -> { - if (topicName.getPartitionIndex() < metadata.partitions) { + } + if (topicName.isPartitioned()) { + final TopicName partitionedTopicName = TopicName.get(topicName.getPartitionedTopicName()); + return this.fetchPartitionedTopicMetadataAsync(partitionedTopicName).thenCompose((metadata) -> { + if (topicName.getPartitionIndex() < metadata.partitions) { + return topics.computeIfAbsent(topicName.toString(), (name) -> { topicEventsDispatcher .notify(topicName.toString(), TopicEvent.CREATE, EventStage.BEFORE); @@ -1138,11 +1142,13 @@ public CompletableFuture> getTopic(final TopicName topicName, bo topicEventsDispatcher .notifyOnCompletion(eventFuture, topicName.toString(), TopicEvent.LOAD); return res; - } - topicEventsDispatcher.notify(topicName.toString(), TopicEvent.LOAD, EventStage.FAILURE); - return CompletableFuture.completedFuture(Optional.empty()); - }); - } else if (createIfMissing) { + }); + } + topicEventsDispatcher.notify(topicName.toString(), TopicEvent.LOAD, EventStage.FAILURE); + return CompletableFuture.completedFuture(Optional.empty()); + }); + } else if (createIfMissing) { + return topics.computeIfAbsent(topicName.toString(), (name) -> { topicEventsDispatcher.notify(topicName.toString(), TopicEvent.CREATE, EventStage.BEFORE); CompletableFuture> res = createNonPersistentTopic(name); @@ -1152,11 +1158,15 @@ public CompletableFuture> getTopic(final TopicName topicName, bo topicEventsDispatcher .notifyOnCompletion(eventFuture, topicName.toString(), TopicEvent.LOAD); return res; - } else { + }); + } else { + CompletableFuture> topicFuture = topics.get(topicName.toString()); + if (topicFuture == null) { topicEventsDispatcher.notify(topicName.toString(), TopicEvent.LOAD, EventStage.FAILURE); - return CompletableFuture.completedFuture(Optional.empty()); + topicFuture = CompletableFuture.completedFuture(Optional.empty()); } - }); + return topicFuture; + } } } catch (IllegalArgumentException e) { log.warn("[{}] Illegalargument exception when loading topic", topicName, e); @@ -1295,15 +1305,9 @@ private CompletableFuture> createNonPersistentTopic(String topic CompletableFuture> topicFuture = new CompletableFuture<>(); topicFuture.exceptionally(t -> { pulsarStats.recordTopicLoadFailed(); + pulsar.getExecutor().execute(() -> topics.remove(topic, topicFuture)); return null; }); - if (!pulsar.getConfiguration().isEnableNonPersistentTopics()) { - if (log.isDebugEnabled()) { - log.debug("Broker is unable to load non-persistent topic {}", topic); - } - return FutureUtil.failedFuture( - new NotAllowedException("Broker is not unable to load non-persistent topic")); - } final long topicCreateTimeMs = TimeUnit.NANOSECONDS.toMillis(System.nanoTime()); NonPersistentTopic nonPersistentTopic; try { @@ -1326,7 +1330,6 @@ private CompletableFuture> createNonPersistentTopic(String topic }).exceptionally(ex -> { log.warn("Replication check failed. Removing topic from topics list {}, {}", topic, ex.getCause()); nonPersistentTopic.stopReplProducers().whenComplete((v, exception) -> { - pulsar.getExecutor().execute(() -> topics.remove(topic, topicFuture)); topicFuture.completeExceptionally(ex); }); return null; @@ -1579,14 +1582,6 @@ protected CompletableFuture> loadOrCreatePersistentTopic(final S final CompletableFuture> topicFuture = FutureUtil.createFutureWithTimeout( Duration.ofSeconds(pulsar.getConfiguration().getTopicLoadTimeoutSeconds()), executor(), () -> FAILED_TO_LOAD_TOPIC_TIMEOUT_EXCEPTION); - if (!pulsar.getConfiguration().isEnablePersistentTopics()) { - if (log.isDebugEnabled()) { - log.debug("Broker is unable to load persistent topic {}", topic); - } - topicFuture.completeExceptionally(new NotAllowedException( - "Broker is unable to load persistent topic")); - return topicFuture; - } checkTopicNsOwnership(topic) .thenRun(() -> { @@ -1609,6 +1604,7 @@ protected CompletableFuture> loadOrCreatePersistentTopic(final S } } }).exceptionally(ex -> { + pulsar.getExecutor().execute(() -> topics.remove(topic, topicFuture)); topicFuture.completeExceptionally(ex.getCause()); return null; }); @@ -1779,6 +1775,7 @@ public void openLedgerComplete(ManagedLedger ledger, Object ctx) { + " topic", topic, FutureUtil.getException(topicFuture)); executor().submit(() -> { persistentTopic.close().whenComplete((ignore, ex) -> { + topics.remove(topic, topicFuture); if (ex != null) { log.warn("[{}] Get an error when closing topic.", topic, ex); @@ -1795,6 +1792,7 @@ public void openLedgerComplete(ManagedLedger ledger, Object ctx) { + " Removing topic from topics list {}, {}", topic, ex); executor().submit(() -> { persistentTopic.close().whenComplete((ignore, closeEx) -> { + topics.remove(topic, topicFuture); if (closeEx != null) { log.warn("[{}] Get an error when closing topic.", topic, closeEx); @@ -3214,65 +3212,66 @@ public CompletableFuture fetchPartitionedTopicMetadata if (pulsar.getNamespaceService() == null) { return FutureUtil.failedFuture(new NamingException("namespace service is not ready")); } - return pulsar.getPulsarResources().getNamespaceResources().getPoliciesAsync(topicName.getNamespaceObject()) - .thenCompose(policies -> pulsar.getNamespaceService().checkTopicExists(topicName) - .thenCompose(topicExists -> fetchPartitionedTopicMetadataAsync(topicName) - .thenCompose(metadata -> { - CompletableFuture future = new CompletableFuture<>(); - - // There are a couple of potentially blocking calls, which we cannot make from the - // MetadataStore callback thread. - pulsar.getExecutor().execute(() -> { - // If topic is already exist, creating partitioned topic is not allowed. - - if (metadata.partitions == 0 - && !topicExists - && !topicName.isPartitioned() - && pulsar.getBrokerService() - .isDefaultTopicTypePartitioned(topicName, policies)) { - isAllowAutoTopicCreationAsync(topicName, policies).thenAccept(allowed -> { - if (allowed) { - pulsar.getBrokerService() - .createDefaultPartitionedTopicAsync(topicName, policies) - .thenAccept(md -> future.complete(md)) - .exceptionally(ex -> { - if (ex.getCause() - instanceof MetadataStoreException - .AlreadyExistsException) { - log.info("[{}] The partitioned topic is already" - + " created, try to refresh the cache and read" - + " again.", topicName); - // The partitioned topic might be created concurrently - fetchPartitionedTopicMetadataAsync(topicName, true) - .whenComplete((metadata2, ex2) -> { - if (ex2 == null) { - future.complete(metadata2); - } else { - future.completeExceptionally(ex2); - } - }); - } else { - log.error("[{}] operation of creating partitioned" - + " topic metadata failed", - topicName, ex); - future.completeExceptionally(ex); - } - return null; - }); - } else { - future.complete(metadata); - } - }).exceptionally(ex -> { - future.completeExceptionally(ex); - return null; - }); - } else { - future.complete(metadata); - } - }); + return pulsar.getNamespaceService().checkTopicExists(topicName).thenComposeAsync(topicExistsInfo -> { + final boolean topicExists = topicExistsInfo.isExists(); + final TopicType topicType = topicExistsInfo.getTopicType(); + final Integer partitions = topicExistsInfo.getPartitions(); + topicExistsInfo.recycle(); + + // Topic exists. + if (topicExists) { + if (topicType.equals(TopicType.PARTITIONED)) { + return CompletableFuture.completedFuture(new PartitionedTopicMetadata(partitions)); + } + return CompletableFuture.completedFuture(new PartitionedTopicMetadata(0)); + } - return future; - }))); + // Try created if allowed to create a partitioned topic automatically. + return pulsar.getPulsarResources().getNamespaceResources().getPoliciesAsync(topicName.getNamespaceObject()) + .thenComposeAsync(policies -> { + return isAllowAutoTopicCreationAsync(topicName, policies).thenComposeAsync(allowed -> { + // Not Allow auto-creation. + if (!allowed) { + // Do not change the original behavior, or default return a non-partitioned topic. + return CompletableFuture.completedFuture(new PartitionedTopicMetadata(0)); + } + + // Allow auto create non-partitioned topic. + boolean autoCreatePartitionedTopic = pulsar.getBrokerService() + .isDefaultTopicTypePartitioned(topicName, policies); + if (!autoCreatePartitionedTopic || topicName.isPartitioned()) { + return CompletableFuture.completedFuture(new PartitionedTopicMetadata(0)); + } + + // Create partitioned metadata. + return pulsar.getBrokerService().createDefaultPartitionedTopicAsync(topicName, policies) + .exceptionallyCompose(ex -> { + // The partitioned topic might be created concurrently. + if (ex.getCause() instanceof MetadataStoreException.AlreadyExistsException) { + log.info("[{}] The partitioned topic is already created, try to refresh the cache" + + " and read again.", topicName); + CompletableFuture recheckFuture = + fetchPartitionedTopicMetadataAsync(topicName, true); + recheckFuture.exceptionally(ex2 -> { + // Just for printing a log if error occurs. + log.error("[{}] Fetch partitioned topic metadata failed", topicName, ex); + return null; + }); + return recheckFuture; + } else { + log.error("[{}] operation of creating partitioned topic metadata failed", + topicName, ex); + return CompletableFuture.failedFuture(ex); + } + }); + }, pulsar.getExecutor()).exceptionallyCompose(ex -> { + log.error("[{}] operation of get partitioned metadata failed due to calling" + + " isAllowAutoTopicCreationAsync failed", + topicName, ex); + return CompletableFuture.failedFuture(ex); + }); + }, pulsar.getExecutor()); + }, pulsar.getExecutor()); } @SuppressWarnings("deprecation") diff --git a/pulsar-broker/src/main/java/org/apache/pulsar/broker/service/ServerCnx.java b/pulsar-broker/src/main/java/org/apache/pulsar/broker/service/ServerCnx.java index 55e1d96b594e3..2514be55fffa9 100644 --- a/pulsar-broker/src/main/java/org/apache/pulsar/broker/service/ServerCnx.java +++ b/pulsar-broker/src/main/java/org/apache/pulsar/broker/service/ServerCnx.java @@ -81,6 +81,7 @@ import org.apache.pulsar.broker.authentication.AuthenticationState; import org.apache.pulsar.broker.intercept.BrokerInterceptor; import org.apache.pulsar.broker.limiter.ConnectionController; +import org.apache.pulsar.broker.namespace.NamespaceService; import org.apache.pulsar.broker.service.BrokerServiceException.ConsumerBusyException; import org.apache.pulsar.broker.service.BrokerServiceException.ServerMetadataException; import org.apache.pulsar.broker.service.BrokerServiceException.ServiceUnitNotReadyException; @@ -155,6 +156,7 @@ import org.apache.pulsar.common.policies.data.ClusterData.ClusterUrl; import org.apache.pulsar.common.policies.data.NamespaceOperation; import org.apache.pulsar.common.policies.data.TopicOperation; +import org.apache.pulsar.common.policies.data.TopicType; import org.apache.pulsar.common.policies.data.stats.ConsumerStatsImpl; import org.apache.pulsar.common.protocol.ByteBufPair; import org.apache.pulsar.common.protocol.CommandUtils; @@ -578,35 +580,68 @@ protected void handlePartitionMetadataRequest(CommandPartitionedTopicMetadata pa isTopicOperationAllowed(topicName, TopicOperation.LOOKUP, authenticationData, originalAuthData).thenApply( isAuthorized -> { if (isAuthorized) { - unsafeGetPartitionedTopicMetadataAsync(getBrokerService().pulsar(), topicName) - .handle((metadata, ex) -> { - if (ex == null) { - int partitions = metadata.partitions; - commandSender.sendPartitionMetadataResponse(partitions, requestId); + // Get if exists, respond not found error if not exists. + getBrokerService().isAllowAutoTopicCreationAsync(topicName).thenAccept(brokerAllowAutoCreate -> { + boolean autoCreateIfNotExist = partitionMetadata.isMetadataAutoCreationEnabled() + && brokerAllowAutoCreate; + if (!autoCreateIfNotExist) { + NamespaceService namespaceService = getBrokerService().getPulsar().getNamespaceService(); + namespaceService.checkTopicExists(topicName).thenAccept(topicExistsInfo -> { + lookupSemaphore.release(); + if (!topicExistsInfo.isExists()) { + writeAndFlush(Commands.newPartitionMetadataResponse( + ServerError.TopicNotFound, "", requestId)); + } else if (topicExistsInfo.getTopicType().equals(TopicType.PARTITIONED)) { + commandSender.sendPartitionMetadataResponse(topicExistsInfo.getPartitions(), + requestId); } else { - if (ex instanceof PulsarClientException) { - log.warn("Failed to authorize {} at [{}] on topic {} : {}", getRole(), - remoteAddress, topicName, ex.getMessage()); - commandSender.sendPartitionMetadataResponse(ServerError.AuthorizationError, - ex.getMessage(), requestId); - } else { - log.warn("Failed to get Partitioned Metadata [{}] {}: {}", remoteAddress, - topicName, ex.getMessage(), ex); - ServerError error = ServerError.ServiceNotReady; - if (ex instanceof RestException restException){ - int responseCode = restException.getResponse().getStatus(); - if (responseCode == NOT_FOUND.getStatusCode()){ - error = ServerError.TopicNotFound; - } else if (responseCode < INTERNAL_SERVER_ERROR.getStatusCode()){ - error = ServerError.MetadataError; - } - } - commandSender.sendPartitionMetadataResponse(error, ex.getMessage(), requestId); - } + commandSender.sendPartitionMetadataResponse(0, requestId); } + // release resources. + topicExistsInfo.recycle(); + }).exceptionally(ex -> { lookupSemaphore.release(); + log.error("{} {} Failed to get partition metadata", topicName, + ServerCnx.this.toString(), ex); + writeAndFlush( + Commands.newPartitionMetadataResponse(ServerError.MetadataError, + "Failed to get partition metadata", + requestId)); return null; }); + } else { + // Get if exists, create a new one if not exists. + unsafeGetPartitionedTopicMetadataAsync(getBrokerService().pulsar(), topicName) + .whenComplete((metadata, ex) -> { + lookupSemaphore.release(); + if (ex == null) { + int partitions = metadata.partitions; + commandSender.sendPartitionMetadataResponse(partitions, requestId); + } else { + if (ex instanceof PulsarClientException) { + log.warn("Failed to authorize {} at [{}] on topic {} : {}", getRole(), + remoteAddress, topicName, ex.getMessage()); + commandSender.sendPartitionMetadataResponse(ServerError.AuthorizationError, + ex.getMessage(), requestId); + } else { + log.warn("Failed to get Partitioned Metadata [{}] {}: {}", remoteAddress, + topicName, ex.getMessage(), ex); + ServerError error = ServerError.ServiceNotReady; + if (ex instanceof RestException restException){ + int responseCode = restException.getResponse().getStatus(); + if (responseCode == NOT_FOUND.getStatusCode()){ + error = ServerError.TopicNotFound; + } else if (responseCode < INTERNAL_SERVER_ERROR.getStatusCode()){ + error = ServerError.MetadataError; + } + } + commandSender.sendPartitionMetadataResponse(error, ex.getMessage(), + requestId); + } + } + }); + } + }); } else { final String msg = "Client is not authorized to Get Partition Metadata"; log.warn("[{}] {} with role {} on topic {}", remoteAddress, msg, getPrincipal(), topicName); diff --git a/pulsar-broker/src/main/java/org/apache/pulsar/broker/service/persistent/PersistentTopic.java b/pulsar-broker/src/main/java/org/apache/pulsar/broker/service/persistent/PersistentTopic.java index b83a8206b4ab4..f2b723c38cbca 100644 --- a/pulsar-broker/src/main/java/org/apache/pulsar/broker/service/persistent/PersistentTopic.java +++ b/pulsar-broker/src/main/java/org/apache/pulsar/broker/service/persistent/PersistentTopic.java @@ -271,6 +271,9 @@ protected TopicStatsHelper initialValue() { private volatile CloseFutures closeFutures; + // The last position that can be dispatched to consumers + private volatile Position lastDispatchablePosition; + /*** * We use 2 futures to prevent a new closing if there is an in-progress deletion or closing. We make Pulsar return * the in-progress one when it is called the second time. @@ -400,14 +403,6 @@ public CompletableFuture initialize() { PersistentTopic.this.topicCompactionService = service; this.createPersistentSubscriptions(); })); - - for (ManagedCursor cursor : ledger.getCursors()) { - if (cursor.getName().startsWith(replicatorPrefix)) { - String localCluster = brokerService.pulsar().getConfiguration().getClusterName(); - String remoteCluster = PersistentReplicator.getRemoteCluster(cursor.getName()); - futures.add(addReplicationCluster(remoteCluster, cursor, localCluster)); - } - } return FutureUtil.waitForAll(futures).thenCompose(__ -> brokerService.pulsar().getPulsarResources().getNamespaceResources() .getPoliciesAsync(TopicName.get(topic).getNamespaceObject()) @@ -438,6 +433,7 @@ public CompletableFuture initialize() { isAllowAutoUpdateSchema = policies.is_allow_auto_update_schema; }, getOrderedExecutor()) .thenCompose(ignore -> initTopicPolicy()) + .thenCompose(ignore -> removeOrphanReplicationCursors()) .exceptionally(ex -> { log.warn("[{}] Error getting policies {} and isEncryptionRequired will be set to false", topic, ex.getMessage()); @@ -515,6 +511,21 @@ private void createPersistentSubscriptions() { checkReplicatedSubscriptionControllerState(); } + private CompletableFuture removeOrphanReplicationCursors() { + List> futures = new ArrayList<>(); + List replicationClusters = topicPolicies.getReplicationClusters().get(); + for (ManagedCursor cursor : ledger.getCursors()) { + if (cursor.getName().startsWith(replicatorPrefix)) { + String remoteCluster = PersistentReplicator.getRemoteCluster(cursor.getName()); + if (!replicationClusters.contains(remoteCluster)) { + log.warn("Remove the orphan replicator because the cluster '{}' does not exist", remoteCluster); + futures.add(removeReplicator(remoteCluster)); + } + } + } + return FutureUtil.waitForAll(futures); + } + /** * Unload a subscriber. * @throws SubscriptionNotFoundException If subscription not founded. @@ -1404,7 +1415,7 @@ private CompletableFuture delete(boolean failIfHasSubscriptions, return FutureUtil.failedFuture( new TopicBusyException("Topic has subscriptions: " + subscriptions.keys())); } else if (failIfHasBacklogs) { - if (hasBacklogs()) { + if (hasBacklogs(false)) { List backlogSubs = subscriptions.values().stream() .filter(sub -> sub.getNumberOfEntriesInBacklog(false) > 0) @@ -1930,30 +1941,17 @@ public void openCursorFailed(ManagedLedgerException exception, Object ctx) { return future; } - private CompletableFuture checkReplicationCluster(String remoteCluster) { - return brokerService.getPulsar().getPulsarResources().getNamespaceResources() - .getPoliciesAsync(TopicName.get(topic).getNamespaceObject()) - .thenApply(optPolicies -> optPolicies.map(policies -> policies.replication_clusters) - .orElse(Collections.emptySet()).contains(remoteCluster) - || topicPolicies.getReplicationClusters().get().contains(remoteCluster)); - } - protected CompletableFuture addReplicationCluster(String remoteCluster, ManagedCursor cursor, String localCluster) { return AbstractReplicator.validatePartitionedTopicAsync(PersistentTopic.this.getName(), brokerService) - .thenCompose(__ -> checkReplicationCluster(remoteCluster)) - .thenCompose(clusterExists -> { - if (!clusterExists) { - log.warn("Remove the replicator because the cluster '{}' does not exist", remoteCluster); - return removeReplicator(remoteCluster).thenApply(__ -> null); - } - return brokerService.pulsar().getPulsarResources().getClusterResources() - .getClusterAsync(remoteCluster) - .thenApply(clusterData -> - brokerService.getReplicationClient(remoteCluster, clusterData)); - }) + .thenCompose(__ -> brokerService.pulsar().getPulsarResources().getClusterResources() + .getClusterAsync(remoteCluster) + .thenApply(clusterData -> + brokerService.getReplicationClient(remoteCluster, clusterData))) .thenAccept(replicationClient -> { if (replicationClient == null) { + log.error("[{}] Can not create replicator because the remote client can not be created." + + " remote cluster: {}.", topic, remoteCluster); return; } lock.readLock().lock(); @@ -2485,12 +2483,9 @@ public CompletableFuture asyncGetStats(boolean getPreciseBacklog stats.backlogQuotaLimitTime = getBacklogQuota(BacklogQuotaType.message_age).getLimitTime(); TimeBasedBacklogQuotaCheckResult backlogQuotaCheckResult = timeBasedBacklogQuotaCheckResult; - stats.oldestBacklogMessageAgeSeconds = (backlogQuotaCheckResult == null) - ? (long) -1 - : TimeUnit.MILLISECONDS.toSeconds( - Clock.systemUTC().millis() - backlogQuotaCheckResult.getPositionPublishTimestampInMillis()); - + stats.oldestBacklogMessageAgeSeconds = getBestEffortOldestUnacknowledgedMessageAgeSeconds(); stats.oldestBacklogMessageSubscriptionName = (backlogQuotaCheckResult == null) + || !hasBacklogs(getPreciseBacklog) ? null : backlogQuotaCheckResult.getCursorName(); @@ -2746,7 +2741,7 @@ public boolean isActive(InactiveTopicDeleteMode deleteMode) { } break; case delete_when_subscriptions_caught_up: - if (hasBacklogs()) { + if (hasBacklogs(false)) { return true; } break; @@ -2759,8 +2754,8 @@ public boolean isActive(InactiveTopicDeleteMode deleteMode) { } } - private boolean hasBacklogs() { - return subscriptions.values().stream().anyMatch(sub -> sub.getNumberOfEntriesInBacklog(false) > 0); + private boolean hasBacklogs(boolean getPreciseBacklog) { + return subscriptions.values().stream().anyMatch(sub -> sub.getNumberOfEntriesInBacklog(getPreciseBacklog) > 0); } @Override @@ -3140,6 +3135,9 @@ public boolean isSizeBacklogExceeded() { @Override public long getBestEffortOldestUnacknowledgedMessageAgeSeconds() { + if (!hasBacklogs(false)) { + return 0; + } TimeBasedBacklogQuotaCheckResult result = timeBasedBacklogQuotaCheckResult; if (result == null) { return -1; @@ -3227,6 +3225,9 @@ public CompletableFuture checkTimeBacklogExceeded() { } if (brokerService.pulsar().getConfiguration().isPreciseTimeBasedBacklogQuotaCheck()) { + if (!hasBacklogs(true)) { + return CompletableFuture.completedFuture(false); + } CompletableFuture future = new CompletableFuture<>(); // Check if first unconsumed message(first message after mark delete position) // for slowest cursor's has expired. @@ -3280,6 +3281,9 @@ public void readEntryFailed(ManagedLedgerException exception, Object ctx) { return future; } else { try { + if (!hasBacklogs(false)) { + return CompletableFuture.completedFuture(false); + } EstimateTimeBasedBacklogQuotaCheckResult checkResult = estimatedTimeBasedBacklogQuotaCheck(oldestMarkDeletePosition); if (checkResult.getEstimatedOldestUnacknowledgedMessageTimestamp() != null) { @@ -3476,18 +3480,57 @@ public Position getLastPosition() { @Override public CompletableFuture getLastDispatchablePosition() { - return ManagedLedgerImplUtils.asyncGetLastValidPosition((ManagedLedgerImpl) ledger, entry -> { - MessageMetadata md = Commands.parseMessageMetadata(entry.getDataBuffer()); - // If a messages has marker will filter by AbstractBaseDispatcher.filterEntriesForConsumer - if (Markers.isServerOnlyMarker(md)) { - return false; - } else if (md.hasTxnidMostBits() && md.hasTxnidLeastBits()) { - // Filter-out transaction aborted messages. - TxnID txnID = new TxnID(md.getTxnidMostBits(), md.getTxnidLeastBits()); - return !isTxnAborted(txnID, (PositionImpl) entry.getPosition()); - } - return true; - }, getMaxReadPosition()); + if (lastDispatchablePosition != null) { + return CompletableFuture.completedFuture(lastDispatchablePosition); + } + return ManagedLedgerImplUtils + .asyncGetLastValidPosition((ManagedLedgerImpl) ledger, entry -> { + MessageMetadata md = Commands.parseMessageMetadata(entry.getDataBuffer()); + // If a messages has marker will filter by AbstractBaseDispatcher.filterEntriesForConsumer + if (Markers.isServerOnlyMarker(md)) { + return false; + } else if (md.hasTxnidMostBits() && md.hasTxnidLeastBits()) { + // Filter-out transaction aborted messages. + TxnID txnID = new TxnID(md.getTxnidMostBits(), md.getTxnidLeastBits()); + return !isTxnAborted(txnID, (PositionImpl) entry.getPosition()); + } + return true; + }, getMaxReadPosition()) + .thenApply(position -> { + // Update lastDispatchablePosition to the given position + updateLastDispatchablePosition(position); + return position; + }); + } + + /** + * Update lastDispatchablePosition if the given position is greater than the lastDispatchablePosition. + * + * @param position + */ + public synchronized void updateLastDispatchablePosition(Position position) { + // Update lastDispatchablePosition to null if the position is null, fallback to + // ManagedLedgerImplUtils#asyncGetLastValidPosition + if (position == null) { + lastDispatchablePosition = null; + return; + } + + PositionImpl position0 = (PositionImpl) position; + // If the position is greater than the maxReadPosition, ignore + if (position0.compareTo(getMaxReadPosition()) > 0) { + return; + } + // If the lastDispatchablePosition is null, set it to the position + if (lastDispatchablePosition == null) { + lastDispatchablePosition = position; + return; + } + // If the position is greater than the lastDispatchablePosition, update it + PositionImpl lastDispatchablePosition0 = (PositionImpl) lastDispatchablePosition; + if (position0.compareTo(lastDispatchablePosition0) > 0) { + lastDispatchablePosition = position; + } } @Override diff --git a/pulsar-broker/src/main/java/org/apache/pulsar/broker/transaction/buffer/impl/InMemTransactionBuffer.java b/pulsar-broker/src/main/java/org/apache/pulsar/broker/transaction/buffer/impl/InMemTransactionBuffer.java index bab7b64c608c4..533d0716d413c 100644 --- a/pulsar-broker/src/main/java/org/apache/pulsar/broker/transaction/buffer/impl/InMemTransactionBuffer.java +++ b/pulsar-broker/src/main/java/org/apache/pulsar/broker/transaction/buffer/impl/InMemTransactionBuffer.java @@ -377,8 +377,11 @@ public boolean isTxnAborted(TxnID txnID, PositionImpl readPosition) { @Override public void syncMaxReadPositionForNormalPublish(PositionImpl position, boolean isMarkerMessage) { - if (!isMarkerMessage && maxReadPositionCallBack != null) { - maxReadPositionCallBack.maxReadPositionMovedForward(null, position); + if (!isMarkerMessage) { + updateLastDispatchablePosition(position); + if (maxReadPositionCallBack != null) { + maxReadPositionCallBack.maxReadPositionMovedForward(null, position); + } } } @@ -436,4 +439,11 @@ public long getCommittedTxnCount() { .filter(txnBuffer -> txnBuffer.status.equals(TxnStatus.COMMITTED)) .count(); } + + // ThreadSafe + private void updateLastDispatchablePosition(Position position) { + if (topic instanceof PersistentTopic t) { + t.updateLastDispatchablePosition(position); + } + } } diff --git a/pulsar-broker/src/main/java/org/apache/pulsar/broker/transaction/buffer/impl/TopicTransactionBuffer.java b/pulsar-broker/src/main/java/org/apache/pulsar/broker/transaction/buffer/impl/TopicTransactionBuffer.java index cbf9630658f19..2eb9e0f2ba0d1 100644 --- a/pulsar-broker/src/main/java/org/apache/pulsar/broker/transaction/buffer/impl/TopicTransactionBuffer.java +++ b/pulsar-broker/src/main/java/org/apache/pulsar/broker/transaction/buffer/impl/TopicTransactionBuffer.java @@ -297,6 +297,11 @@ private void handleTransactionMessage(TxnID txnId, Position position) { } } + // ThreadSafe + private void updateLastDispatchablePosition(Position position) { + topic.updateLastDispatchablePosition(position); + } + @Override public CompletableFuture openTransactionBufferReader(TxnID txnID, long startSequenceId) { return null; @@ -459,6 +464,8 @@ void removeTxnAndUpdateMaxReadPosition(TxnID txnID) { } else { updateMaxReadPosition((PositionImpl) topic.getManagedLedger().getLastConfirmedEntry(), false); } + // Update the last dispatchable position to null if there is a TXN finished. + updateLastDispatchablePosition(null); } /** @@ -523,6 +530,10 @@ public void syncMaxReadPositionForNormalPublish(PositionImpl position, boolean i } } } + // If the message is a normal message, update the last dispatchable position. + if (!isMarkerMessage) { + updateLastDispatchablePosition(position); + } } @Override diff --git a/pulsar-broker/src/main/java/org/apache/pulsar/broker/transaction/buffer/impl/TransactionBufferDisable.java b/pulsar-broker/src/main/java/org/apache/pulsar/broker/transaction/buffer/impl/TransactionBufferDisable.java index ebd61dbaa82ec..6f5dc0cd4d0dd 100644 --- a/pulsar-broker/src/main/java/org/apache/pulsar/broker/transaction/buffer/impl/TransactionBufferDisable.java +++ b/pulsar-broker/src/main/java/org/apache/pulsar/broker/transaction/buffer/impl/TransactionBufferDisable.java @@ -99,8 +99,11 @@ public boolean isTxnAborted(TxnID txnID, PositionImpl readPosition) { @Override public void syncMaxReadPositionForNormalPublish(PositionImpl position, boolean isMarkerMessage) { - if (!isMarkerMessage && maxReadPositionCallBack != null) { - maxReadPositionCallBack.maxReadPositionMovedForward(null, position); + if (!isMarkerMessage) { + updateLastDispatchablePosition(position); + if (maxReadPositionCallBack != null) { + maxReadPositionCallBack.maxReadPositionMovedForward(null, position); + } } } @@ -148,4 +151,11 @@ public long getAbortedTxnCount() { public long getCommittedTxnCount() { return 0; } + + // ThreadSafe + private void updateLastDispatchablePosition(Position position) { + if (topic instanceof PersistentTopic t) { + t.updateLastDispatchablePosition(position); + } + } } diff --git a/pulsar-broker/src/main/java/org/apache/pulsar/zookeeper/LocalBookkeeperEnsemble.java b/pulsar-broker/src/main/java/org/apache/pulsar/zookeeper/LocalBookkeeperEnsemble.java index 63d146a3a1521..4c8d2dbbfa7d3 100644 --- a/pulsar-broker/src/main/java/org/apache/pulsar/zookeeper/LocalBookkeeperEnsemble.java +++ b/pulsar-broker/src/main/java/org/apache/pulsar/zookeeper/LocalBookkeeperEnsemble.java @@ -194,6 +194,7 @@ private void runZookeeper(int maxCC) throws IOException { : createTempDirectory("zktest"); if (this.clearOldData) { + LOG.info("Wiping Zookeeper data directory at {}", zkDataDir.getAbsolutePath()); cleanDirectory(zkDataDir); } @@ -291,6 +292,7 @@ private void runBookies(ServerConfiguration baseConf) throws Exception { : createTempDirectory("bk" + i + "test"); if (this.clearOldData) { + LOG.info("Wiping Bookie data directory at {}", bkDataDir.getAbsolutePath()); cleanDirectory(bkDataDir); } diff --git a/pulsar-broker/src/test/java/org/apache/pulsar/PulsarStandaloneTest.java b/pulsar-broker/src/test/java/org/apache/pulsar/PulsarStandaloneTest.java index 6ed93a75a3fb5..3d22feb822e32 100644 --- a/pulsar-broker/src/test/java/org/apache/pulsar/PulsarStandaloneTest.java +++ b/pulsar-broker/src/test/java/org/apache/pulsar/PulsarStandaloneTest.java @@ -31,6 +31,7 @@ import org.apache.pulsar.client.admin.PulsarAdmin; import org.apache.pulsar.common.policies.data.ClusterData; import org.apache.pulsar.common.policies.data.TenantInfo; +import org.apache.pulsar.metadata.bookkeeper.BKCluster; import org.testng.Assert; import org.testng.annotations.DataProvider; import org.testng.annotations.Test; @@ -46,12 +47,15 @@ public Object[][] enableBrokerClientAuth() { @Test public void testStandaloneWithRocksDB() throws Exception { String[] args = new String[]{"--config", - "./src/test/resources/configurations/pulsar_broker_test_standalone_with_rocksdb.conf"}; + "./src/test/resources/configurations/pulsar_broker_test_standalone_with_rocksdb.conf", + "-nss", + "-nfw"}; final int bookieNum = 3; final File tempDir = IOUtils.createTempDir("standalone", "test"); PulsarStandaloneStarter standalone = new PulsarStandaloneStarter(args); standalone.setBkDir(tempDir.getAbsolutePath()); + standalone.setBkPort(0); standalone.setNumOfBk(bookieNum); standalone.startBookieWithMetadataStore(); @@ -90,11 +94,12 @@ public void testMetadataInitialization(boolean enableBrokerClientAuth) throws Ex } final File bkDir = IOUtils.createTempDir("standalone", "bk"); standalone.setNumOfBk(1); + standalone.setBkPort(0); standalone.setBkDir(bkDir.getAbsolutePath()); standalone.start(); @Cleanup PulsarAdmin admin = PulsarAdmin.builder() - .serviceHttpUrl("http://localhost:8080") + .serviceHttpUrl(standalone.getWebServiceUrl()) .authentication(new MockTokenAuthenticationProvider.MockAuthentication()) .build(); if (enableBrokerClientAuth) { @@ -104,8 +109,8 @@ public void testMetadataInitialization(boolean enableBrokerClientAuth) throws Ex } else { assertTrue(admin.clusters().getClusters().isEmpty()); admin.clusters().createCluster("test_cluster", ClusterData.builder() - .serviceUrl("http://localhost:8080/") - .brokerServiceUrl("pulsar://localhost:6650/") + .serviceUrl(standalone.getWebServiceUrl()) + .brokerServiceUrl(standalone.getBrokerServiceUrl()) .build()); assertTrue(admin.tenants().getTenants().isEmpty()); admin.tenants().createTenant("public", TenantInfo.builder() @@ -125,4 +130,39 @@ public void testMetadataInitialization(boolean enableBrokerClientAuth) throws Ex cleanDirectory(bkDir); cleanDirectory(metadataDir); } + + + @Test + public void testShutdownHookClosesBkCluster() throws Exception { + File dataDir = IOUtils.createTempDir("data", ""); + File metadataDir = new File(dataDir, "metadata"); + File bkDir = new File(dataDir, "bookkeeper"); + @Cleanup + PulsarStandaloneStarter standalone = new PulsarStandaloneStarter(new String[] { + "--config", + "./src/test/resources/configurations/pulsar_broker_test_standalone_with_rocksdb.conf", + "-nss", + "-nfw", + "--metadata-dir", + metadataDir.getAbsolutePath(), + "--bookkeeper-dir", + bkDir.getAbsolutePath() + }); + standalone.setTestMode(true); + standalone.setBkPort(0); + standalone.start(); + BKCluster bkCluster = standalone.bkCluster; + standalone.runShutdownHook(); + assertTrue(bkCluster.isClosed()); + } + + @Test + public void testWipeData() throws Exception { + PulsarStandaloneStarter standalone = new PulsarStandaloneStarter(new String[] { + "--config", + "./src/test/resources/configurations/standalone_no_client_auth.conf", + "--wipe-data" + }); + assertTrue(standalone.isWipeData()); + } } diff --git a/pulsar-broker/src/test/java/org/apache/pulsar/broker/admin/GetPartitionMetadataMultiBrokerTest.java b/pulsar-broker/src/test/java/org/apache/pulsar/broker/admin/GetPartitionMetadataMultiBrokerTest.java new file mode 100644 index 0000000000000..28cf91ee165e2 --- /dev/null +++ b/pulsar-broker/src/test/java/org/apache/pulsar/broker/admin/GetPartitionMetadataMultiBrokerTest.java @@ -0,0 +1,222 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. The ASF licenses this file + * to you under the Apache License, Version 2.0 (the + * "License"); you may not use this file except in compliance + * with the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, + * software distributed under the License is distributed on an + * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY + * KIND, either express or implied. See the License for the + * specific language governing permissions and limitations + * under the License. + */ +package org.apache.pulsar.broker.admin; + +import static org.testng.Assert.assertFalse; +import static org.testng.Assert.assertTrue; +import java.net.URL; +import java.util.List; +import lombok.extern.slf4j.Slf4j; +import org.apache.pulsar.broker.PulsarService; +import org.apache.pulsar.client.admin.PulsarAdmin; +import org.apache.pulsar.client.api.PulsarClient; +import org.apache.pulsar.client.impl.PulsarClientImpl; +import org.apache.pulsar.common.naming.TopicDomain; +import org.apache.pulsar.common.naming.TopicName; +import org.apache.pulsar.common.policies.data.TopicType; +import org.testng.annotations.AfterClass; +import org.testng.annotations.BeforeClass; +import org.testng.annotations.Test; + +@Test(groups = "broker-admin") +@Slf4j +public class GetPartitionMetadataMultiBrokerTest extends GetPartitionMetadataTest { + + private PulsarService pulsar2; + private URL url2; + private PulsarAdmin admin2; + private PulsarClientImpl clientWithHttpLookup2; + private PulsarClientImpl clientWitBinaryLookup2; + + @BeforeClass(alwaysRun = true) + protected void setup() throws Exception { + super.setup(); + } + + @Override + @AfterClass(alwaysRun = true) + protected void cleanup() throws Exception { + super.cleanup(); + } + + @Override + protected void cleanupBrokers() throws Exception { + // Cleanup broker2. + if (clientWithHttpLookup2 != null) { + clientWithHttpLookup2.close(); + clientWithHttpLookup2 = null; + } + if (clientWitBinaryLookup2 != null) { + clientWitBinaryLookup2.close(); + clientWitBinaryLookup2 = null; + } + if (admin2 != null) { + admin2.close(); + admin2 = null; + } + if (pulsar2 != null) { + pulsar2.close(); + pulsar2 = null; + } + + // Super cleanup. + super.cleanupBrokers(); + } + + @Override + protected void setupBrokers() throws Exception { + super.setupBrokers(); + doInitConf(); + pulsar2 = new PulsarService(conf); + pulsar2.start(); + url2 = new URL(pulsar2.getWebServiceAddress()); + admin2 = PulsarAdmin.builder().serviceHttpUrl(url2.toString()).build(); + clientWithHttpLookup2 = + (PulsarClientImpl) PulsarClient.builder().serviceUrl(pulsar2.getWebServiceAddress()).build(); + clientWitBinaryLookup2 = + (PulsarClientImpl) PulsarClient.builder().serviceUrl(pulsar2.getBrokerServiceUrl()).build(); + } + + @Override + protected PulsarClientImpl[] getClientsToTest() { + return new PulsarClientImpl[] {clientWithHttpLookup1, clientWitBinaryLookup1, + clientWithHttpLookup2, clientWitBinaryLookup2}; + } + + protected PulsarClientImpl[] getClientsToTest(boolean isUsingHttpLookup) { + if (isUsingHttpLookup) { + return new PulsarClientImpl[]{clientWithHttpLookup1, clientWithHttpLookup2}; + } else { + return new PulsarClientImpl[]{clientWitBinaryLookup1, clientWitBinaryLookup2}; + } + } + + @Override + protected int getLookupRequestPermits() { + return pulsar1.getBrokerService().getLookupRequestSemaphore().availablePermits() + + pulsar2.getBrokerService().getLookupRequestSemaphore().availablePermits(); + } + + protected void verifyPartitionsNeverCreated(String topicNameStr) throws Exception { + TopicName topicName = TopicName.get(topicNameStr); + try { + List topicList = admin1.topics().getList("public/default"); + for (int i = 0; i < 3; i++) { + assertFalse(topicList.contains(topicName.getPartition(i))); + } + } catch (Exception ex) { + // If the namespace bundle has not been loaded yet, it means no non-persistent topic was created. So + // this behavior is also correct. + // This error is not expected, a seperated PR is needed to fix this issue. + assertTrue(ex.getMessage().contains("Failed to find ownership for")); + } + } + + protected void verifyNonPartitionedTopicNeverCreated(String topicNameStr) throws Exception { + TopicName topicName = TopicName.get(topicNameStr); + try { + List topicList = admin1.topics().getList("public/default"); + assertFalse(topicList.contains(topicName.getPartitionedTopicName())); + } catch (Exception ex) { + // If the namespace bundle has not been loaded yet, it means no non-persistent topic was created. So + // this behavior is also correct. + // This error is not expected, a seperated PR is needed to fix this issue. + assertTrue(ex.getMessage().contains("Failed to find ownership for")); + } + } + + protected void modifyTopicAutoCreation(boolean allowAutoTopicCreation, + TopicType allowAutoTopicCreationType, + int defaultNumPartitions) throws Exception { + doModifyTopicAutoCreation(admin1, pulsar1, allowAutoTopicCreation, allowAutoTopicCreationType, + defaultNumPartitions); + doModifyTopicAutoCreation(admin2, pulsar2, allowAutoTopicCreation, allowAutoTopicCreationType, + defaultNumPartitions); + } + + /** + * {@inheritDoc} + */ + @Test(dataProvider = "topicDomains") + public void testAutoCreatingMetadataWhenCallingOldAPI(TopicDomain topicDomain) throws Exception { + super.testAutoCreatingMetadataWhenCallingOldAPI(topicDomain); + } + + /** + * {@inheritDoc} + */ + @Test(dataProvider = "autoCreationParamsAll", enabled = false) + public void testGetMetadataIfNonPartitionedTopicExists(boolean configAllowAutoTopicCreation, + boolean paramMetadataAutoCreationEnabled, + boolean isUsingHttpLookup, + TopicDomain topicDomain) throws Exception { + super.testGetMetadataIfNonPartitionedTopicExists(configAllowAutoTopicCreation, paramMetadataAutoCreationEnabled, + isUsingHttpLookup, topicDomain); + } + + /** + * {@inheritDoc} + */ + @Test(dataProvider = "autoCreationParamsAll") + public void testGetMetadataIfPartitionedTopicExists(boolean configAllowAutoTopicCreation, + boolean paramMetadataAutoCreationEnabled, + boolean isUsingHttpLookup, + TopicDomain topicDomain) throws Exception { + super.testGetMetadataIfNonPartitionedTopicExists(configAllowAutoTopicCreation, paramMetadataAutoCreationEnabled, + isUsingHttpLookup, topicDomain); + } + + /** + * {@inheritDoc} + */ + @Test(dataProvider = "clients") + public void testAutoCreatePartitionedTopic(boolean isUsingHttpLookup, TopicDomain topicDomain) throws Exception { + super.testAutoCreatePartitionedTopic(isUsingHttpLookup, topicDomain); + } + + /** + * {@inheritDoc} + */ + @Test(dataProvider = "clients") + public void testAutoCreateNonPartitionedTopic(boolean isUsingHttpLookup, TopicDomain topicDomain) throws Exception { + super.testAutoCreateNonPartitionedTopic(isUsingHttpLookup, topicDomain); + } + + /** + * {@inheritDoc} + */ + @Test(dataProvider = "autoCreationParamsNotAllow") + public void testGetMetadataIfNotAllowedCreate(boolean configAllowAutoTopicCreation, + boolean paramMetadataAutoCreationEnabled, + boolean isUsingHttpLookup) throws Exception { + super.testGetMetadataIfNotAllowedCreate(configAllowAutoTopicCreation, paramMetadataAutoCreationEnabled, + isUsingHttpLookup); + } + + /** + * {@inheritDoc} + */ + @Test(dataProvider = "autoCreationParamsNotAllow") + public void testGetMetadataIfNotAllowedCreateOfNonPersistentTopic(boolean configAllowAutoTopicCreation, + boolean paramMetadataAutoCreationEnabled, + boolean isUsingHttpLookup) throws Exception { + super.testGetMetadataIfNotAllowedCreateOfNonPersistentTopic(configAllowAutoTopicCreation, + paramMetadataAutoCreationEnabled, isUsingHttpLookup); + } +} diff --git a/pulsar-broker/src/test/java/org/apache/pulsar/broker/admin/GetPartitionMetadataTest.java b/pulsar-broker/src/test/java/org/apache/pulsar/broker/admin/GetPartitionMetadataTest.java new file mode 100644 index 0000000000000..bf99b172829a7 --- /dev/null +++ b/pulsar-broker/src/test/java/org/apache/pulsar/broker/admin/GetPartitionMetadataTest.java @@ -0,0 +1,523 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. The ASF licenses this file + * to you under the Apache License, Version 2.0 (the + * "License"); you may not use this file except in compliance + * with the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, + * software distributed under the License is distributed on an + * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY + * KIND, either express or implied. See the License for the + * specific language governing permissions and limitations + * under the License. + */ +package org.apache.pulsar.broker.admin; + +import static org.testng.Assert.assertEquals; +import static org.testng.Assert.assertFalse; +import static org.testng.Assert.assertTrue; +import static org.testng.Assert.fail; +import com.google.common.collect.Sets; +import java.net.URL; +import java.util.Collections; +import java.util.List; +import java.util.Optional; +import lombok.extern.slf4j.Slf4j; +import org.apache.pulsar.broker.BrokerTestUtil; +import org.apache.pulsar.broker.PulsarService; +import org.apache.pulsar.broker.ServiceConfiguration; +import org.apache.pulsar.client.admin.PulsarAdmin; +import org.apache.pulsar.client.api.PulsarClient; +import org.apache.pulsar.client.api.PulsarClientException; +import org.apache.pulsar.client.impl.PulsarClientImpl; +import org.apache.pulsar.common.naming.NamespaceName; +import org.apache.pulsar.common.naming.TopicDomain; +import org.apache.pulsar.common.naming.TopicName; +import org.apache.pulsar.common.partition.PartitionedTopicMetadata; +import org.apache.pulsar.common.policies.data.ClusterDataImpl; +import org.apache.pulsar.common.policies.data.TenantInfoImpl; +import org.apache.pulsar.common.policies.data.TopicType; +import org.apache.pulsar.common.util.FutureUtil; +import org.apache.pulsar.zookeeper.LocalBookkeeperEnsemble; +import org.awaitility.Awaitility; +import org.testng.annotations.AfterClass; +import org.testng.annotations.BeforeClass; +import org.testng.annotations.DataProvider; +import org.testng.annotations.Test; + +@Test(groups = "broker-admin") +@Slf4j +public class GetPartitionMetadataTest { + + protected static final String DEFAULT_NS = "public/default"; + + protected String clusterName = "c1"; + + protected LocalBookkeeperEnsemble bkEnsemble; + + protected ServiceConfiguration conf = new ServiceConfiguration(); + + protected PulsarService pulsar1; + protected URL url1; + protected PulsarAdmin admin1; + protected PulsarClientImpl clientWithHttpLookup1; + protected PulsarClientImpl clientWitBinaryLookup1; + + @BeforeClass(alwaysRun = true) + protected void setup() throws Exception { + bkEnsemble = new LocalBookkeeperEnsemble(3, 0, () -> 0); + bkEnsemble.start(); + // Start broker. + setupBrokers(); + // Create default NS. + admin1.clusters().createCluster(clusterName, new ClusterDataImpl()); + admin1.tenants().createTenant(NamespaceName.get(DEFAULT_NS).getTenant(), + new TenantInfoImpl(Collections.emptySet(), Sets.newHashSet(clusterName))); + admin1.namespaces().createNamespace(DEFAULT_NS); + } + + @AfterClass(alwaysRun = true) + protected void cleanup() throws Exception { + cleanupBrokers(); + if (bkEnsemble != null) { + bkEnsemble.stop(); + bkEnsemble = null; + } + } + + protected void cleanupBrokers() throws Exception { + // Cleanup broker2. + if (clientWithHttpLookup1 != null) { + clientWithHttpLookup1.close(); + clientWithHttpLookup1 = null; + } + if (clientWitBinaryLookup1 != null) { + clientWitBinaryLookup1.close(); + clientWitBinaryLookup1 = null; + } + if (admin1 != null) { + admin1.close(); + admin1 = null; + } + if (pulsar1 != null) { + pulsar1.close(); + pulsar1 = null; + } + // Reset configs. + conf = new ServiceConfiguration(); + } + + protected void setupBrokers() throws Exception { + doInitConf(); + // Start broker. + pulsar1 = new PulsarService(conf); + pulsar1.start(); + url1 = new URL(pulsar1.getWebServiceAddress()); + admin1 = PulsarAdmin.builder().serviceHttpUrl(url1.toString()).build(); + clientWithHttpLookup1 = + (PulsarClientImpl) PulsarClient.builder().serviceUrl(pulsar1.getWebServiceAddress()).build(); + clientWitBinaryLookup1 = + (PulsarClientImpl) PulsarClient.builder().serviceUrl(pulsar1.getBrokerServiceUrl()).build(); + } + + protected void doInitConf() { + conf.setClusterName(clusterName); + conf.setAdvertisedAddress("localhost"); + conf.setBrokerServicePort(Optional.of(0)); + conf.setWebServicePort(Optional.of(0)); + conf.setMetadataStoreUrl("zk:127.0.0.1:" + bkEnsemble.getZookeeperPort()); + conf.setConfigurationMetadataStoreUrl("zk:127.0.0.1:" + bkEnsemble.getZookeeperPort() + "/foo"); + conf.setBrokerDeleteInactiveTopicsEnabled(false); + conf.setBrokerShutdownTimeoutMs(0L); + conf.setLoadBalancerSheddingEnabled(false); + } + + protected PulsarClientImpl[] getClientsToTest() { + return new PulsarClientImpl[] {clientWithHttpLookup1, clientWitBinaryLookup1}; + } + + protected PulsarClientImpl[] getClientsToTest(boolean isUsingHttpLookup) { + if (isUsingHttpLookup) { + return new PulsarClientImpl[] {clientWithHttpLookup1}; + } else { + return new PulsarClientImpl[] {clientWitBinaryLookup1}; + } + + } + + protected int getLookupRequestPermits() { + return pulsar1.getBrokerService().getLookupRequestSemaphore().availablePermits(); + } + + protected void verifyPartitionsNeverCreated(String topicNameStr) throws Exception { + TopicName topicName = TopicName.get(topicNameStr); + List topicList = admin1.topics().getList("public/default"); + for (int i = 0; i < 3; i++) { + assertFalse(topicList.contains(topicName.getPartition(i))); + } + } + + protected void verifyNonPartitionedTopicNeverCreated(String topicNameStr) throws Exception { + TopicName topicName = TopicName.get(topicNameStr); + List topicList = admin1.topics().getList("public/default"); + assertFalse(topicList.contains(topicName.getPartitionedTopicName())); + } + + @DataProvider(name = "topicDomains") + public Object[][] topicDomains() { + return new Object[][]{ + {TopicDomain.persistent}, + {TopicDomain.non_persistent} + }; + } + + protected static void doModifyTopicAutoCreation(PulsarAdmin admin1, PulsarService pulsar1, + boolean allowAutoTopicCreation, TopicType allowAutoTopicCreationType, + int defaultNumPartitions) throws Exception { + admin1.brokers().updateDynamicConfiguration( + "allowAutoTopicCreation", allowAutoTopicCreation + ""); + admin1.brokers().updateDynamicConfiguration( + "allowAutoTopicCreationType", allowAutoTopicCreationType + ""); + admin1.brokers().updateDynamicConfiguration( + "defaultNumPartitions", defaultNumPartitions + ""); + Awaitility.await().untilAsserted(() -> { + assertEquals(pulsar1.getConfiguration().isAllowAutoTopicCreation(), allowAutoTopicCreation); + assertEquals(pulsar1.getConfiguration().getAllowAutoTopicCreationType(), allowAutoTopicCreationType); + assertEquals(pulsar1.getConfiguration().getDefaultNumPartitions(), defaultNumPartitions); + }); + } + + protected void modifyTopicAutoCreation(boolean allowAutoTopicCreation, + TopicType allowAutoTopicCreationType, + int defaultNumPartitions) throws Exception { + doModifyTopicAutoCreation(admin1, pulsar1, allowAutoTopicCreation, allowAutoTopicCreationType, + defaultNumPartitions); + } + + @Test(dataProvider = "topicDomains") + public void testAutoCreatingMetadataWhenCallingOldAPI(TopicDomain topicDomain) throws Exception { + modifyTopicAutoCreation(true, TopicType.PARTITIONED, 3); + + int lookupPermitsBefore = getLookupRequestPermits(); + + for (PulsarClientImpl client : getClientsToTest()) { + // Verify: the behavior of topic creation. + final String tp = BrokerTestUtil.newUniqueName(topicDomain.value() + "://" + DEFAULT_NS + "/tp"); + client.getPartitionsForTopic(tp).join(); + Optional metadata1 = pulsar1.getPulsarResources().getNamespaceResources() + .getPartitionedTopicResources() + .getPartitionedTopicMetadataAsync(TopicName.get(tp), true).join(); + assertTrue(metadata1.isPresent()); + assertEquals(metadata1.get().partitions, 3); + + // Verify: lookup semaphore has been releases. + Awaitility.await().untilAsserted(() -> { + assertEquals(getLookupRequestPermits(), lookupPermitsBefore); + }); + + // Cleanup. + admin1.topics().deletePartitionedTopic(tp, false); + } + } + + @DataProvider(name = "autoCreationParamsAll") + public Object[][] autoCreationParamsAll(){ + return new Object[][]{ + // configAllowAutoTopicCreation, paramCreateIfAutoCreationEnabled, isUsingHttpLookup. + {true, true, true, TopicDomain.persistent}, + {true, true, false, TopicDomain.persistent}, + {true, false, true, TopicDomain.persistent}, + {true, false, false, TopicDomain.persistent}, + {false, true, true, TopicDomain.persistent}, + {false, true, false, TopicDomain.persistent}, + {false, false, true, TopicDomain.persistent}, + {false, false, false, TopicDomain.persistent}, + {true, true, true, TopicDomain.non_persistent}, + {true, true, false, TopicDomain.non_persistent}, + {true, false, true, TopicDomain.non_persistent}, + {true, false, false, TopicDomain.non_persistent}, + {false, true, true, TopicDomain.non_persistent}, + {false, true, false, TopicDomain.non_persistent}, + {false, false, true, TopicDomain.non_persistent}, + {false, false, false, TopicDomain.non_persistent} + }; + } + + @Test(dataProvider = "autoCreationParamsAll") + public void testGetMetadataIfNonPartitionedTopicExists(boolean configAllowAutoTopicCreation, + boolean paramMetadataAutoCreationEnabled, + boolean isUsingHttpLookup, + TopicDomain topicDomain) throws Exception { + modifyTopicAutoCreation(configAllowAutoTopicCreation, TopicType.PARTITIONED, 3); + + int lookupPermitsBefore = getLookupRequestPermits(); + + // Create topic. + final String topicNameStr = BrokerTestUtil.newUniqueName(topicDomain.value() + "://" + DEFAULT_NS + "/tp_"); + admin1.topics().createNonPartitionedTopic(topicNameStr); + + PulsarClientImpl[] clientArray = getClientsToTest(isUsingHttpLookup); + for (PulsarClientImpl client : clientArray) { + // Verify: the result of get partitioned topic metadata. + PartitionedTopicMetadata response = + client.getPartitionedTopicMetadata(topicNameStr, paramMetadataAutoCreationEnabled).join(); + assertEquals(response.partitions, 0); + List partitionedTopics = admin1.topics().getPartitionedTopicList("public/default"); + assertFalse(partitionedTopics.contains(topicNameStr)); + verifyPartitionsNeverCreated(topicNameStr); + + // Verify: lookup semaphore has been releases. + Awaitility.await().untilAsserted(() -> { + assertEquals(getLookupRequestPermits(), lookupPermitsBefore); + }); + } + + // Cleanup. + admin1.topics().delete(topicNameStr, false); + } + + @Test(dataProvider = "autoCreationParamsAll") + public void testGetMetadataIfPartitionedTopicExists(boolean configAllowAutoTopicCreation, + boolean paramMetadataAutoCreationEnabled, + boolean isUsingHttpLookup, + TopicDomain topicDomain) throws Exception { + modifyTopicAutoCreation(configAllowAutoTopicCreation, TopicType.PARTITIONED, 3); + + int lookupPermitsBefore = getLookupRequestPermits(); + + // Create topic. + final String topicNameStr = BrokerTestUtil.newUniqueName(topicDomain.value() + "://" + DEFAULT_NS + "/tp"); + admin1.topics().createPartitionedTopic(topicNameStr, 3); + + PulsarClientImpl[] clientArray = getClientsToTest(isUsingHttpLookup); + for (PulsarClientImpl client : clientArray) { + // Verify: the result of get partitioned topic metadata. + PartitionedTopicMetadata response = + client.getPartitionedTopicMetadata(topicNameStr, paramMetadataAutoCreationEnabled).join(); + assertEquals(response.partitions, 3); + verifyNonPartitionedTopicNeverCreated(topicNameStr); + + // Verify: lookup semaphore has been releases. + Awaitility.await().untilAsserted(() -> { + assertEquals(getLookupRequestPermits(), lookupPermitsBefore); + }); + } + + // Cleanup. + admin1.topics().deletePartitionedTopic(topicNameStr, false); + } + + @DataProvider(name = "clients") + public Object[][] clients(){ + return new Object[][]{ + // isUsingHttpLookup. + {true, TopicDomain.persistent}, + {false, TopicDomain.non_persistent} + }; + } + + @Test(dataProvider = "clients") + public void testAutoCreatePartitionedTopic(boolean isUsingHttpLookup, TopicDomain topicDomain) throws Exception { + modifyTopicAutoCreation(true, TopicType.PARTITIONED, 3); + + int lookupPermitsBefore = getLookupRequestPermits(); + + PulsarClientImpl[] clientArray = getClientsToTest(isUsingHttpLookup); + for (PulsarClientImpl client : clientArray) { + // Case-1: normal topic. + final String topicNameStr = BrokerTestUtil.newUniqueName(topicDomain.value() + "://" + DEFAULT_NS + "/tp"); + // Verify: the result of get partitioned topic metadata. + PartitionedTopicMetadata response = client.getPartitionedTopicMetadata(topicNameStr, true).join(); + assertEquals(response.partitions, 3); + // Verify: the behavior of topic creation. + List partitionedTopics = admin1.topics().getPartitionedTopicList("public/default"); + assertTrue(partitionedTopics.contains(topicNameStr)); + verifyNonPartitionedTopicNeverCreated(topicNameStr); + // The API "getPartitionedTopicMetadata" only creates the partitioned metadata, it will not create the + // partitions. + verifyPartitionsNeverCreated(topicNameStr); + + // Case-2: topic with suffix "-partition-1". + final String topicNameStrWithSuffix = BrokerTestUtil.newUniqueName( + topicDomain.value() + "://" + DEFAULT_NS + "/tp") + "-partition-1"; + // Verify: the result of get partitioned topic metadata. + PartitionedTopicMetadata response2 = + client.getPartitionedTopicMetadata(topicNameStrWithSuffix, true).join(); + assertEquals(response2.partitions, 0); + // Verify: the behavior of topic creation. + List partitionedTopics2 = + admin1.topics().getPartitionedTopicList("public/default"); + assertFalse(partitionedTopics2.contains(topicNameStrWithSuffix)); + assertFalse(partitionedTopics2.contains( + TopicName.get(topicNameStrWithSuffix).getPartitionedTopicName())); + + // Verify: lookup semaphore has been releases. + Awaitility.await().untilAsserted(() -> { + assertEquals(getLookupRequestPermits(), lookupPermitsBefore); + }); + // Cleanup. + admin1.topics().deletePartitionedTopic(topicNameStr, false); + try { + admin1.topics().delete(topicNameStrWithSuffix, false); + } catch (Exception ex) {} + } + + } + + @Test(dataProvider = "clients") + public void testAutoCreateNonPartitionedTopic(boolean isUsingHttpLookup, TopicDomain topicDomain) throws Exception { + modifyTopicAutoCreation(true, TopicType.NON_PARTITIONED, 3); + + int lookupPermitsBefore = getLookupRequestPermits(); + + PulsarClientImpl[] clientArray = getClientsToTest(isUsingHttpLookup); + for (PulsarClientImpl client : clientArray) { + // Case 1: normal topic. + final String topicNameStr = BrokerTestUtil.newUniqueName(topicDomain.value() + "://" + DEFAULT_NS + "/tp"); + // Verify: the result of get partitioned topic metadata. + PartitionedTopicMetadata response = client.getPartitionedTopicMetadata(topicNameStr, true).join(); + assertEquals(response.partitions, 0); + // Verify: the behavior of topic creation. + List partitionedTopics = admin1.topics().getPartitionedTopicList("public/default"); + assertFalse(partitionedTopics.contains(topicNameStr)); + verifyPartitionsNeverCreated(topicNameStr); + + // Case-2: topic with suffix "-partition-1". + final String topicNameStrWithSuffix = BrokerTestUtil.newUniqueName( + topicDomain.value() + "://" + DEFAULT_NS + "/tp") + "-partition-1"; + // Verify: the result of get partitioned topic metadata. + PartitionedTopicMetadata response2 = + client.getPartitionedTopicMetadata(topicNameStrWithSuffix, true).join(); + assertEquals(response2.partitions, 0); + // Verify: the behavior of topic creation. + List partitionedTopics2 = + admin1.topics().getPartitionedTopicList("public/default"); + assertFalse(partitionedTopics2.contains(topicNameStrWithSuffix)); + assertFalse(partitionedTopics2.contains( + TopicName.get(topicNameStrWithSuffix).getPartitionedTopicName())); + + // Verify: lookup semaphore has been releases. + Awaitility.await().untilAsserted(() -> { + assertEquals(getLookupRequestPermits(), lookupPermitsBefore); + }); + // Cleanup. + try { + admin1.topics().delete(topicNameStr, false); + } catch (Exception ex) {} + try { + admin1.topics().delete(topicNameStrWithSuffix, false); + } catch (Exception ex) {} + } + } + + @DataProvider(name = "autoCreationParamsNotAllow") + public Object[][] autoCreationParamsNotAllow(){ + return new Object[][]{ + // configAllowAutoTopicCreation, paramCreateIfAutoCreationEnabled, isUsingHttpLookup. + {true, false, true}, + {true, false, false}, + {false, false, true}, + {false, false, false}, + {false, true, true}, + {false, true, false}, + }; + } + + @Test(dataProvider = "autoCreationParamsNotAllow") + public void testGetMetadataIfNotAllowedCreate(boolean configAllowAutoTopicCreation, + boolean paramMetadataAutoCreationEnabled, + boolean isUsingHttpLookup) throws Exception { + modifyTopicAutoCreation(configAllowAutoTopicCreation, TopicType.PARTITIONED, 3); + + int lookupPermitsBefore = getLookupRequestPermits(); + + PulsarClientImpl[] clientArray = getClientsToTest(isUsingHttpLookup); + for (PulsarClientImpl client : clientArray) { + // Define topic. + final String topicNameStr = BrokerTestUtil.newUniqueName("persistent://" + DEFAULT_NS + "/tp"); + final TopicName topicName = TopicName.get(topicNameStr); + // Verify: the result of get partitioned topic metadata. + try { + client.getPartitionedTopicMetadata(topicNameStr, paramMetadataAutoCreationEnabled) + .join(); + fail("Expect a not found exception"); + } catch (Exception e) { + Throwable unwrapEx = FutureUtil.unwrapCompletionException(e); + assertTrue(unwrapEx instanceof PulsarClientException.TopicDoesNotExistException + || unwrapEx instanceof PulsarClientException.NotFoundException); + } + // Verify: the behavior of topic creation. + List partitionedTopics = admin1.topics().getPartitionedTopicList("public/default"); + pulsar1.getPulsarResources().getNamespaceResources().getPartitionedTopicResources() + .partitionedTopicExists(topicName); + assertFalse(partitionedTopics.contains(topicNameStr)); + verifyNonPartitionedTopicNeverCreated(topicNameStr); + verifyPartitionsNeverCreated(topicNameStr); + + // Verify: lookup semaphore has been releases. + Awaitility.await().untilAsserted(() -> { + assertEquals(getLookupRequestPermits(), lookupPermitsBefore); + }); + } + } + + /** + * Regarding the API "get partitioned metadata" about non-persistent topic. + * The original behavior is: + * param-auto-create = true, broker-config-auto-create = true + * HTTP API: default configuration {@link ServiceConfiguration#getDefaultNumPartitions()} + * binary API: default configuration {@link ServiceConfiguration#getDefaultNumPartitions()} + * param-auto-create = true, broker-config-auto-create = false + * HTTP API: {partitions: 0} + * binary API: {partitions: 0} + * param-auto-create = false + * HTTP API: not found error + * binary API: not support + * After PIP-344, the behavior will be the same as persistent topics, which was described in PIP-344. + */ + @Test(dataProvider = "autoCreationParamsNotAllow") + public void testGetMetadataIfNotAllowedCreateOfNonPersistentTopic(boolean configAllowAutoTopicCreation, + boolean paramMetadataAutoCreationEnabled, + boolean isUsingHttpLookup) throws Exception { + modifyTopicAutoCreation(configAllowAutoTopicCreation, TopicType.PARTITIONED, 3); + + int lookupPermitsBefore = getLookupRequestPermits(); + + PulsarClientImpl[] clientArray = getClientsToTest(isUsingHttpLookup); + for (PulsarClientImpl client : clientArray) { + // Define topic. + final String topicNameStr = BrokerTestUtil.newUniqueName("non-persistent://" + DEFAULT_NS + "/tp"); + final TopicName topicName = TopicName.get(topicNameStr); + // Verify: the result of get partitioned topic metadata. + try { + PartitionedTopicMetadata topicMetadata = client + .getPartitionedTopicMetadata(topicNameStr, paramMetadataAutoCreationEnabled) + .join(); + log.info("Get topic metadata: {}", topicMetadata.partitions); + fail("Expected a not found ex"); + } catch (Exception ex) { + Throwable unwrapEx = FutureUtil.unwrapCompletionException(ex); + assertTrue(unwrapEx instanceof PulsarClientException.TopicDoesNotExistException + || unwrapEx instanceof PulsarClientException.NotFoundException); + } + + // Verify: the behavior of topic creation. + List partitionedTopics = admin1.topics().getPartitionedTopicList("public/default"); + pulsar1.getPulsarResources().getNamespaceResources().getPartitionedTopicResources() + .partitionedTopicExists(topicName); + assertFalse(partitionedTopics.contains(topicNameStr)); + verifyNonPartitionedTopicNeverCreated(topicNameStr); + verifyPartitionsNeverCreated(topicNameStr); + } + + // Verify: lookup semaphore has been releases. + Awaitility.await().untilAsserted(() -> { + assertEquals(getLookupRequestPermits(), lookupPermitsBefore); + }); + } +} diff --git a/pulsar-broker/src/test/java/org/apache/pulsar/broker/admin/PulsarClientImplMultiBrokersTest.java b/pulsar-broker/src/test/java/org/apache/pulsar/broker/admin/PulsarClientImplMultiBrokersTest.java new file mode 100644 index 0000000000000..29604d0440b05 --- /dev/null +++ b/pulsar-broker/src/test/java/org/apache/pulsar/broker/admin/PulsarClientImplMultiBrokersTest.java @@ -0,0 +1,79 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. The ASF licenses this file + * to you under the Apache License, Version 2.0 (the + * "License"); you may not use this file except in compliance + * with the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, + * software distributed under the License is distributed on an + * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY + * KIND, either express or implied. See the License for the + * specific language governing permissions and limitations + * under the License. + */ +package org.apache.pulsar.broker.admin; + +import static org.testng.Assert.fail; +import static org.testng.AssertJUnit.assertEquals; +import static org.testng.AssertJUnit.assertTrue; +import java.util.Map; +import lombok.extern.slf4j.Slf4j; +import org.apache.pulsar.broker.MultiBrokerBaseTest; +import org.apache.pulsar.broker.PulsarService; +import org.apache.pulsar.client.impl.LookupService; +import org.apache.pulsar.client.impl.PulsarClientImpl; +import org.awaitility.reflect.WhiteboxImpl; +import org.testng.annotations.Test; + +/** + * Test multi-broker admin api. + */ +@Slf4j +@Test(groups = "broker-admin") +public class PulsarClientImplMultiBrokersTest extends MultiBrokerBaseTest { + @Override + protected int numberOfAdditionalBrokers() { + return 3; + } + + @Override + protected void doInitConf() throws Exception { + super.doInitConf(); + this.conf.setManagedLedgerMaxEntriesPerLedger(10); + } + + @Override + protected void onCleanup() { + super.onCleanup(); + } + + @Test(timeOut = 30 * 1000) + public void testReleaseUrlLookupServices() throws Exception { + PulsarClientImpl pulsarClient = (PulsarClientImpl) additionalBrokerClients.get(0); + Map urlLookupMap = WhiteboxImpl.getInternalState(pulsarClient, "urlLookupMap"); + assertEquals(urlLookupMap.size(), 0); + for (PulsarService pulsar : additionalBrokers) { + pulsarClient.getLookup(pulsar.getBrokerServiceUrl()); + pulsarClient.getLookup(pulsar.getWebServiceAddress()); + } + assertEquals(urlLookupMap.size(), additionalBrokers.size() * 2); + // Verify: lookup services will be release. + pulsarClient.close(); + assertEquals(urlLookupMap.size(), 0); + try { + for (PulsarService pulsar : additionalBrokers) { + pulsarClient.getLookup(pulsar.getBrokerServiceUrl()); + pulsarClient.getLookup(pulsar.getWebServiceAddress()); + } + fail("Expected a error when calling pulsarClient.getLookup if getLookup was closed"); + } catch (IllegalStateException illegalArgumentException) { + assertTrue(illegalArgumentException.getMessage().contains("has been closed")); + } + assertEquals(urlLookupMap.size(), 0); + } +} diff --git a/pulsar-broker/src/test/java/org/apache/pulsar/broker/admin/TopicAutoCreationTest.java b/pulsar-broker/src/test/java/org/apache/pulsar/broker/admin/TopicAutoCreationTest.java index a75ae78cef393..4712682e71b57 100644 --- a/pulsar-broker/src/test/java/org/apache/pulsar/broker/admin/TopicAutoCreationTest.java +++ b/pulsar-broker/src/test/java/org/apache/pulsar/broker/admin/TopicAutoCreationTest.java @@ -19,6 +19,7 @@ package org.apache.pulsar.broker.admin; import static org.mockito.ArgumentMatchers.any; +import static org.mockito.ArgumentMatchers.anyBoolean; import static org.mockito.Mockito.mock; import static org.mockito.Mockito.when; import static org.testng.Assert.assertEquals; @@ -133,7 +134,7 @@ public void testPartitionedTopicAutoCreationForbiddenDuringNamespaceDeletion() // we want to skip the "lookup" phase, because it is blocked by the HTTP API LookupService mockLookup = mock(LookupService.class); ((PulsarClientImpl) pulsarClient).setLookup(mockLookup); - when(mockLookup.getPartitionedTopicMetadata(any())).thenAnswer( + when(mockLookup.getPartitionedTopicMetadata(any(), anyBoolean())).thenAnswer( i -> CompletableFuture.completedFuture(new PartitionedTopicMetadata(0))); when(mockLookup.getBroker(any())).thenAnswer(i -> { InetSocketAddress brokerAddress = diff --git a/pulsar-broker/src/test/java/org/apache/pulsar/broker/admin/TopicsTest.java b/pulsar-broker/src/test/java/org/apache/pulsar/broker/admin/TopicsTest.java index 9aa29f08c5ce8..c9457e1a8883f 100644 --- a/pulsar-broker/src/test/java/org/apache/pulsar/broker/admin/TopicsTest.java +++ b/pulsar-broker/src/test/java/org/apache/pulsar/broker/admin/TopicsTest.java @@ -56,6 +56,7 @@ import org.apache.pulsar.broker.auth.MockedPulsarServiceBaseTest; import org.apache.pulsar.broker.authentication.AuthenticationDataHttps; import org.apache.pulsar.broker.namespace.NamespaceService; +import org.apache.pulsar.broker.namespace.TopicExistsInfo; import org.apache.pulsar.broker.rest.Topics; import org.apache.pulsar.broker.service.BrokerService; import org.apache.pulsar.broker.service.BrokerServiceException; @@ -357,9 +358,12 @@ public void testLookUpWithException() throws Exception { CompletableFuture future = new CompletableFuture(); future.completeExceptionally(new BrokerServiceException("Fake Exception")); CompletableFuture existFuture = new CompletableFuture(); - existFuture.complete(true); + existFuture.complete(TopicExistsInfo.newNonPartitionedTopicExists()); doReturn(future).when(nameSpaceService).getBrokerServiceUrlAsync(any(), any()); doReturn(existFuture).when(nameSpaceService).checkTopicExists(any()); + CompletableFuture existBooleanFuture = new CompletableFuture(); + existBooleanFuture.complete(false); + doReturn(existBooleanFuture).when(nameSpaceService).checkNonPartitionedTopicExists(any()); doReturn(nameSpaceService).when(pulsar).getNamespaceService(); AsyncResponse asyncResponse = mock(AsyncResponse.class); ProducerMessages producerMessages = new ProducerMessages(); @@ -370,7 +374,7 @@ public void testLookUpWithException() throws Exception { topics.produceOnPersistentTopic(asyncResponse, testTenant, testNamespace, testTopicName, false, producerMessages); ArgumentCaptor responseCaptor = ArgumentCaptor.forClass(RestException.class); verify(asyncResponse, timeout(5000).times(1)).resume(responseCaptor.capture()); - Assert.assertEquals(responseCaptor.getValue().getMessage(), "Can't find owner of given topic."); + Assert.assertTrue(responseCaptor.getValue().getMessage().contains(topicName + " not found")); } @Test @@ -378,8 +382,11 @@ public void testLookUpTopicNotExist() throws Exception { String topicName = "persistent://" + testTenant + "/" + testNamespace + "/" + testTopicName; NamespaceService nameSpaceService = mock(NamespaceService.class); CompletableFuture existFuture = new CompletableFuture(); - existFuture.complete(false); + existFuture.complete(TopicExistsInfo.newTopicNotExists()); + CompletableFuture existBooleanFuture = new CompletableFuture(); + existBooleanFuture.complete(false); doReturn(existFuture).when(nameSpaceService).checkTopicExists(any()); + doReturn(existBooleanFuture).when(nameSpaceService).checkNonPartitionedTopicExists(any()); doReturn(nameSpaceService).when(pulsar).getNamespaceService(); AsyncResponse asyncResponse = mock(AsyncResponse.class); ProducerMessages producerMessages = new ProducerMessages(); diff --git a/pulsar-broker/src/test/java/org/apache/pulsar/broker/lookup/http/HttpTopicLookupv2Test.java b/pulsar-broker/src/test/java/org/apache/pulsar/broker/lookup/http/HttpTopicLookupv2Test.java index 6a6065bc289f6..783a92b485ed8 100644 --- a/pulsar-broker/src/test/java/org/apache/pulsar/broker/lookup/http/HttpTopicLookupv2Test.java +++ b/pulsar-broker/src/test/java/org/apache/pulsar/broker/lookup/http/HttpTopicLookupv2Test.java @@ -44,6 +44,7 @@ import org.apache.pulsar.broker.lookup.RedirectData; import org.apache.pulsar.broker.lookup.v1.TopicLookup; import org.apache.pulsar.broker.namespace.NamespaceService; +import org.apache.pulsar.broker.namespace.TopicExistsInfo; import org.apache.pulsar.broker.resources.ClusterResources; import org.apache.pulsar.broker.resources.NamespaceResources; import org.apache.pulsar.broker.resources.PulsarResources; @@ -149,9 +150,12 @@ public void testLookupTopicNotExist() throws Exception { config.setAuthorizationEnabled(true); NamespaceService namespaceService = pulsar.getNamespaceService(); - CompletableFuture future = new CompletableFuture<>(); - future.complete(false); + CompletableFuture future = new CompletableFuture<>(); + future.complete(TopicExistsInfo.newTopicNotExists()); doReturn(future).when(namespaceService).checkTopicExists(any(TopicName.class)); + CompletableFuture booleanFuture = new CompletableFuture<>(); + booleanFuture.complete(false); + doReturn(booleanFuture).when(namespaceService).checkNonPartitionedTopicExists(any(TopicName.class)); AsyncResponse asyncResponse1 = mock(AsyncResponse.class); destLookup.lookupTopicAsync(asyncResponse1, TopicDomain.persistent.value(), "myprop", "usc", "ns2", "topic_not_exist", false, null, null); @@ -260,9 +264,12 @@ public void testValidateReplicationSettingsOnNamespace() throws Exception { policies3Future.complete(Optional.of(policies3)); doReturn(policies3Future).when(namespaceResources).getPoliciesAsync(namespaceName2); NamespaceService namespaceService = pulsar.getNamespaceService(); - CompletableFuture future = new CompletableFuture<>(); - future.complete(false); + CompletableFuture future = new CompletableFuture<>(); + future.complete(TopicExistsInfo.newTopicNotExists()); doReturn(future).when(namespaceService).checkTopicExists(any(TopicName.class)); + CompletableFuture booleanFuture = new CompletableFuture<>(); + booleanFuture.complete(false); + doReturn(future).when(namespaceService).checkNonPartitionedTopicExists(any(TopicName.class)); destLookup.lookupTopicAsync(asyncResponse, TopicDomain.persistent.value(), property, cluster, ns2, "invalid-localCluster", false, null, null); verify(asyncResponse).resume(arg.capture()); @@ -278,4 +285,35 @@ public void testDataPojo() { assertEquals(data2.getRedirectLookupAddress(), url); } + @Test + public void topicNotFound() throws Exception { + MockTopicLookup destLookup = spy(MockTopicLookup.class); + doReturn(false).when(destLookup).isRequestHttps(); + BrokerService brokerService = pulsar.getBrokerService(); + doReturn(new Semaphore(1000,true)).when(brokerService).getLookupRequestSemaphore(); + destLookup.setPulsar(pulsar); + doReturn("null").when(destLookup).clientAppId(); + Field uriField = PulsarWebResource.class.getDeclaredField("uri"); + uriField.setAccessible(true); + UriInfo uriInfo = mock(UriInfo.class); + uriField.set(destLookup, uriInfo); + URI uri = URI.create("http://localhost:8080/lookup/v2/destination/topic/myprop/usc/ns2/topic1"); + doReturn(uri).when(uriInfo).getRequestUri(); + config.setAuthorizationEnabled(true); + NamespaceService namespaceService = pulsar.getNamespaceService(); + CompletableFuture future = new CompletableFuture<>(); + future.complete(TopicExistsInfo.newTopicNotExists()); + doReturn(future).when(namespaceService).checkTopicExists(any(TopicName.class)); + + // Get the current semaphore first + Integer state1 = pulsar.getBrokerService().getLookupRequestSemaphore().availablePermits(); + AsyncResponse asyncResponse1 = mock(AsyncResponse.class); + // We used a nonexistent topic to test + destLookup.lookupTopicAsync(asyncResponse1, TopicDomain.persistent.value(), "myprop", "usc", "ns2", "topic2", false, null, null); + // Gets semaphore status + Integer state2 = pulsar.getBrokerService().getLookupRequestSemaphore().availablePermits(); + // If it is successfully released, it should be equal + assertEquals(state1, state2); + + } } diff --git a/pulsar-broker/src/test/java/org/apache/pulsar/broker/namespace/NamespaceServiceTest.java b/pulsar-broker/src/test/java/org/apache/pulsar/broker/namespace/NamespaceServiceTest.java index 2a8a849ef9c06..38a60165d5606 100644 --- a/pulsar-broker/src/test/java/org/apache/pulsar/broker/namespace/NamespaceServiceTest.java +++ b/pulsar-broker/src/test/java/org/apache/pulsar/broker/namespace/NamespaceServiceTest.java @@ -816,14 +816,15 @@ public void testCheckTopicExists(String topicDomain) throws Exception { String topic = topicDomain + "://prop/ns-abc/" + UUID.randomUUID(); admin.topics().createNonPartitionedTopic(topic); Awaitility.await().untilAsserted(() -> { - assertTrue(pulsar.getNamespaceService().checkTopicExists(TopicName.get(topic)).get()); + assertTrue(pulsar.getNamespaceService().checkTopicExists(TopicName.get(topic)).get().isExists()); }); String partitionedTopic = topicDomain + "://prop/ns-abc/" + UUID.randomUUID(); admin.topics().createPartitionedTopic(partitionedTopic, 5); Awaitility.await().untilAsserted(() -> { - assertTrue(pulsar.getNamespaceService().checkTopicExists(TopicName.get(partitionedTopic)).get()); - assertTrue(pulsar.getNamespaceService().checkTopicExists(TopicName.get(partitionedTopic + "-partition-2")).get()); + assertTrue(pulsar.getNamespaceService().checkTopicExists(TopicName.get(partitionedTopic)).get().isExists()); + assertTrue(pulsar.getNamespaceService() + .checkTopicExists(TopicName.get(partitionedTopic + "-partition-2")).get().isExists()); }); } diff --git a/pulsar-broker/src/test/java/org/apache/pulsar/broker/service/BacklogQuotaManagerTest.java b/pulsar-broker/src/test/java/org/apache/pulsar/broker/service/BacklogQuotaManagerTest.java index f194dfe1340e9..2bdd76469ef4d 100644 --- a/pulsar-broker/src/test/java/org/apache/pulsar/broker/service/BacklogQuotaManagerTest.java +++ b/pulsar-broker/src/test/java/org/apache/pulsar/broker/service/BacklogQuotaManagerTest.java @@ -27,6 +27,7 @@ import static org.assertj.core.api.AssertionsForClassTypes.within; import static org.testng.Assert.assertEquals; import static org.testng.Assert.assertFalse; +import static org.testng.Assert.assertNotNull; import static org.testng.Assert.assertTrue; import static org.testng.Assert.fail; import com.google.common.collect.Sets; @@ -284,8 +285,12 @@ public void testBacklogQuotaWithReader() throws Exception { } private TopicStats getTopicStats(String topic1) throws PulsarAdminException { + return getTopicStats(topic1, true); + } + + private TopicStats getTopicStats(String topic1, boolean getPreciseBacklog) throws PulsarAdminException { TopicStats stats = - admin.topics().getStats(topic1, GetStatsOptions.builder().getPreciseBacklog(true).build()); + admin.topics().getStats(topic1, GetStatsOptions.builder().getPreciseBacklog(getPreciseBacklog).build()); return stats; } @@ -490,9 +495,117 @@ public void backlogsStatsPrecise() throws PulsarAdminException, PulsarClientExce // Cache should be used, since position hasn't changed assertThat(getReadEntries(topic1)).isEqualTo(readEntries); + + // Move subscription 1 and 2 to end + Message msg = consumer1.receive(); + consumer1.acknowledge(msg); + consumer2.acknowledge(secondOldestMessage); + for (int i = 0; i < 2; i++) { + Message message = consumer2.receive(); + log.info("Subscription 2 about to ack message ID {}", message.getMessageId()); + consumer2.acknowledge(message); + } + + log.info("Subscription 1 and 2 moved to end. Now should not backlog"); + waitForMarkDeletePositionToChange(topic1, subName1, c1MarkDeletePositionBefore); + waitForQuotaCheckToRunTwice(); + + topicStats = getTopicStats(topic1); + assertThat(topicStats.getBacklogSize()).isEqualTo(0); + assertThat(topicStats.getSubscriptions().get(subName1).getMsgBacklog()).isEqualTo(0); + assertThat(topicStats.getSubscriptions().get(subName2).getMsgBacklog()).isEqualTo(0); + assertThat(topicStats.getOldestBacklogMessageAgeSeconds()).isEqualTo(0); + assertThat(topicStats.getOldestBacklogMessageSubscriptionName()).isNull(); + + metrics = prometheusMetricsClient.getMetrics(); + backlogAgeMetric = + metrics.findSingleMetricByNameAndLabels("pulsar_storage_backlog_age_seconds", + Pair.of("topic", topic1)); + assertThat(backlogAgeMetric.tags).containsExactly( + entry("cluster", CLUSTER_NAME), + entry("namespace", namespace), + entry("topic", topic1)); + assertThat((long) backlogAgeMetric.value).isEqualTo(0); + + // producer should create success. + Producer producer2 = createProducer(client, topic1); + assertNotNull(producer2); } } + @Test + public void backlogsStatsPreciseWithNoBacklog() throws PulsarAdminException, PulsarClientException, InterruptedException { + config.setPreciseTimeBasedBacklogQuotaCheck(true); + config.setExposePreciseBacklogInPrometheus(true); + final String namespace = "prop/ns-quota"; + assertEquals(admin.namespaces().getBacklogQuotaMap(namespace), new HashMap<>()); + final int timeLimitSeconds = 2; + admin.namespaces().setBacklogQuota( + namespace, + BacklogQuota.builder() + .limitTime(timeLimitSeconds) + .retentionPolicy(BacklogQuota.RetentionPolicy.producer_exception) + .build(), + message_age); + + try (PulsarClient client = PulsarClient.builder().serviceUrl(adminUrl.toString()) + .maxBackoffInterval(5, SECONDS) + .statsInterval(0, SECONDS).build()) { + final String topic1 = "persistent://prop/ns-quota/topic2" + UUID.randomUUID(); + + final String subName1 = "c1"; + final int numMsgs = 4; + + Consumer consumer1 = client.newConsumer().topic(topic1).subscriptionName(subName1) + .acknowledgmentGroupTime(0, SECONDS) + .subscribe(); + Producer producer = createProducer(client, topic1); + + byte[] content = new byte[1024]; + for (int i = 0; i < numMsgs; i++) { + MessageId send = producer.send(content); + System.out.println(i + ":msg:" + MILLISECONDS.toSeconds(System.currentTimeMillis())); + } + + String c1MarkDeletePositionBefore = + admin.topics().getInternalStats(topic1).cursors.get(subName1).markDeletePosition; + + // Move subscription 1 to end + for (int i = 0; i < numMsgs; i++) { + Message message1 = consumer1.receive(); + consumer1.acknowledge(message1); + } + + // This code will wait about 4~5 Seconds, to make sure the oldest message is 4~5 seconds old + c1MarkDeletePositionBefore = waitForMarkDeletePositionToChange(topic1, subName1, c1MarkDeletePositionBefore); + waitForQuotaCheckToRunTwice(); + + Metrics metrics = prometheusMetricsClient.getMetrics(); + TopicStats topicStats = getTopicStats(topic1); + + assertThat(topicStats.getBacklogQuotaLimitTime()).isEqualTo(timeLimitSeconds); + assertThat(topicStats.getBacklogSize()).isEqualTo(0); + assertThat(topicStats.getSubscriptions().get(subName1).getMsgBacklog()).isEqualTo(0); + assertThat(topicStats.getOldestBacklogMessageAgeSeconds()).isEqualTo(0); + assertThat(topicStats.getOldestBacklogMessageSubscriptionName()).isNull(); + + Metric backlogAgeMetric = + metrics.findSingleMetricByNameAndLabels("pulsar_storage_backlog_age_seconds", + Pair.of("topic", topic1)); + assertThat(backlogAgeMetric.tags).containsExactly( + entry("cluster", CLUSTER_NAME), + entry("namespace", namespace), + entry("topic", topic1)); + assertThat((long) backlogAgeMetric.value).isEqualTo(0); + + // producer should create success. + Producer producer2 = createProducer(client, topic1); + assertNotNull(producer2); + } + config.setPreciseTimeBasedBacklogQuotaCheck(false); + config.setExposePreciseBacklogInPrometheus(false); + } + private long getReadEntries(String topic1) { return ((PersistentTopic) pulsar.getBrokerService().getTopicReference(topic1).get()) .getManagedLedger().getStats().getEntriesReadTotalCount(); @@ -597,6 +710,71 @@ public void backlogsStatsNotPrecise() throws PulsarAdminException, PulsarClientE config.setManagedLedgerMaxEntriesPerLedger(MAX_ENTRIES_PER_LEDGER); } + @Test + public void backlogsStatsNotPreciseWithNoBacklog() throws PulsarAdminException, PulsarClientException, InterruptedException { + config.setPreciseTimeBasedBacklogQuotaCheck(false); + config.setExposePreciseBacklogInPrometheus(false); + config.setManagedLedgerMaxEntriesPerLedger(6); + final String namespace = "prop/ns-quota"; + assertEquals(admin.namespaces().getBacklogQuotaMap(namespace), new HashMap<>()); + final int timeLimitSeconds = 2; + admin.namespaces().setBacklogQuota( + namespace, + BacklogQuota.builder() + .limitTime(timeLimitSeconds) + .retentionPolicy(BacklogQuota.RetentionPolicy.producer_exception) + .build(), + message_age); + + try (PulsarClient client = PulsarClient.builder().serviceUrl(adminUrl.toString()) + .maxBackoffInterval(3, SECONDS) + .statsInterval(0, SECONDS).build()) { + final String topic1 = "persistent://prop/ns-quota/topic2" + UUID.randomUUID(); + + final String subName1 = "brandNewC1"; + final int numMsgs = 5; + + Consumer consumer1 = client.newConsumer().topic(topic1).subscriptionName(subName1) + .acknowledgmentGroupTime(0, SECONDS) + .isAckReceiptEnabled(true) + .subscribe(); + Producer producer = createProducer(client, topic1); + + byte[] content = new byte[1024]; + for (int i = 0; i < numMsgs; i++) { + producer.send(content); + } + + String c1MarkDeletePositionBefore = + admin.topics().getInternalStats(topic1).cursors.get(subName1).markDeletePosition; + + log.info("Moved subscription 1 to end"); + for (int i = 0; i < numMsgs; i++) { + consumer1.acknowledge(consumer1.receive()); + } + + c1MarkDeletePositionBefore = waitForMarkDeletePositionToChange(topic1, subName1, c1MarkDeletePositionBefore); + waitForQuotaCheckToRunTwice(); + + // backlog and backlogAceSeconds should be 0 + TopicStats topicStats = getTopicStats(topic1, false); + Metrics metrics = prometheusMetricsClient.getMetrics(); + assertEquals(topicStats.getSubscriptions().get(subName1).getMsgBacklog(), 0); + assertThat(topicStats.getOldestBacklogMessageSubscriptionName()).isNull(); + assertThat(topicStats.getOldestBacklogMessageAgeSeconds()).isEqualTo(0); + Metric backlogAgeMetric = + metrics.findSingleMetricByNameAndLabels("pulsar_storage_backlog_age_seconds", + Pair.of("topic", topic1)); + assertThat(backlogAgeMetric.value).isEqualTo(0); + + // producer should create success. + Producer producer2 = createProducer(client, topic1); + assertNotNull(producer2); + + config.setManagedLedgerMaxEntriesPerLedger(MAX_ENTRIES_PER_LEDGER); + } + } + private void unloadAndLoadTopic(String topic, Producer producer) throws PulsarAdminException, PulsarClientException { admin.topics().unload(topic); diff --git a/pulsar-broker/src/test/java/org/apache/pulsar/broker/service/BrokerServiceAutoTopicCreationTest.java b/pulsar-broker/src/test/java/org/apache/pulsar/broker/service/BrokerServiceAutoTopicCreationTest.java index 0a6cffc7685d4..ea5365bcf4b2c 100644 --- a/pulsar-broker/src/test/java/org/apache/pulsar/broker/service/BrokerServiceAutoTopicCreationTest.java +++ b/pulsar-broker/src/test/java/org/apache/pulsar/broker/service/BrokerServiceAutoTopicCreationTest.java @@ -566,13 +566,13 @@ public void testExtensibleLoadManagerImplInternalTopicAutoCreations() try { pulsarClient.newProducer().topic(ExtensibleLoadManagerImpl.BROKER_LOAD_DATA_STORE_TOPIC).create(); Assert.fail("Create should have failed."); - } catch (PulsarClientException.TopicDoesNotExistException e) { + } catch (PulsarClientException.TopicDoesNotExistException | PulsarClientException.NotFoundException e) { // expected } try { pulsarClient.newProducer().topic(ExtensibleLoadManagerImpl.TOP_BUNDLES_LOAD_DATA_STORE_TOPIC).create(); Assert.fail("Create should have failed."); - } catch (PulsarClientException.TopicDoesNotExistException e) { + } catch (PulsarClientException.TopicDoesNotExistException | PulsarClientException.NotFoundException e) { // expected } diff --git a/pulsar-broker/src/test/java/org/apache/pulsar/broker/service/BrokerServiceTest.java b/pulsar-broker/src/test/java/org/apache/pulsar/broker/service/BrokerServiceTest.java index 9f561889aa825..2ce6728e98a08 100644 --- a/pulsar-broker/src/test/java/org/apache/pulsar/broker/service/BrokerServiceTest.java +++ b/pulsar-broker/src/test/java/org/apache/pulsar/broker/service/BrokerServiceTest.java @@ -1022,12 +1022,12 @@ protected void handlePartitionResponse(CommandPartitionedTopicMetadataResponse l // for PMR // 2 lookup will succeed long reqId1 = reqId++; - ByteBuf request1 = Commands.newPartitionMetadataRequest(topicName, reqId1); + ByteBuf request1 = Commands.newPartitionMetadataRequest(topicName, reqId1, true); CompletableFuture f1 = pool.getConnection(resolver.resolveHost()) .thenCompose(clientCnx -> clientCnx.newLookup(request1, reqId1)); long reqId2 = reqId++; - ByteBuf request2 = Commands.newPartitionMetadataRequest(topicName, reqId2); + ByteBuf request2 = Commands.newPartitionMetadataRequest(topicName, reqId2, true); CompletableFuture f2 = pool.getConnection(resolver.resolveHost()) .thenCompose(clientCnx -> { CompletableFuture future = clientCnx.newLookup(request2, reqId2); @@ -1042,17 +1042,17 @@ protected void handlePartitionResponse(CommandPartitionedTopicMetadataResponse l // 3 lookup will fail latchRef.set(new CountDownLatch(1)); long reqId3 = reqId++; - ByteBuf request3 = Commands.newPartitionMetadataRequest(topicName, reqId3); + ByteBuf request3 = Commands.newPartitionMetadataRequest(topicName, reqId3, true); f1 = pool.getConnection(resolver.resolveHost()) .thenCompose(clientCnx -> clientCnx.newLookup(request3, reqId3)); long reqId4 = reqId++; - ByteBuf request4 = Commands.newPartitionMetadataRequest(topicName, reqId4); + ByteBuf request4 = Commands.newPartitionMetadataRequest(topicName, reqId4, true); f2 = pool.getConnection(resolver.resolveHost()) .thenCompose(clientCnx -> clientCnx.newLookup(request4, reqId4)); long reqId5 = reqId++; - ByteBuf request5 = Commands.newPartitionMetadataRequest(topicName, reqId5); + ByteBuf request5 = Commands.newPartitionMetadataRequest(topicName, reqId5, true); CompletableFuture f3 = pool.getConnection(resolver.resolveHost()) .thenCompose(clientCnx -> { CompletableFuture future = clientCnx.newLookup(request5, reqId5); diff --git a/pulsar-broker/src/test/java/org/apache/pulsar/broker/service/BrokerServiceThrottlingTest.java b/pulsar-broker/src/test/java/org/apache/pulsar/broker/service/BrokerServiceThrottlingTest.java index b2cfe63e2e5b4..0d984e0675db5 100644 --- a/pulsar-broker/src/test/java/org/apache/pulsar/broker/service/BrokerServiceThrottlingTest.java +++ b/pulsar-broker/src/test/java/org/apache/pulsar/broker/service/BrokerServiceThrottlingTest.java @@ -167,7 +167,7 @@ public void testLookupThrottlingForClientByBroker() throws Exception { for (int i = 0; i < totalConsumers; i++) { long reqId = 0xdeadbeef + i; Future f = executor.submit(() -> { - ByteBuf request = Commands.newPartitionMetadataRequest(topicName, reqId); + ByteBuf request = Commands.newPartitionMetadataRequest(topicName, reqId, true); pool.getConnection(resolver.resolveHost()) .thenCompose(clientCnx -> clientCnx.newLookup(request, reqId)) .get(); diff --git a/pulsar-broker/src/test/java/org/apache/pulsar/broker/service/OneWayReplicatorTest.java b/pulsar-broker/src/test/java/org/apache/pulsar/broker/service/OneWayReplicatorTest.java index e6e45eebf8d24..e69165fe9495c 100644 --- a/pulsar-broker/src/test/java/org/apache/pulsar/broker/service/OneWayReplicatorTest.java +++ b/pulsar-broker/src/test/java/org/apache/pulsar/broker/service/OneWayReplicatorTest.java @@ -43,6 +43,7 @@ import java.util.concurrent.atomic.AtomicInteger; import java.util.concurrent.atomic.AtomicReference; import java.util.function.BiFunction; +import java.util.function.Supplier; import lombok.AllArgsConstructor; import lombok.Data; import lombok.SneakyThrows; @@ -55,7 +56,9 @@ import org.apache.pulsar.broker.service.persistent.GeoPersistentReplicator; import org.apache.pulsar.broker.service.persistent.PersistentReplicator; import org.apache.pulsar.broker.service.persistent.PersistentTopic; +import org.apache.pulsar.client.admin.PulsarAdmin; import org.apache.pulsar.client.api.Consumer; +import org.apache.pulsar.client.api.Message; import org.apache.pulsar.client.api.MessageId; import org.apache.pulsar.client.api.Producer; import org.apache.pulsar.client.api.ProducerBuilder; @@ -76,6 +79,7 @@ import org.testng.Assert; import org.testng.annotations.AfterClass; import org.testng.annotations.BeforeClass; +import org.testng.annotations.DataProvider; import org.testng.annotations.Test; @Slf4j @@ -778,4 +782,102 @@ public void testExpandTopicPartitionsOnNamespaceLevelReplication() throws Except admin2.topics().deletePartitionedTopic(topicName, false); }); } + + private String getTheLatestMessage(String topic, PulsarClient client, PulsarAdmin admin) throws Exception { + String dummySubscription = "s_" + UUID.randomUUID().toString().replace("-", ""); + admin.topics().createSubscription(topic, dummySubscription, MessageId.earliest); + Consumer c = client.newConsumer(Schema.STRING).topic(topic).subscriptionName(dummySubscription) + .subscribe(); + String lastMsgValue = null; + while (true) { + Message msg = c.receive(2, TimeUnit.SECONDS); + if (msg == null) { + break; + } + lastMsgValue = msg.getValue(); + } + c.unsubscribe(); + return lastMsgValue; + } + + enum ReplicationLevel { + TOPIC_LEVEL, + NAMESPACE_LEVEL; + } + + @DataProvider(name = "replicationLevels") + public Object[][] replicationLevels() { + return new Object[][]{ + {ReplicationLevel.TOPIC_LEVEL}, + {ReplicationLevel.NAMESPACE_LEVEL} + }; + } + + @Test(dataProvider = "replicationLevels") + public void testReloadWithTopicLevelGeoReplication(ReplicationLevel replicationLevel) throws Exception { + final String topicName = ((Supplier) () -> { + if (replicationLevel.equals(ReplicationLevel.TOPIC_LEVEL)) { + return BrokerTestUtil.newUniqueName("persistent://" + nonReplicatedNamespace + "/tp_"); + } else { + return BrokerTestUtil.newUniqueName("persistent://" + replicatedNamespace + "/tp_"); + } + }).get(); + admin1.topics().createNonPartitionedTopic(topicName); + admin2.topics().createNonPartitionedTopic(topicName); + admin2.topics().createSubscription(topicName, "s1", MessageId.earliest); + if (replicationLevel.equals(ReplicationLevel.TOPIC_LEVEL)) { + admin1.topics().setReplicationClusters(topicName, Arrays.asList(cluster1, cluster2)); + } else { + pulsar1.getConfig().setTopicLevelPoliciesEnabled(false); + } + verifyReplicationWorks(topicName); + + /** + * Verify: + * 1. Inject an error to make the replicator is not able to work. + * 2. Send one message, since the replicator does not work anymore, this message will not be replicated. + * 3. Unload topic, the replicator will be re-created. + * 4. Verify: the message can be replicated to the remote cluster. + */ + // Step 1: Inject an error to make the replicator is not able to work. + Replicator replicator = broker1.getTopic(topicName, false).join().get().getReplicators().get(cluster2); + replicator.terminate(); + + // Step 2: Send one message, since the replicator does not work anymore, this message will not be replicated. + String msg = UUID.randomUUID().toString(); + Producer p1 = client1.newProducer(Schema.STRING).topic(topicName).create(); + p1.send(msg); + p1.close(); + // The result of "peek message" will be the messages generated, so it is not the same as the message just sent. + Thread.sleep(3000); + assertNotEquals(getTheLatestMessage(topicName, client2, admin2), msg); + assertEquals(admin1.topics().getStats(topicName).getReplication().get(cluster2).getReplicationBacklog(), 1); + + // Step 3: Unload topic, the replicator will be re-created. + admin1.topics().unload(topicName); + + // Step 4. Verify: the message can be replicated to the remote cluster. + Awaitility.await().atMost(Duration.ofSeconds(300)).untilAsserted(() -> { + log.info("replication backlog: {}", + admin1.topics().getStats(topicName).getReplication().get(cluster2).getReplicationBacklog()); + assertEquals(admin1.topics().getStats(topicName).getReplication().get(cluster2).getReplicationBacklog(), 0); + assertEquals(getTheLatestMessage(topicName, client2, admin2), msg); + }); + + // Cleanup. + if (replicationLevel.equals(ReplicationLevel.TOPIC_LEVEL)) { + admin1.topics().setReplicationClusters(topicName, Arrays.asList(cluster1)); + Awaitility.await().untilAsserted(() -> { + assertEquals(broker1.getTopic(topicName, false).join().get().getReplicators().size(), 0); + }); + admin1.topics().delete(topicName, false); + admin2.topics().delete(topicName, false); + } else { + pulsar1.getConfig().setTopicLevelPoliciesEnabled(true); + cleanupTopics(() -> { + admin1.topics().delete(topicName); + admin2.topics().delete(topicName); + }); + } + } } diff --git a/pulsar-broker/src/test/java/org/apache/pulsar/broker/service/OneWayReplicatorTestBase.java b/pulsar-broker/src/test/java/org/apache/pulsar/broker/service/OneWayReplicatorTestBase.java index 7372b2e478475..ffe6147412e56 100644 --- a/pulsar-broker/src/test/java/org/apache/pulsar/broker/service/OneWayReplicatorTestBase.java +++ b/pulsar-broker/src/test/java/org/apache/pulsar/broker/service/OneWayReplicatorTestBase.java @@ -350,6 +350,28 @@ protected PulsarClient initClient(ClientBuilder clientBuilder) throws Exception } protected void verifyReplicationWorks(String topic) throws Exception { + // Wait for replicator starting. + Awaitility.await().until(() -> { + try { + PersistentTopic persistentTopic = (PersistentTopic) pulsar1.getBrokerService() + .getTopic(topic, false).join().get(); + if (persistentTopic.getReplicators().size() > 0) { + return true; + } + } catch (Exception ex) {} + + try { + String partition0 = TopicName.get(topic).getPartition(0).toString(); + PersistentTopic persistentTopic = (PersistentTopic) pulsar1.getBrokerService() + .getTopic(partition0, false).join().get(); + if (persistentTopic.getReplicators().size() > 0) { + return true; + } + } catch (Exception ex) {} + + return false; + }); + // Verify: pub & sub. final String subscription = "__subscribe_1"; final String msgValue = "__msg1"; Producer producer1 = client1.newProducer(Schema.STRING).topic(topic).create(); diff --git a/pulsar-broker/src/test/java/org/apache/pulsar/broker/service/OneWayReplicatorUsingGlobalZKTest.java b/pulsar-broker/src/test/java/org/apache/pulsar/broker/service/OneWayReplicatorUsingGlobalZKTest.java index b4747a8bd0e47..b8f8edce2477e 100644 --- a/pulsar-broker/src/test/java/org/apache/pulsar/broker/service/OneWayReplicatorUsingGlobalZKTest.java +++ b/pulsar-broker/src/test/java/org/apache/pulsar/broker/service/OneWayReplicatorUsingGlobalZKTest.java @@ -104,4 +104,9 @@ public void testNoExpandTopicPartitionsWhenDisableTopicLevelReplication() throws public void testExpandTopicPartitionsOnNamespaceLevelReplication() throws Exception { super.testExpandTopicPartitionsOnNamespaceLevelReplication(); } + + @Test(enabled = false) + public void testReloadWithTopicLevelGeoReplication(ReplicationLevel replicationLevel) throws Exception { + super.testReloadWithTopicLevelGeoReplication(replicationLevel); + } } diff --git a/pulsar-broker/src/test/java/org/apache/pulsar/broker/service/ReplicatorTest.java b/pulsar-broker/src/test/java/org/apache/pulsar/broker/service/ReplicatorTest.java index 1176106735bb8..3cc2ca2457a4b 100644 --- a/pulsar-broker/src/test/java/org/apache/pulsar/broker/service/ReplicatorTest.java +++ b/pulsar-broker/src/test/java/org/apache/pulsar/broker/service/ReplicatorTest.java @@ -19,12 +19,10 @@ package org.apache.pulsar.broker.service; import static org.apache.pulsar.broker.BrokerTestUtil.newUniqueName; -import static org.apache.pulsar.broker.auth.MockedPulsarServiceBaseTest.retryStrategically; import static org.mockito.ArgumentMatchers.eq; import static org.mockito.Mockito.doAnswer; import static org.mockito.Mockito.spy; import static org.testng.Assert.assertEquals; -import static org.testng.Assert.assertNotEquals; import static org.testng.Assert.assertNotNull; import static org.testng.Assert.assertNull; import static org.testng.Assert.assertTrue; @@ -1404,13 +1402,6 @@ public void testCleanupTopic() throws Exception { // Ok } - final CompletableFuture> timedOutTopicFuture = topicFuture; - // timeout topic future should be removed from cache - retryStrategically((test) -> pulsar1.getBrokerService().getTopic(topicName, false) != timedOutTopicFuture, 5, - 1000); - - assertNotEquals(timedOutTopicFuture, pulsar1.getBrokerService().getTopics().get(topicName)); - try { Consumer consumer = client1.newConsumer().topic(topicName).subscriptionType(SubscriptionType.Shared) .subscriptionName("my-subscriber-name").subscribeAsync().get(100, TimeUnit.MILLISECONDS); @@ -1422,6 +1413,7 @@ public void testCleanupTopic() throws Exception { ManagedLedgerImpl ml = (ManagedLedgerImpl) mlFactory.open(topicMlName + "-2"); mlFuture.complete(ml); + // Re-create topic will success. Consumer consumer = client1.newConsumer().topic(topicName).subscriptionName("my-subscriber-name") .subscriptionType(SubscriptionType.Shared).subscribeAsync() .get(2 * topicLoadTimeoutSeconds, TimeUnit.SECONDS); diff --git a/pulsar-broker/src/test/java/org/apache/pulsar/broker/service/ServerCnxTest.java b/pulsar-broker/src/test/java/org/apache/pulsar/broker/service/ServerCnxTest.java index ad7728319c9a7..626bce380b9de 100644 --- a/pulsar-broker/src/test/java/org/apache/pulsar/broker/service/ServerCnxTest.java +++ b/pulsar-broker/src/test/java/org/apache/pulsar/broker/service/ServerCnxTest.java @@ -3573,7 +3573,7 @@ public void handlePartitionMetadataRequestWithServiceNotReady() throws Exception doReturn(false).when(pulsar).isRunning(); assertTrue(channel.isActive()); - ByteBuf clientCommand = Commands.newPartitionMetadataRequest(successTopicName, 1); + ByteBuf clientCommand = Commands.newPartitionMetadataRequest(successTopicName, 1, true); channel.writeInbound(clientCommand); Object response = getResponse(); assertTrue(response instanceof CommandPartitionedTopicMetadataResponse); diff --git a/pulsar-broker/src/test/java/org/apache/pulsar/broker/service/TopicGCTest.java b/pulsar-broker/src/test/java/org/apache/pulsar/broker/service/TopicGCTest.java index 7790940c1327f..8fdf0723ea8d1 100644 --- a/pulsar-broker/src/test/java/org/apache/pulsar/broker/service/TopicGCTest.java +++ b/pulsar-broker/src/test/java/org/apache/pulsar/broker/service/TopicGCTest.java @@ -18,6 +18,7 @@ */ package org.apache.pulsar.broker.service; +import static org.testng.Assert.assertNotNull; import static org.testng.Assert.assertTrue; import java.util.Optional; import java.util.concurrent.CompletableFuture; @@ -99,6 +100,7 @@ public void testCreateConsumerAfterOnePartDeleted() throws Exception { Consumer consumerAllPartition = pulsarClient.newConsumer(Schema.STRING).topic(topic) .subscriptionName(subscription).isAckReceiptEnabled(true).subscribe(); Message msg = consumerAllPartition.receive(2, TimeUnit.SECONDS); + assertNotNull(msg); String receivedMsgValue = msg.getValue(); log.info("received msg: {}", receivedMsgValue); consumerAllPartition.acknowledge(msg); diff --git a/pulsar-broker/src/test/java/org/apache/pulsar/broker/testcontext/PulsarTestContext.java b/pulsar-broker/src/test/java/org/apache/pulsar/broker/testcontext/PulsarTestContext.java index 449f7fc7d57e1..326cfaf2b713d 100644 --- a/pulsar-broker/src/test/java/org/apache/pulsar/broker/testcontext/PulsarTestContext.java +++ b/pulsar-broker/src/test/java/org/apache/pulsar/broker/testcontext/PulsarTestContext.java @@ -635,7 +635,8 @@ private void initializeCommonPulsarServices(SpyConfig spyConfig) { } else { try { MetadataStoreExtended store = MetadataStoreFactoryImpl.createExtended("memory:local", - MetadataStoreConfig.builder().build()); + MetadataStoreConfig.builder() + .metadataStoreName(MetadataStoreConfig.METADATA_STORE).build()); registerCloseable(() -> { store.close(); resetSpyOrMock(store); diff --git a/pulsar-broker/src/test/java/org/apache/pulsar/broker/transaction/buffer/TransactionLowWaterMarkTest.java b/pulsar-broker/src/test/java/org/apache/pulsar/broker/transaction/buffer/TransactionLowWaterMarkTest.java index 3901f186d81c7..3f268c4b7c973 100644 --- a/pulsar-broker/src/test/java/org/apache/pulsar/broker/transaction/buffer/TransactionLowWaterMarkTest.java +++ b/pulsar-broker/src/test/java/org/apache/pulsar/broker/transaction/buffer/TransactionLowWaterMarkTest.java @@ -146,7 +146,7 @@ public void testTransactionBufferLowWaterMark() throws Exception { PartitionedTopicMetadata partitionedTopicMetadata = ((PulsarClientImpl) pulsarClient).getLookup() - .getPartitionedTopicMetadata(SystemTopicNames.TRANSACTION_COORDINATOR_ASSIGN).get(); + .getPartitionedTopicMetadata(SystemTopicNames.TRANSACTION_COORDINATOR_ASSIGN, false).get(); Transaction lowWaterMarkTxn = null; for (int i = 0; i < partitionedTopicMetadata.partitions; i++) { lowWaterMarkTxn = pulsarClient.newTransaction() @@ -251,7 +251,7 @@ public void testPendingAckLowWaterMark() throws Exception { PartitionedTopicMetadata partitionedTopicMetadata = ((PulsarClientImpl) pulsarClient).getLookup() - .getPartitionedTopicMetadata(SystemTopicNames.TRANSACTION_COORDINATOR_ASSIGN).get(); + .getPartitionedTopicMetadata(SystemTopicNames.TRANSACTION_COORDINATOR_ASSIGN, false).get(); Transaction lowWaterMarkTxn = null; for (int i = 0; i < partitionedTopicMetadata.partitions; i++) { lowWaterMarkTxn = pulsarClient.newTransaction() diff --git a/pulsar-broker/src/test/java/org/apache/pulsar/client/api/BrokerServiceLookupTest.java b/pulsar-broker/src/test/java/org/apache/pulsar/client/api/BrokerServiceLookupTest.java index 0a4c5b7a318b3..9319ea4e876b7 100644 --- a/pulsar-broker/src/test/java/org/apache/pulsar/client/api/BrokerServiceLookupTest.java +++ b/pulsar-broker/src/test/java/org/apache/pulsar/client/api/BrokerServiceLookupTest.java @@ -852,7 +852,7 @@ public void testMergeGetPartitionedMetadataRequests() throws Exception { // Verify the request is works after merge the requests. List> futures = new ArrayList<>(); for (int i = 0; i < 100; i++) { - futures.add(lookupService.getPartitionedTopicMetadata(TopicName.get(tpName))); + futures.add(lookupService.getPartitionedTopicMetadata(TopicName.get(tpName), false)); } for (CompletableFuture future : futures) { assertEquals(future.join().partitions, topicPartitions); diff --git a/pulsar-broker/src/test/java/org/apache/pulsar/client/api/OrphanPersistentTopicTest.java b/pulsar-broker/src/test/java/org/apache/pulsar/client/api/OrphanPersistentTopicTest.java index 7a6189702dd8c..54b9ff1f2ea27 100644 --- a/pulsar-broker/src/test/java/org/apache/pulsar/client/api/OrphanPersistentTopicTest.java +++ b/pulsar-broker/src/test/java/org/apache/pulsar/client/api/OrphanPersistentTopicTest.java @@ -18,7 +18,9 @@ */ package org.apache.pulsar.client.api; +import static org.apache.pulsar.broker.service.persistent.PersistentTopic.DEDUPLICATION_CURSOR_NAME; import static org.testng.Assert.assertEquals; +import static org.testng.Assert.assertNotNull; import static org.testng.Assert.assertTrue; import java.util.List; import java.util.Map; @@ -26,12 +28,16 @@ import java.util.concurrent.CompletableFuture; import java.util.concurrent.TimeUnit; import java.util.concurrent.atomic.AtomicInteger; +import lombok.SneakyThrows; import lombok.extern.slf4j.Slf4j; import org.apache.pulsar.broker.BrokerTestUtil; import org.apache.pulsar.broker.service.BrokerService; import org.apache.pulsar.broker.service.Topic; import org.apache.pulsar.broker.service.TopicPoliciesService; import org.apache.pulsar.broker.service.TopicPolicyListener; +import org.apache.pulsar.broker.transaction.buffer.TransactionBuffer; +import org.apache.pulsar.broker.transaction.buffer.TransactionBufferProvider; +import org.apache.pulsar.broker.transaction.buffer.impl.TransactionBufferDisable; import org.apache.pulsar.common.naming.TopicName; import org.apache.pulsar.common.policies.data.TopicPolicies; import org.awaitility.Awaitility; @@ -104,4 +110,66 @@ public void testNoOrphanTopicAfterCreateTimeout() throws Exception { admin.topics().delete(tpName, false); pulsar.getConfig().setTopicLoadTimeoutSeconds(originalTopicLoadTimeoutSeconds); } + + @Test + public void testCloseLedgerThatTopicAfterCreateTimeout() throws Exception { + // Make the topic loading timeout faster. + long originalTopicLoadTimeoutSeconds = pulsar.getConfig().getTopicLoadTimeoutSeconds(); + int topicLoadTimeoutSeconds = 1; + pulsar.getConfig().setTopicLoadTimeoutSeconds(topicLoadTimeoutSeconds); + pulsar.getConfig().setBrokerDeduplicationEnabled(true); + pulsar.getConfig().setTransactionCoordinatorEnabled(true); + String tpName = BrokerTestUtil.newUniqueName("persistent://public/default/tp2"); + + // Mock message deduplication recovery speed topicLoadTimeoutSeconds + String mlPath = BrokerService.MANAGED_LEDGER_PATH_ZNODE + "/" + + TopicName.get(tpName).getPersistenceNamingEncoding() + "/" + DEDUPLICATION_CURSOR_NAME; + mockZooKeeper.delay(topicLoadTimeoutSeconds * 1000, (op, path) -> { + if (mlPath.equals(path)) { + log.info("Topic load timeout: " + path); + return true; + } + return false; + }); + + // First load topic will trigger timeout + // The first topic load will trigger a timeout. When the topic closes, it will call transactionBuffer.close. + // Here, we simulate a sleep to ensure that the ledger is not immediately closed. + TransactionBufferProvider mockTransactionBufferProvider = new TransactionBufferProvider() { + @Override + public TransactionBuffer newTransactionBuffer(Topic originTopic) { + return new TransactionBufferDisable(originTopic) { + @SneakyThrows + @Override + public CompletableFuture closeAsync() { + Thread.sleep(500); + return super.closeAsync(); + } + }; + } + }; + TransactionBufferProvider originalTransactionBufferProvider = pulsar.getTransactionBufferProvider(); + pulsar.setTransactionExecutorProvider(mockTransactionBufferProvider); + CompletableFuture> firstLoad = pulsar.getBrokerService().getTopic(tpName, true); + Awaitility.await().ignoreExceptions().atMost(5, TimeUnit.SECONDS) + .pollInterval(100, TimeUnit.MILLISECONDS) + // assert first create topic timeout + .untilAsserted(() -> { + assertTrue(firstLoad.isCompletedExceptionally()); + }); + + // Once the first load topic times out, immediately to load the topic again. + Producer producer = pulsarClient.newProducer().topic(tpName).create(); + for (int i = 0; i < 10; i++) { + MessageId send = producer.send("msg".getBytes()); + Thread.sleep(100); + assertNotNull(send); + } + + // set to back + pulsar.setTransactionExecutorProvider(originalTransactionBufferProvider); + pulsar.getConfig().setTopicLoadTimeoutSeconds(originalTopicLoadTimeoutSeconds); + pulsar.getConfig().setBrokerDeduplicationEnabled(false); + pulsar.getConfig().setTransactionCoordinatorEnabled(false); + } } diff --git a/pulsar-broker/src/test/java/org/apache/pulsar/client/impl/ClientCnxTest.java b/pulsar-broker/src/test/java/org/apache/pulsar/client/impl/ClientCnxTest.java index d2f610ae53f65..1a9b4bbcb0d21 100644 --- a/pulsar-broker/src/test/java/org/apache/pulsar/client/impl/ClientCnxTest.java +++ b/pulsar-broker/src/test/java/org/apache/pulsar/client/impl/ClientCnxTest.java @@ -20,13 +20,17 @@ import com.google.common.collect.Sets; import io.netty.channel.ChannelHandlerContext; +import java.lang.reflect.Field; +import java.util.concurrent.CompletableFuture; import java.util.concurrent.CountDownLatch; import java.util.concurrent.ExecutorService; import java.util.concurrent.Executors; import org.apache.pulsar.PulsarVersion; +import org.apache.pulsar.broker.BrokerTestUtil; import org.apache.pulsar.broker.auth.MockedPulsarServiceBaseTest; import org.apache.pulsar.client.api.Consumer; import org.apache.pulsar.client.api.Producer; +import org.apache.pulsar.client.api.PulsarClient; import org.apache.pulsar.client.api.Schema; import org.apache.pulsar.common.policies.data.ClusterData; import org.apache.pulsar.common.policies.data.TenantInfoImpl; @@ -123,4 +127,44 @@ public void testClientVersion() throws Exception { producer.close(); consumer.close(); } + + @Test + public void testSupportsGetPartitionedMetadataWithoutAutoCreation() throws Exception { + final String topic = BrokerTestUtil.newUniqueName( "persistent://" + NAMESPACE + "/tp"); + admin.topics().createNonPartitionedTopic(topic); + PulsarClientImpl clientWitBinaryLookup = (PulsarClientImpl) PulsarClient.builder() + .maxNumberOfRejectedRequestPerConnection(1) + .connectionMaxIdleSeconds(Integer.MAX_VALUE) + .serviceUrl(pulsar.getBrokerServiceUrl()) + .build(); + ProducerImpl producer = (ProducerImpl) clientWitBinaryLookup.newProducer().topic(topic).create(); + + // Verify: the variable "isSupportsGetPartitionedMetadataWithoutAutoCreation" responded from the broker is true. + Awaitility.await().untilAsserted(() -> { + ClientCnx clientCnx = producer.getClientCnx(); + Assert.assertNotNull(clientCnx); + Assert.assertTrue(clientCnx.isSupportsGetPartitionedMetadataWithoutAutoCreation()); + }); + Assert.assertEquals( + clientWitBinaryLookup.getPartitionsForTopic(topic, true).get().size(), 1); + + // Inject a "false" value for the variable "isSupportsGetPartitionedMetadataWithoutAutoCreation". + // Verify: client will get a not support error. + Field field = ClientCnx.class.getDeclaredField("supportsGetPartitionedMetadataWithoutAutoCreation"); + field.setAccessible(true); + for (CompletableFuture clientCnxFuture : clientWitBinaryLookup.getCnxPool().getConnections()) { + field.set(clientCnxFuture.get(), false); + } + try { + clientWitBinaryLookup.getPartitionsForTopic(topic, false).join(); + Assert.fail("Expected an error that the broker version is too old."); + } catch (Exception ex) { + Assert.assertTrue(ex.getMessage().contains("without auto-creation is not supported from the broker")); + } + + // cleanup. + producer.close(); + clientWitBinaryLookup.close(); + admin.topics().delete(topic, false); + } } diff --git a/pulsar-broker/src/test/resources/configurations/pulsar_broker_test.conf b/pulsar-broker/src/test/resources/configurations/pulsar_broker_test.conf index bfbbfb7487c42..36f5869d73de6 100644 --- a/pulsar-broker/src/test/resources/configurations/pulsar_broker_test.conf +++ b/pulsar-broker/src/test/resources/configurations/pulsar_broker_test.conf @@ -17,17 +17,17 @@ # under the License. # -applicationName="pulsar_broker" -zookeeperServers="localhost" -configurationStoreServers="localhost" +applicationName=pulsar_broker +zookeeperServers=localhost +configurationStoreServers=localhost brokerServicePort=6650 -brokerServicePortTls=6651 +brokerServicePortTls= webServicePort=8080 -webServicePortTls=4443 +webServicePortTls= httpMaxRequestHeaderSize=1234 bindAddress=0.0.0.0 advertisedAddress= -clusterName="test_cluster" +clusterName=test_cluster brokerShutdownTimeoutMs=3000 backlogQuotaCheckEnabled=true backlogQuotaCheckIntervalInSeconds=60 @@ -42,17 +42,17 @@ clientLibraryVersionCheckEnabled=false clientLibraryVersionCheckAllowUnversioned=true statusFilePath=/tmp/status.html tlsEnabled=false -tlsCertificateFilePath=/usr/local/conf/pulsar/server.crt -tlsKeyFilePath=/home/local/conf/pulsar/server.key +tlsCertificateFilePath= +tlsKeyFilePath= tlsTrustCertsFilePath= tlsAllowInsecureConnection=false authenticationEnabled=false authorizationEnabled=false -superUserRoles="test_user" -brokerClientAuthenticationPlugin="org.apache.pulsar.client.impl.auth.AuthenticationDisabled" +superUserRoles=test_user +brokerClientAuthenticationPlugin=org.apache.pulsar.client.impl.auth.AuthenticationDisabled brokerClientAuthenticationParameters= -bookkeeperClientAuthenticationPlugin="test_auth_plugin" -bookkeeperClientAuthenticationAppId="test_auth_id" +bookkeeperClientAuthenticationPlugin= +bookkeeperClientAuthenticationAppId=test_auth_id bookkeeperClientTimeoutInSeconds=30 bookkeeperClientSpeculativeReadTimeoutInMillis=0 bookkeeperClientHealthCheckEnabled=true @@ -64,7 +64,7 @@ bookkeeperClientRegionawarePolicyEnabled=false bookkeeperClientMinNumRacksPerWriteQuorum=2 bookkeeperClientEnforceMinNumRacksPerWriteQuorum=false bookkeeperClientReorderReadSequenceEnabled=false -bookkeeperClientIsolationGroups="test_group" +bookkeeperClientIsolationGroups=test_group managedLedgerDefaultEnsembleSize=3 managedLedgerDefaultWriteQuorum=2 managedLedgerDefaultAckQuorum=2 diff --git a/pulsar-broker/src/test/resources/configurations/pulsar_broker_test_standalone.conf b/pulsar-broker/src/test/resources/configurations/pulsar_broker_test_standalone.conf index c733409fc0043..0748418be6390 100644 --- a/pulsar-broker/src/test/resources/configurations/pulsar_broker_test_standalone.conf +++ b/pulsar-broker/src/test/resources/configurations/pulsar_broker_test_standalone.conf @@ -17,18 +17,18 @@ # under the License. # -applicationName="pulsar_broker" -metadataStoreUrl="zk:localhost:2181/ledger" -configurationMetadataStoreUrl="zk:localhost:2181" -brokerServicePort=6650 -brokerServicePortTls=6651 -webServicePort=8080 -webServicePortTls=4443 +applicationName=pulsar_broker +metadataStoreUrl=zk:localhost:2181/ledger +configurationMetadataStoreUrl=zk:localhost:2181 +brokerServicePort=0 +brokerServicePortTls= +webServicePort=0 +webServicePortTls= bindAddress=0.0.0.0 advertisedAddress= advertisedListeners=internal:pulsar://192.168.1.11:6660,internal:pulsar+ssl://192.168.1.11:6651 internalListenerName=internal -clusterName="test_cluster" +clusterName=test_cluster brokerShutdownTimeoutMs=3000 backlogQuotaCheckEnabled=true backlogQuotaCheckIntervalInSeconds=60 @@ -49,11 +49,11 @@ tlsTrustCertsFilePath= tlsAllowInsecureConnection=false authenticationEnabled=false authorizationEnabled=false -superUserRoles="test_user" -brokerClientAuthenticationPlugin="org.apache.pulsar.client.impl.auth.AuthenticationDisabled" +superUserRoles=test_user +brokerClientAuthenticationPlugin=org.apache.pulsar.client.impl.auth.AuthenticationDisabled brokerClientAuthenticationParameters= -bookkeeperClientAuthenticationPlugin="test_auth_plugin" -bookkeeperClientAuthenticationAppId="test_auth_id" +bookkeeperClientAuthenticationPlugin= +bookkeeperClientAuthenticationAppId= bookkeeperClientTimeoutInSeconds=30 bookkeeperClientSpeculativeReadTimeoutInMillis=0 bookkeeperClientHealthCheckEnabled=true @@ -65,7 +65,7 @@ bookkeeperClientRegionawarePolicyEnabled=false bookkeeperClientMinNumRacksPerWriteQuorum=2 bookkeeperClientEnforceMinNumRacksPerWriteQuorum=false bookkeeperClientReorderReadSequenceEnabled=false -bookkeeperClientIsolationGroups="test_group" +bookkeeperClientIsolationGroups= managedLedgerDefaultEnsembleSize=3 managedLedgerDefaultWriteQuorum=2 managedLedgerDefaultAckQuorum=2 diff --git a/pulsar-broker/src/test/resources/configurations/pulsar_broker_test_standalone_with_rocksdb.conf b/pulsar-broker/src/test/resources/configurations/pulsar_broker_test_standalone_with_rocksdb.conf index d8b26bbbfa99d..46c876686b05b 100644 --- a/pulsar-broker/src/test/resources/configurations/pulsar_broker_test_standalone_with_rocksdb.conf +++ b/pulsar-broker/src/test/resources/configurations/pulsar_broker_test_standalone_with_rocksdb.conf @@ -17,19 +17,19 @@ # under the License. # -applicationName="pulsar_broker" +applicationName=pulsar_broker metadataStoreUrl= configurationMetadataStoreUrl= -brokerServicePort=6650 -brokerServicePortTls=6651 -webServicePort=8080 +brokerServicePort=0 +brokerServicePortTls= +webServicePort=0 allowLoopback=true -webServicePortTls=4443 +webServicePortTls= bindAddress=0.0.0.0 advertisedAddress= advertisedListeners= internalListenerName=internal -clusterName="test_cluster" +clusterName=test_cluster brokerShutdownTimeoutMs=3000 backlogQuotaCheckEnabled=true backlogQuotaCheckIntervalInSeconds=60 @@ -44,17 +44,17 @@ clientLibraryVersionCheckEnabled=false clientLibraryVersionCheckAllowUnversioned=true statusFilePath=/tmp/status.html tlsEnabled=false -tlsCertificateFilePath=/usr/local/conf/pulsar/server.crt -tlsKeyFilePath=/home/local/conf/pulsar/server.key +tlsCertificateFilePath= +tlsKeyFilePath= tlsTrustCertsFilePath= tlsAllowInsecureConnection=false authenticationEnabled=false authorizationEnabled=false -superUserRoles="test_user" -brokerClientAuthenticationPlugin="org.apache.pulsar.client.impl.auth.AuthenticationDisabled" +superUserRoles=test_user +brokerClientAuthenticationPlugin=org.apache.pulsar.client.impl.auth.AuthenticationDisabled brokerClientAuthenticationParameters= -bookkeeperClientAuthenticationPlugin="test_auth_plugin" -bookkeeperClientAuthenticationAppId="test_auth_id" +bookkeeperClientAuthenticationPlugin= +bookkeeperClientAuthenticationAppId=test_auth_id bookkeeperClientTimeoutInSeconds=30 bookkeeperClientSpeculativeReadTimeoutInMillis=0 bookkeeperClientHealthCheckEnabled=true @@ -66,7 +66,7 @@ bookkeeperClientRegionawarePolicyEnabled=false bookkeeperClientMinNumRacksPerWriteQuorum=2 bookkeeperClientEnforceMinNumRacksPerWriteQuorum=false bookkeeperClientReorderReadSequenceEnabled=false -bookkeeperClientIsolationGroups="test_group" +bookkeeperClientIsolationGroups=test_group managedLedgerDefaultEnsembleSize=3 managedLedgerDefaultWriteQuorum=2 managedLedgerDefaultAckQuorum=2 diff --git a/pulsar-broker/src/test/resources/configurations/standalone_no_client_auth.conf b/pulsar-broker/src/test/resources/configurations/standalone_no_client_auth.conf index 4e2fd40298354..6f0d82cef17bc 100644 --- a/pulsar-broker/src/test/resources/configurations/standalone_no_client_auth.conf +++ b/pulsar-broker/src/test/resources/configurations/standalone_no_client_auth.conf @@ -17,8 +17,8 @@ # under the License. # -brokerServicePort=6650 -webServicePort=8080 +brokerServicePort=0 +webServicePort=0 allowLoopback=true clusterName=test_cluster superUserRoles=admin diff --git a/pulsar-client-api/src/main/java/org/apache/pulsar/client/api/PulsarClient.java b/pulsar-client-api/src/main/java/org/apache/pulsar/client/api/PulsarClient.java index 78952fcaed8b3..6c46bce254f6f 100644 --- a/pulsar-client-api/src/main/java/org/apache/pulsar/client/api/PulsarClient.java +++ b/pulsar-client-api/src/main/java/org/apache/pulsar/client/api/PulsarClient.java @@ -308,14 +308,33 @@ static ClientBuilder builder() { * *

This can be used to discover the partitions and create {@link Reader}, {@link Consumer} or {@link Producer} * instances directly on a particular partition. - * + * @Deprecated it is not suggested to use now; please use {@link #getPartitionsForTopic(String, boolean)}. * @param topic * the topic name * @return a future that will yield a list of the topic partitions or {@link PulsarClientException} if there was any * error in the operation. + * * @since 2.3.0 */ - CompletableFuture> getPartitionsForTopic(String topic); + @Deprecated + default CompletableFuture> getPartitionsForTopic(String topic) { + return getPartitionsForTopic(topic, true); + } + + /** + * 1. Get the partitions if the topic exists. Return "[{partition-0}, {partition-1}....{partition-n}}]" if a + * partitioned topic exists; return "[{topic}]" if a non-partitioned topic exists. + * 2. When {@param metadataAutoCreationEnabled} is "false", neither the partitioned topic nor non-partitioned + * topic does not exist. You will get an {@link PulsarClientException.NotFoundException} or a + * {@link PulsarClientException.TopicDoesNotExistException}. + * 2-1. You will get a {@link PulsarClientException.NotSupportedException} with metadataAutoCreationEnabled=false + * on an old broker version which does not support getting partitions without partitioned metadata auto-creation. + * 3. When {@param metadataAutoCreationEnabled} is "true," it will trigger an auto-creation for this topic(using + * the default topic auto-creation strategy you set for the broker), and the corresponding result is returned. + * For the result, see case 1. + * @version 3.3.0. + */ + CompletableFuture> getPartitionsForTopic(String topic, boolean metadataAutoCreationEnabled); /** * Close the PulsarClient and release all the resources. diff --git a/pulsar-client/src/main/java/org/apache/pulsar/client/impl/BinaryProtoLookupService.java b/pulsar-client/src/main/java/org/apache/pulsar/client/impl/BinaryProtoLookupService.java index 9d01d863143e2..980e8a0c78627 100644 --- a/pulsar-client/src/main/java/org/apache/pulsar/client/impl/BinaryProtoLookupService.java +++ b/pulsar-client/src/main/java/org/apache/pulsar/client/impl/BinaryProtoLookupService.java @@ -121,12 +121,14 @@ public CompletableFuture> getBroker(T * calls broker binaryProto-lookup api to get metadata of partitioned-topic. * */ - public CompletableFuture getPartitionedTopicMetadata(TopicName topicName) { + @Override + public CompletableFuture getPartitionedTopicMetadata( + TopicName topicName, boolean metadataAutoCreationEnabled) { final MutableObject newFutureCreated = new MutableObject<>(); try { return partitionedMetadataInProgress.computeIfAbsent(topicName, tpName -> { - CompletableFuture newFuture = - getPartitionedTopicMetadata(serviceNameResolver.resolveHost(), topicName); + CompletableFuture newFuture = getPartitionedTopicMetadata( + serviceNameResolver.resolveHost(), topicName, metadataAutoCreationEnabled); newFutureCreated.setValue(newFuture); return newFuture; }); @@ -222,13 +224,20 @@ private CompletableFuture> findBroker } private CompletableFuture getPartitionedTopicMetadata(InetSocketAddress socketAddress, - TopicName topicName) { + TopicName topicName, boolean metadataAutoCreationEnabled) { CompletableFuture partitionFuture = new CompletableFuture<>(); client.getCnxPool().getConnection(socketAddress).thenAccept(clientCnx -> { + if (!metadataAutoCreationEnabled && !clientCnx.isSupportsGetPartitionedMetadataWithoutAutoCreation()) { + partitionFuture.completeExceptionally(new PulsarClientException.NotSupportedException("The feature of" + + " getting partitions without auto-creation is not supported from the broker," + + " please upgrade the broker to the latest version.")); + return; + } long requestId = client.newRequestId(); - ByteBuf request = Commands.newPartitionMetadataRequest(topicName.toString(), requestId); + ByteBuf request = Commands.newPartitionMetadataRequest(topicName.toString(), requestId, + metadataAutoCreationEnabled); clientCnx.newLookup(request, requestId).whenComplete((r, t) -> { if (t != null) { log.warn("[{}] failed to get Partitioned metadata : {}", topicName, diff --git a/pulsar-client/src/main/java/org/apache/pulsar/client/impl/ClientCnx.java b/pulsar-client/src/main/java/org/apache/pulsar/client/impl/ClientCnx.java index 8694dad7a2d84..b8caa0f438361 100644 --- a/pulsar-client/src/main/java/org/apache/pulsar/client/impl/ClientCnx.java +++ b/pulsar-client/src/main/java/org/apache/pulsar/client/impl/ClientCnx.java @@ -186,6 +186,8 @@ public class ClientCnx extends PulsarHandler { protected AuthenticationDataProvider authenticationDataProvider; private TransactionBufferHandler transactionBufferHandler; private boolean supportsTopicWatchers; + @Getter + private boolean supportsGetPartitionedMetadataWithoutAutoCreation; /** Idle stat. **/ @Getter @@ -382,6 +384,9 @@ protected void handleConnected(CommandConnected connected) { supportsTopicWatchers = connected.hasFeatureFlags() && connected.getFeatureFlags().isSupportsTopicWatchers(); + supportsGetPartitionedMetadataWithoutAutoCreation = + connected.hasFeatureFlags() + && connected.getFeatureFlags().isSupportsGetPartitionedMetadataWithoutAutoCreation(); // set remote protocol version to the correct version before we complete the connection future setRemoteEndpointProtocolVersion(connected.getProtocolVersion()); diff --git a/pulsar-client/src/main/java/org/apache/pulsar/client/impl/ConnectionHandler.java b/pulsar-client/src/main/java/org/apache/pulsar/client/impl/ConnectionHandler.java index 6403d48d7be0c..2b7fb90b14a47 100644 --- a/pulsar-client/src/main/java/org/apache/pulsar/client/impl/ConnectionHandler.java +++ b/pulsar-client/src/main/java/org/apache/pulsar/client/impl/ConnectionHandler.java @@ -96,9 +96,16 @@ protected void grabCnx() { try { CompletableFuture cnxFuture; if (state.redirectedClusterURI != null) { - InetSocketAddress address = InetSocketAddress.createUnresolved(state.redirectedClusterURI.getHost(), - state.redirectedClusterURI.getPort()); - cnxFuture = state.client.getConnection(address, address, randomKeyForSelectConnection); + if (state.topic == null) { + InetSocketAddress address = InetSocketAddress.createUnresolved(state.redirectedClusterURI.getHost(), + state.redirectedClusterURI.getPort()); + cnxFuture = state.client.getConnection(address, address, randomKeyForSelectConnection); + } else { + // once, client receives redirection url, client has to perform lookup on migrated + // cluster to find the broker that owns the topic and then create connection. + // below method, performs the lookup for a given topic and then creates connection + cnxFuture = state.client.getConnection(state.topic, (state.redirectedClusterURI.toString())); + } } else if (state.topic == null) { cnxFuture = state.client.getConnectionToServiceUrl(); } else { diff --git a/pulsar-client/src/main/java/org/apache/pulsar/client/impl/ConsumerBuilderImpl.java b/pulsar-client/src/main/java/org/apache/pulsar/client/impl/ConsumerBuilderImpl.java index 7686d0072cffb..4d6cf96a01068 100644 --- a/pulsar-client/src/main/java/org/apache/pulsar/client/impl/ConsumerBuilderImpl.java +++ b/pulsar-client/src/main/java/org/apache/pulsar/client/impl/ConsumerBuilderImpl.java @@ -32,6 +32,7 @@ import lombok.Getter; import lombok.NonNull; import org.apache.commons.lang3.StringUtils; +import org.apache.pulsar.client.admin.PulsarAdminException; import org.apache.pulsar.client.api.BatchReceivePolicy; import org.apache.pulsar.client.api.Consumer; import org.apache.pulsar.client.api.ConsumerBuilder; @@ -58,7 +59,6 @@ import org.apache.pulsar.client.impl.conf.TopicConsumerConfigurationData; import org.apache.pulsar.client.util.RetryMessageUtil; import org.apache.pulsar.common.naming.TopicName; -import org.apache.pulsar.common.partition.PartitionedTopicMetadata; import org.apache.pulsar.common.util.FutureUtil; @Getter(AccessLevel.PUBLIC) @@ -104,6 +104,31 @@ public Consumer subscribe() throws PulsarClientException { } } + private CompletableFuture checkDlqAlreadyExists(String topic) { + CompletableFuture existsFuture = new CompletableFuture<>(); + client.getPartitionedTopicMetadata(topic, false).thenAccept(metadata -> { + TopicName topicName = TopicName.get(topic); + if (topicName.isPersistent()) { + // Either partitioned or non-partitioned, it exists. + existsFuture.complete(true); + } else { + // If it is a non-persistent topic, return true only it is a partitioned topic. + existsFuture.complete(metadata != null && metadata.partitions > 0); + } + }).exceptionally(ex -> { + Throwable actEx = FutureUtil.unwrapCompletionException(ex); + if (actEx instanceof PulsarClientException.NotFoundException + || actEx instanceof PulsarClientException.TopicDoesNotExistException + || actEx instanceof PulsarAdminException.NotFoundException) { + existsFuture.complete(false); + } else { + existsFuture.completeExceptionally(ex); + } + return null; + }); + return existsFuture; + } + @Override public CompletableFuture> subscribeAsync() { if (conf.getTopicNames().isEmpty() && conf.getTopicsPattern() == null) { @@ -135,20 +160,18 @@ public CompletableFuture> subscribeAsync() { DeadLetterPolicy deadLetterPolicy = conf.getDeadLetterPolicy(); if (deadLetterPolicy == null || StringUtils.isBlank(deadLetterPolicy.getRetryLetterTopic()) || StringUtils.isBlank(deadLetterPolicy.getDeadLetterTopic())) { - CompletableFuture retryLetterTopicMetadata = - client.getPartitionedTopicMetadata(oldRetryLetterTopic); - CompletableFuture deadLetterTopicMetadata = - client.getPartitionedTopicMetadata(oldDeadLetterTopic); + CompletableFuture retryLetterTopicMetadata = checkDlqAlreadyExists(oldRetryLetterTopic); + CompletableFuture deadLetterTopicMetadata = checkDlqAlreadyExists(oldDeadLetterTopic); applyDLQConfig = CompletableFuture.allOf(retryLetterTopicMetadata, deadLetterTopicMetadata) .thenAccept(__ -> { String retryLetterTopic = topicFirst + "-" + conf.getSubscriptionName() + RetryMessageUtil.RETRY_GROUP_TOPIC_SUFFIX; String deadLetterTopic = topicFirst + "-" + conf.getSubscriptionName() + RetryMessageUtil.DLQ_GROUP_TOPIC_SUFFIX; - if (retryLetterTopicMetadata.join().partitions > 0) { + if (retryLetterTopicMetadata.join()) { retryLetterTopic = oldRetryLetterTopic; } - if (deadLetterTopicMetadata.join().partitions > 0) { + if (deadLetterTopicMetadata.join()) { deadLetterTopic = oldDeadLetterTopic; } if (deadLetterPolicy == null) { diff --git a/pulsar-client/src/main/java/org/apache/pulsar/client/impl/HttpLookupService.java b/pulsar-client/src/main/java/org/apache/pulsar/client/impl/HttpLookupService.java index 6e8c2b4314e17..ba04aaa3b3117 100644 --- a/pulsar-client/src/main/java/org/apache/pulsar/client/impl/HttpLookupService.java +++ b/pulsar-client/src/main/java/org/apache/pulsar/client/impl/HttpLookupService.java @@ -108,10 +108,11 @@ public CompletableFuture> getBroker(T } @Override - public CompletableFuture getPartitionedTopicMetadata(TopicName topicName) { + public CompletableFuture getPartitionedTopicMetadata( + TopicName topicName, boolean metadataAutoCreationEnabled) { String format = topicName.isV2() ? "admin/v2/%s/partitions" : "admin/%s/partitions"; - return httpClient.get(String.format(format, topicName.getLookupName()) + "?checkAllowAutoCreation=true", - PartitionedTopicMetadata.class); + return httpClient.get(String.format(format, topicName.getLookupName()) + "?checkAllowAutoCreation=" + + metadataAutoCreationEnabled, PartitionedTopicMetadata.class); } @Override diff --git a/pulsar-client/src/main/java/org/apache/pulsar/client/impl/LookupService.java b/pulsar-client/src/main/java/org/apache/pulsar/client/impl/LookupService.java index 48ef67eae2047..ba99cb77550f5 100644 --- a/pulsar-client/src/main/java/org/apache/pulsar/client/impl/LookupService.java +++ b/pulsar-client/src/main/java/org/apache/pulsar/client/impl/LookupService.java @@ -60,11 +60,30 @@ public interface LookupService extends AutoCloseable { /** * Returns {@link PartitionedTopicMetadata} for a given topic. - * - * @param topicName topic-name - * @return + * Note: this method will try to create the topic partitioned metadata if it does not exist. + * @deprecated Please call {{@link #getPartitionedTopicMetadata(TopicName, boolean)}}. */ - CompletableFuture getPartitionedTopicMetadata(TopicName topicName); + @Deprecated + default CompletableFuture getPartitionedTopicMetadata(TopicName topicName) { + return getPartitionedTopicMetadata(topicName, true); + } + + /** + * 1.Get the partitions if the topic exists. Return "{partition: n}" if a partitioned topic exists; + * return "{partition: 0}" if a non-partitioned topic exists. + * 2. When {@param metadataAutoCreationEnabled} is "false," neither partitioned topic nor non-partitioned topic + * does not exist. You will get a {@link PulsarClientException.NotFoundException} or + * a {@link PulsarClientException.TopicDoesNotExistException}. + * 2-1. You will get a {@link PulsarClientException.NotSupportedException} with metadataAutoCreationEnabled=false + * on an old broker version which does not support getting partitions without partitioned metadata + * auto-creation. + * 3.When {@param metadataAutoCreationEnabled} is "true," it will trigger an auto-creation for this topic(using + * the default topic auto-creation strategy you set for the broker), and the corresponding result is returned. + * For the result, see case 1. + * @version 3.3.0. + */ + CompletableFuture getPartitionedTopicMetadata(TopicName topicName, + boolean metadataAutoCreationEnabled); /** * Returns current SchemaInfo {@link SchemaInfo} for a given topic. @@ -105,5 +124,4 @@ public interface LookupService extends AutoCloseable { */ CompletableFuture getTopicsUnderNamespace(NamespaceName namespace, Mode mode, String topicPattern, String topicsHash); - } diff --git a/pulsar-client/src/main/java/org/apache/pulsar/client/impl/MultiTopicsConsumerImpl.java b/pulsar-client/src/main/java/org/apache/pulsar/client/impl/MultiTopicsConsumerImpl.java index f2bce59a1e68e..62b6612fa3c26 100644 --- a/pulsar-client/src/main/java/org/apache/pulsar/client/impl/MultiTopicsConsumerImpl.java +++ b/pulsar-client/src/main/java/org/apache/pulsar/client/impl/MultiTopicsConsumerImpl.java @@ -954,7 +954,7 @@ public CompletableFuture subscribeAsync(String topicName, boolean createTo CompletableFuture subscribeResult = new CompletableFuture<>(); - client.getPartitionedTopicMetadata(topicName) + client.getPartitionedTopicMetadata(topicName, true) .thenAccept(metadata -> subscribeTopicPartitions(subscribeResult, fullTopicName, metadata.partitions, createTopicIfDoesNotExist)) .exceptionally(ex1 -> { diff --git a/pulsar-client/src/main/java/org/apache/pulsar/client/impl/PulsarClientImpl.java b/pulsar-client/src/main/java/org/apache/pulsar/client/impl/PulsarClientImpl.java index b92e039e5facd..899e4e8fae7c7 100644 --- a/pulsar-client/src/main/java/org/apache/pulsar/client/impl/PulsarClientImpl.java +++ b/pulsar-client/src/main/java/org/apache/pulsar/client/impl/PulsarClientImpl.java @@ -32,7 +32,9 @@ import java.time.Duration; import java.util.ArrayList; import java.util.Collections; +import java.util.HashMap; import java.util.List; +import java.util.Map; import java.util.Optional; import java.util.Set; import java.util.concurrent.CompletableFuture; @@ -106,6 +108,7 @@ public class PulsarClientImpl implements PulsarClient { private final boolean createdScheduledProviders; private LookupService lookup; + private Map urlLookupMap = new ConcurrentHashMap<>(); private final ConnectionPool cnxPool; @Getter private final Timer timer; @@ -377,7 +380,7 @@ private CompletableFuture> createProducerAsync(String topic, ProducerInterceptors interceptors) { CompletableFuture> producerCreatedFuture = new CompletableFuture<>(); - getPartitionedTopicMetadata(topic).thenAccept(metadata -> { + getPartitionedTopicMetadata(topic, true).thenAccept(metadata -> { if (log.isDebugEnabled()) { log.debug("[{}] Received topic metadata. partitions: {}", topic, metadata.partitions); } @@ -519,7 +522,7 @@ private CompletableFuture> doSingleTopicSubscribeAsync(ConsumerC String topic = conf.getSingleTopic(); - getPartitionedTopicMetadata(topic).thenAccept(metadata -> { + getPartitionedTopicMetadata(topic, true).thenAccept(metadata -> { if (log.isDebugEnabled()) { log.debug("[{}] Received topic metadata. partitions: {}", topic, metadata.partitions); } @@ -659,7 +662,7 @@ protected CompletableFuture> createSingleTopicReaderAsync( CompletableFuture> readerFuture = new CompletableFuture<>(); - getPartitionedTopicMetadata(topic).thenAccept(metadata -> { + getPartitionedTopicMetadata(topic, true).thenAccept(metadata -> { if (log.isDebugEnabled()) { log.debug("[{}] Received topic metadata. partitions: {}", topic, metadata.partitions); } @@ -733,6 +736,21 @@ public void close() throws PulsarClientException { } } + private void closeUrlLookupMap() { + Map closedUrlLookupServices = new HashMap(urlLookupMap.size()); + urlLookupMap.entrySet().forEach(e -> { + try { + e.getValue().close(); + } catch (Exception ex) { + log.error("Error closing lookup service {}", e.getKey(), ex); + } + closedUrlLookupServices.put(e.getKey(), e.getValue()); + }); + closedUrlLookupServices.entrySet().forEach(e -> { + urlLookupMap.remove(e.getKey(), e.getValue()); + }); + } + @Override public CompletableFuture closeAsync() { log.info("Client closing. URL: {}", lookup.getServiceUrl()); @@ -743,6 +761,8 @@ public CompletableFuture closeAsync() { final CompletableFuture closeFuture = new CompletableFuture<>(); List> futures = new ArrayList<>(); + closeUrlLookupMap(); + producers.forEach(p -> futures.add(p.closeAsync().handle((__, t) -> { if (t != null) { log.error("Error closing producer {}", p, t); @@ -962,6 +982,27 @@ public CompletableFuture getConnection(final String topic) { .thenCompose(pair -> getConnection(pair.getLeft(), pair.getRight(), cnxPool.genRandomKeyToSelectCon())); } + public CompletableFuture getConnection(final String topic, final String url) { + TopicName topicName = TopicName.get(topic); + return getLookup(url).getBroker(topicName) + .thenCompose(pair -> getConnection(pair.getLeft(), pair.getRight(), cnxPool.genRandomKeyToSelectCon())); + } + + public LookupService getLookup(String serviceUrl) { + return urlLookupMap.computeIfAbsent(serviceUrl, url -> { + if (isClosed()) { + throw new IllegalStateException("Pulsar client has been closed, can not build LookupService when" + + " calling get lookup with an url"); + } + try { + return createLookup(serviceUrl); + } catch (PulsarClientException e) { + log.warn("Failed to update url to lookup service {}, {}", url, e.getMessage()); + throw new IllegalStateException("Failed to update url " + url); + } + }); + } + public CompletableFuture getConnectionToServiceUrl() { if (!(lookup instanceof BinaryProtoLookupService)) { return FutureUtil.failedFuture(new PulsarClientException.InvalidServiceURL( @@ -1020,19 +1061,20 @@ public LookupService getLookup() { } public void reloadLookUp() throws PulsarClientException { - if (conf.getServiceUrl().startsWith("http")) { - lookup = new HttpLookupService(conf, eventLoopGroup); + lookup = createLookup(conf.getServiceUrl()); + } + + public LookupService createLookup(String url) throws PulsarClientException { + if (url.startsWith("http")) { + return new HttpLookupService(conf, eventLoopGroup); } else { - lookup = new BinaryProtoLookupService(this, conf.getServiceUrl(), conf.getListenerName(), conf.isUseTls(), + return new BinaryProtoLookupService(this, url, conf.getListenerName(), conf.isUseTls(), externalExecutorProvider.getExecutor()); } } - public CompletableFuture getNumberOfPartitions(String topic) { - return getPartitionedTopicMetadata(topic).thenApply(metadata -> metadata.partitions); - } - - public CompletableFuture getPartitionedTopicMetadata(String topic) { + public CompletableFuture getPartitionedTopicMetadata( + String topic, boolean metadataAutoCreationEnabled) { CompletableFuture metadataFuture = new CompletableFuture<>(); @@ -1045,7 +1087,7 @@ public CompletableFuture getPartitionedTopicMetadata(S .setMax(conf.getMaxBackoffIntervalNanos(), TimeUnit.NANOSECONDS) .create(); getPartitionedTopicMetadata(topicName, backoff, opTimeoutMs, - metadataFuture, new ArrayList<>()); + metadataFuture, new ArrayList<>(), metadataAutoCreationEnabled); } catch (IllegalArgumentException e) { return FutureUtil.failedFuture(new PulsarClientException.InvalidConfigurationException(e.getMessage())); } @@ -1056,15 +1098,19 @@ private void getPartitionedTopicMetadata(TopicName topicName, Backoff backoff, AtomicLong remainingTime, CompletableFuture future, - List previousExceptions) { + List previousExceptions, + boolean metadataAutoCreationEnabled) { long startTime = System.nanoTime(); - lookup.getPartitionedTopicMetadata(topicName).thenAccept(future::complete).exceptionally(e -> { + CompletableFuture queryFuture = + lookup.getPartitionedTopicMetadata(topicName, metadataAutoCreationEnabled); + queryFuture.thenAccept(future::complete).exceptionally(e -> { remainingTime.addAndGet(-1 * TimeUnit.NANOSECONDS.toMillis(System.nanoTime() - startTime)); long nextDelay = Math.min(backoff.next(), remainingTime.get()); // skip retry scheduler when set lookup throttle in client or server side which will lead to // `TooManyRequestsException` boolean isLookupThrottling = !PulsarClientException.isRetriableError(e.getCause()) - || e.getCause() instanceof PulsarClientException.AuthenticationException; + || e.getCause() instanceof PulsarClientException.AuthenticationException + || e.getCause() instanceof PulsarClientException.NotFoundException; if (nextDelay <= 0 || isLookupThrottling) { PulsarClientException.setPreviousExceptions(e, previousExceptions); future.completeExceptionally(e); @@ -1076,15 +1122,16 @@ private void getPartitionedTopicMetadata(TopicName topicName, log.warn("[topic: {}] Could not get connection while getPartitionedTopicMetadata -- " + "Will try again in {} ms", topicName, nextDelay); remainingTime.addAndGet(-nextDelay); - getPartitionedTopicMetadata(topicName, backoff, remainingTime, future, previousExceptions); + getPartitionedTopicMetadata(topicName, backoff, remainingTime, future, previousExceptions, + metadataAutoCreationEnabled); }, nextDelay, TimeUnit.MILLISECONDS); return null; }); } @Override - public CompletableFuture> getPartitionsForTopic(String topic) { - return getPartitionedTopicMetadata(topic).thenApply(metadata -> { + public CompletableFuture> getPartitionsForTopic(String topic, boolean metadataAutoCreationEnabled) { + return getPartitionedTopicMetadata(topic, metadataAutoCreationEnabled).thenApply(metadata -> { if (metadata.partitions > 0) { TopicName topicName = TopicName.get(topic); List partitions = new ArrayList<>(metadata.partitions); diff --git a/pulsar-client/src/main/java/org/apache/pulsar/client/impl/transaction/TransactionCoordinatorClientImpl.java b/pulsar-client/src/main/java/org/apache/pulsar/client/impl/transaction/TransactionCoordinatorClientImpl.java index 9e79fc203c225..499627f9c73f2 100644 --- a/pulsar-client/src/main/java/org/apache/pulsar/client/impl/transaction/TransactionCoordinatorClientImpl.java +++ b/pulsar-client/src/main/java/org/apache/pulsar/client/impl/transaction/TransactionCoordinatorClientImpl.java @@ -79,7 +79,8 @@ public void start() throws TransactionCoordinatorClientException { @Override public CompletableFuture startAsync() { if (STATE_UPDATER.compareAndSet(this, State.NONE, State.STARTING)) { - return pulsarClient.getLookup().getPartitionedTopicMetadata(SystemTopicNames.TRANSACTION_COORDINATOR_ASSIGN) + return pulsarClient.getLookup() + .getPartitionedTopicMetadata(SystemTopicNames.TRANSACTION_COORDINATOR_ASSIGN, true) .thenCompose(partitionMeta -> { List> connectFutureList = new ArrayList<>(); if (LOG.isDebugEnabled()) { diff --git a/pulsar-client/src/test/java/org/apache/pulsar/client/impl/MultiTopicsConsumerImplTest.java b/pulsar-client/src/test/java/org/apache/pulsar/client/impl/MultiTopicsConsumerImplTest.java index febec2bff3285..191124bb7b002 100644 --- a/pulsar-client/src/test/java/org/apache/pulsar/client/impl/MultiTopicsConsumerImplTest.java +++ b/pulsar-client/src/test/java/org/apache/pulsar/client/impl/MultiTopicsConsumerImplTest.java @@ -22,6 +22,7 @@ import static org.apache.pulsar.client.impl.ClientTestFixtures.createExceptionFuture; import static org.apache.pulsar.client.impl.ClientTestFixtures.createPulsarClientMockWithMockedClientCnx; import static org.mockito.ArgumentMatchers.any; +import static org.mockito.ArgumentMatchers.anyBoolean; import static org.mockito.Mockito.mock; import static org.mockito.Mockito.times; import static org.mockito.Mockito.verify; @@ -153,7 +154,8 @@ private MultiTopicsConsumerImpl createMultiTopicsConsumer( int completionDelayMillis = 100; Schema schema = Schema.BYTES; PulsarClientImpl clientMock = createPulsarClientMockWithMockedClientCnx(executorProvider, internalExecutor); - when(clientMock.getPartitionedTopicMetadata(any())).thenAnswer(invocation -> createDelayedCompletedFuture( + when(clientMock.getPartitionedTopicMetadata(any(), anyBoolean())) + .thenAnswer(invocation -> createDelayedCompletedFuture( new PartitionedTopicMetadata(), completionDelayMillis)); MultiTopicsConsumerImpl impl = new MultiTopicsConsumerImpl( clientMock, consumerConfData, executorProvider, @@ -201,7 +203,8 @@ public void testConsumerCleanupOnSubscribeFailure() { int completionDelayMillis = 10; Schema schema = Schema.BYTES; PulsarClientImpl clientMock = createPulsarClientMockWithMockedClientCnx(executorProvider, internalExecutor); - when(clientMock.getPartitionedTopicMetadata(any())).thenAnswer(invocation -> createExceptionFuture( + when(clientMock.getPartitionedTopicMetadata(any(), anyBoolean())) + .thenAnswer(invocation -> createExceptionFuture( new PulsarClientException.InvalidConfigurationException("a mock exception"), completionDelayMillis)); CompletableFuture> completeFuture = new CompletableFuture<>(); MultiTopicsConsumerImpl impl = new MultiTopicsConsumerImpl(clientMock, consumerConfData, @@ -237,7 +240,8 @@ public void testDontCheckForPartitionsUpdatesOnNonPartitionedTopics() throws Exc // Simulate non partitioned topics PartitionedTopicMetadata metadata = new PartitionedTopicMetadata(0); - when(clientMock.getPartitionedTopicMetadata(any())).thenReturn(CompletableFuture.completedFuture(metadata)); + when(clientMock.getPartitionedTopicMetadata(any(), anyBoolean())) + .thenReturn(CompletableFuture.completedFuture(metadata)); CompletableFuture> completeFuture = new CompletableFuture<>(); MultiTopicsConsumerImpl impl = new MultiTopicsConsumerImpl<>( @@ -248,7 +252,7 @@ public void testDontCheckForPartitionsUpdatesOnNonPartitionedTopics() throws Exc // getPartitionedTopicMetadata should have been called only the first time, for each of the 3 topics, // but not anymore since the topics are not partitioned. - verify(clientMock, times(3)).getPartitionedTopicMetadata(any()); + verify(clientMock, times(3)).getPartitionedTopicMetadata(any(), anyBoolean()); } } diff --git a/pulsar-client/src/test/java/org/apache/pulsar/client/impl/PulsarClientImplTest.java b/pulsar-client/src/test/java/org/apache/pulsar/client/impl/PulsarClientImplTest.java index e0b25db891247..e13c060a052ec 100644 --- a/pulsar-client/src/test/java/org/apache/pulsar/client/impl/PulsarClientImplTest.java +++ b/pulsar-client/src/test/java/org/apache/pulsar/client/impl/PulsarClientImplTest.java @@ -19,6 +19,7 @@ package org.apache.pulsar.client.impl; import static org.mockito.ArgumentMatchers.any; +import static org.mockito.ArgumentMatchers.anyBoolean; import static org.mockito.ArgumentMatchers.anyInt; import static org.mockito.ArgumentMatchers.anyLong; import static org.mockito.ArgumentMatchers.nullable; @@ -106,7 +107,7 @@ public void testConsumerIsClosed() throws Exception { nullable(String.class))) .thenReturn(CompletableFuture.completedFuture( new GetTopicsResult(Collections.emptyList(), null, false, true))); - when(lookup.getPartitionedTopicMetadata(any(TopicName.class))) + when(lookup.getPartitionedTopicMetadata(any(TopicName.class), anyBoolean())) .thenReturn(CompletableFuture.completedFuture(new PartitionedTopicMetadata())); when(lookup.getBroker(any())) .thenReturn(CompletableFuture.completedFuture( diff --git a/pulsar-client/src/test/java/org/apache/pulsar/client/impl/TableViewBuilderImplTest.java b/pulsar-client/src/test/java/org/apache/pulsar/client/impl/TableViewBuilderImplTest.java index eee8ba4e8f41a..01353e47cd0cb 100644 --- a/pulsar-client/src/test/java/org/apache/pulsar/client/impl/TableViewBuilderImplTest.java +++ b/pulsar-client/src/test/java/org/apache/pulsar/client/impl/TableViewBuilderImplTest.java @@ -18,6 +18,14 @@ */ package org.apache.pulsar.client.impl; +import static org.mockito.Mockito.any; +import static org.mockito.Mockito.mock; +import static org.mockito.Mockito.when; +import static org.testng.Assert.assertNotNull; +import java.util.HashMap; +import java.util.Map; +import java.util.concurrent.CompletableFuture; +import java.util.concurrent.TimeUnit; import org.apache.pulsar.client.api.ConsumerCryptoFailureAction; import org.apache.pulsar.client.api.CryptoKeyReader; import org.apache.pulsar.client.api.PulsarClientException; @@ -25,32 +33,25 @@ import org.apache.pulsar.client.api.Schema; import org.apache.pulsar.client.api.TableView; import org.apache.pulsar.client.impl.conf.ReaderConfigurationData; +import org.testng.annotations.AfterClass; import org.testng.annotations.BeforeClass; import org.testng.annotations.Test; -import java.util.HashMap; -import java.util.Map; -import java.util.concurrent.CompletableFuture; -import java.util.concurrent.TimeUnit; - -import static org.mockito.Mockito.any; -import static org.mockito.Mockito.mock; -import static org.mockito.Mockito.when; -import static org.testng.Assert.assertNotNull; - /** - * Unit tests of {@link TablewViewBuilderImpl}. + * Unit tests of {@link TableViewBuilderImpl}. */ public class TableViewBuilderImplTest { private static final String TOPIC_NAME = "testTopicName"; private PulsarClientImpl client; private TableViewBuilderImpl tableViewBuilderImpl; + private CompletableFuture readNextFuture; @BeforeClass(alwaysRun = true) public void setup() { Reader reader = mock(Reader.class); - when(reader.readNextAsync()).thenReturn(CompletableFuture.allOf()); + readNextFuture = new CompletableFuture(); + when(reader.readNextAsync()).thenReturn(readNextFuture); client = mock(PulsarClientImpl.class); ConnectionPool connectionPool = mock(ConnectionPool.class); when(client.getCnxPool()).thenReturn(connectionPool); @@ -61,6 +62,14 @@ public void setup() { tableViewBuilderImpl = new TableViewBuilderImpl(client, Schema.BYTES); } + @AfterClass(alwaysRun = true) + public void cleanup() { + if (readNextFuture != null) { + readNextFuture.completeExceptionally(new PulsarClientException.AlreadyClosedException("Closing test case")); + readNextFuture = null; + } + } + @Test public void testTableViewBuilderImpl() throws PulsarClientException { TableView tableView = tableViewBuilderImpl.topic(TOPIC_NAME) diff --git a/pulsar-common/pom.xml b/pulsar-common/pom.xml index 2332a91a469d4..e39dd288abf6e 100644 --- a/pulsar-common/pom.xml +++ b/pulsar-common/pom.xml @@ -99,6 +99,12 @@ linux-x86_64 + + io.netty + netty-transport-native-epoll + linux-aarch_64 + + io.netty netty-transport-native-unix-common @@ -238,6 +244,11 @@ awaitility test + + + org.roaringbitmap + RoaringBitmap + diff --git a/pulsar-common/src/main/java/org/apache/pulsar/common/naming/TopicName.java b/pulsar-common/src/main/java/org/apache/pulsar/common/naming/TopicName.java index eebca0e0d7214..e051e01495dbe 100644 --- a/pulsar-common/src/main/java/org/apache/pulsar/common/naming/TopicName.java +++ b/pulsar-common/src/main/java/org/apache/pulsar/common/naming/TopicName.java @@ -358,17 +358,16 @@ public static String fromPersistenceNamingEncoding(String mlName) { String localName; if (parts.size() == 4) { tenant = parts.get(0); - cluster = null; namespacePortion = parts.get(1); domain = parts.get(2); - localName = parts.get(3); + localName = Codec.decode(parts.get(3)); return String.format("%s://%s/%s/%s", domain, tenant, namespacePortion, localName); } else if (parts.size() == 5) { tenant = parts.get(0); cluster = parts.get(1); namespacePortion = parts.get(2); domain = parts.get(3); - localName = parts.get(4); + localName = Codec.decode(parts.get(4)); return String.format("%s://%s/%s/%s/%s", domain, tenant, cluster, namespacePortion, localName); } else { throw new IllegalArgumentException("Invalid managedLedger name: " + mlName); diff --git a/pulsar-common/src/main/java/org/apache/pulsar/common/protocol/Commands.java b/pulsar-common/src/main/java/org/apache/pulsar/common/protocol/Commands.java index faa5fbcd30130..c352da0c871ed 100644 --- a/pulsar-common/src/main/java/org/apache/pulsar/common/protocol/Commands.java +++ b/pulsar-common/src/main/java/org/apache/pulsar/common/protocol/Commands.java @@ -188,6 +188,7 @@ private static void setFeatureFlags(FeatureFlags flags) { flags.setSupportsAuthRefresh(true); flags.setSupportsBrokerEntryMetadata(true); flags.setSupportsPartialProducer(true); + flags.setSupportsGetPartitionedMetadataWithoutAutoCreation(true); } public static ByteBuf newConnect(String authMethodName, String authData, int protocolVersion, String libVersion, @@ -298,6 +299,7 @@ public static BaseCommand newConnectedCommand(int clientProtocolVersion, int max connected.setProtocolVersion(versionToAdvertise); connected.setFeatureFlags().setSupportsTopicWatchers(supportsTopicWatchers); + connected.setFeatureFlags().setSupportsGetPartitionedMetadataWithoutAutoCreation(true); return cmd; } @@ -880,11 +882,13 @@ public static ByteBuf newPartitionMetadataResponse(ServerError error, String err return serializeWithSize(newPartitionMetadataResponseCommand(error, errorMsg, requestId)); } - public static ByteBuf newPartitionMetadataRequest(String topic, long requestId) { + public static ByteBuf newPartitionMetadataRequest(String topic, long requestId, + boolean metadataAutoCreationEnabled) { BaseCommand cmd = localCmd(Type.PARTITIONED_METADATA); cmd.setPartitionMetadata() .setTopic(topic) - .setRequestId(requestId); + .setRequestId(requestId) + .setMetadataAutoCreationEnabled(metadataAutoCreationEnabled); return serializeWithSize(cmd); } diff --git a/pulsar-common/src/main/java/org/apache/pulsar/common/util/FutureUtil.java b/pulsar-common/src/main/java/org/apache/pulsar/common/util/FutureUtil.java index f6fcb12f35939..0628d494af3af 100644 --- a/pulsar-common/src/main/java/org/apache/pulsar/common/util/FutureUtil.java +++ b/pulsar-common/src/main/java/org/apache/pulsar/common/util/FutureUtil.java @@ -199,9 +199,9 @@ public static CompletableFuture failedFuture(Throwable t) { public static Throwable unwrapCompletionException(Throwable ex) { if (ex instanceof CompletionException) { - return ex.getCause(); + return unwrapCompletionException(ex.getCause()); } else if (ex instanceof ExecutionException) { - return ex.getCause(); + return unwrapCompletionException(ex.getCause()); } else { return ex; } diff --git a/pulsar-common/src/main/java/org/apache/pulsar/common/util/collections/ConcurrentOpenLongPairRangeSet.java b/pulsar-common/src/main/java/org/apache/pulsar/common/util/collections/ConcurrentOpenLongPairRangeSet.java index 72215d7296cc3..b5ad89d1695d4 100644 --- a/pulsar-common/src/main/java/org/apache/pulsar/common/util/collections/ConcurrentOpenLongPairRangeSet.java +++ b/pulsar-common/src/main/java/org/apache/pulsar/common/util/collections/ConcurrentOpenLongPairRangeSet.java @@ -29,6 +29,7 @@ import java.util.concurrent.ConcurrentSkipListMap; import java.util.concurrent.atomic.AtomicBoolean; import org.apache.commons.lang.mutable.MutableInt; +import org.roaringbitmap.RoaringBitSet; /** * A Concurrent set comprising zero or more ranges of type {@link LongPair}. This can be alternative of @@ -44,7 +45,7 @@ public class ConcurrentOpenLongPairRangeSet> implements LongPairRangeSet { protected final NavigableMap rangeBitSetMap = new ConcurrentSkipListMap<>(); - private boolean threadSafe = true; + private final boolean threadSafe; private final int bitSetSize; private final LongPairConsumer consumer; @@ -95,9 +96,7 @@ public void addOpenClosed(long lowerKey, long lowerValueOpen, long upperKey, lon // (2) set 0th-index to upper-index in upperRange.getKey() if (isValid(upperKey, upperValue)) { BitSet rangeBitSet = rangeBitSetMap.computeIfAbsent(upperKey, (key) -> createNewBitSet()); - if (rangeBitSet != null) { - rangeBitSet.set(0, (int) upperValue + 1); - } + rangeBitSet.set(0, (int) upperValue + 1); } // No-op if values are not valid eg: if lower == LongPair.earliest or upper == LongPair.latest then nothing // to set @@ -414,7 +413,6 @@ private int getSafeEntry(long value) { } private BitSet createNewBitSet() { - return this.threadSafe ? new ConcurrentBitSet(bitSetSize) : new BitSet(bitSetSize); + return this.threadSafe ? new ConcurrentRoaringBitSet() : new RoaringBitSet(); } - -} +} \ No newline at end of file diff --git a/pulsar-common/src/main/java/org/apache/pulsar/common/util/collections/ConcurrentRoaringBitSet.java b/pulsar-common/src/main/java/org/apache/pulsar/common/util/collections/ConcurrentRoaringBitSet.java new file mode 100644 index 0000000000000..814e58400993b --- /dev/null +++ b/pulsar-common/src/main/java/org/apache/pulsar/common/util/collections/ConcurrentRoaringBitSet.java @@ -0,0 +1,439 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. The ASF licenses this file + * to you under the Apache License, Version 2.0 (the + * "License"); you may not use this file except in compliance + * with the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, + * software distributed under the License is distributed on an + * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY + * KIND, either express or implied. See the License for the + * specific language governing permissions and limitations + * under the License. + */ +package org.apache.pulsar.common.util.collections; + +import java.util.BitSet; +import java.util.concurrent.locks.StampedLock; +import java.util.stream.IntStream; +import org.roaringbitmap.RoaringBitSet; + +public class ConcurrentRoaringBitSet extends RoaringBitSet { + private final StampedLock rwLock = new StampedLock(); + + public ConcurrentRoaringBitSet() { + super(); + } + + @Override + public boolean get(int bitIndex) { + long stamp = rwLock.tryOptimisticRead(); + boolean isSet = super.get(bitIndex); + if (!rwLock.validate(stamp)) { + stamp = rwLock.readLock(); + try { + isSet = super.get(bitIndex); + } finally { + rwLock.unlockRead(stamp); + } + } + return isSet; + } + + @Override + public void set(int bitIndex) { + long stamp = rwLock.writeLock(); + try { + super.set(bitIndex); + } finally { + rwLock.unlockWrite(stamp); + } + } + + @Override + public void clear(int bitIndex) { + long stamp = rwLock.writeLock(); + try { + super.clear(bitIndex); + } finally { + rwLock.unlockWrite(stamp); + } + } + + @Override + public void set(int fromIndex, int toIndex) { + long stamp = rwLock.writeLock(); + try { + super.set(fromIndex, toIndex); + } finally { + rwLock.unlockWrite(stamp); + } + } + + @Override + public void clear(int fromIndex, int toIndex) { + long stamp = rwLock.writeLock(); + try { + super.clear(fromIndex, toIndex); + } finally { + rwLock.unlockWrite(stamp); + } + } + + @Override + public void clear() { + long stamp = rwLock.writeLock(); + try { + super.clear(); + } finally { + rwLock.unlockWrite(stamp); + } + } + + @Override + public int nextSetBit(int fromIndex) { + long stamp = rwLock.tryOptimisticRead(); + int nextSetBit = super.nextSetBit(fromIndex); + if (!rwLock.validate(stamp)) { + // Fallback to read lock + stamp = rwLock.readLock(); + try { + nextSetBit = super.nextSetBit(fromIndex); + } finally { + rwLock.unlockRead(stamp); + } + } + return nextSetBit; + } + + @Override + public int nextClearBit(int fromIndex) { + long stamp = rwLock.tryOptimisticRead(); + int nextClearBit = super.nextClearBit(fromIndex); + if (!rwLock.validate(stamp)) { + // Fallback to read lock + stamp = rwLock.readLock(); + try { + nextClearBit = super.nextClearBit(fromIndex); + } finally { + rwLock.unlockRead(stamp); + } + } + return nextClearBit; + } + + @Override + public int previousSetBit(int fromIndex) { + long stamp = rwLock.tryOptimisticRead(); + int previousSetBit = super.previousSetBit(fromIndex); + if (!rwLock.validate(stamp)) { + // Fallback to read lock + stamp = rwLock.readLock(); + try { + previousSetBit = super.previousSetBit(fromIndex); + } finally { + rwLock.unlockRead(stamp); + } + } + return previousSetBit; + } + + @Override + public int previousClearBit(int fromIndex) { + long stamp = rwLock.tryOptimisticRead(); + int previousClearBit = super.previousClearBit(fromIndex); + if (!rwLock.validate(stamp)) { + // Fallback to read lock + stamp = rwLock.readLock(); + try { + previousClearBit = super.previousClearBit(fromIndex); + } finally { + rwLock.unlockRead(stamp); + } + } + return previousClearBit; + } + + @Override + public int length() { + long stamp = rwLock.tryOptimisticRead(); + int length = super.length(); + if (!rwLock.validate(stamp)) { + // Fallback to read lock + stamp = rwLock.readLock(); + try { + length = super.length(); + } finally { + rwLock.unlockRead(stamp); + } + } + return length; + } + + @Override + public boolean isEmpty() { + long stamp = rwLock.tryOptimisticRead(); + boolean isEmpty = super.isEmpty(); + if (!rwLock.validate(stamp)) { + // Fallback to read lock + stamp = rwLock.readLock(); + try { + isEmpty = super.isEmpty(); + } finally { + rwLock.unlockRead(stamp); + } + } + return isEmpty; + } + + @Override + public int cardinality() { + long stamp = rwLock.tryOptimisticRead(); + int cardinality = super.cardinality(); + if (!rwLock.validate(stamp)) { + // Fallback to read lock + stamp = rwLock.readLock(); + try { + cardinality = super.cardinality(); + } finally { + rwLock.unlockRead(stamp); + } + } + return cardinality; + } + + @Override + public int size() { + long stamp = rwLock.tryOptimisticRead(); + int size = super.size(); + if (!rwLock.validate(stamp)) { + // Fallback to read lock + stamp = rwLock.readLock(); + try { + size = super.size(); + } finally { + rwLock.unlockRead(stamp); + } + } + return size; + } + + @Override + public byte[] toByteArray() { + long stamp = rwLock.tryOptimisticRead(); + byte[] byteArray = super.toByteArray(); + if (!rwLock.validate(stamp)) { + // Fallback to read lock + stamp = rwLock.readLock(); + try { + byteArray = super.toByteArray(); + } finally { + rwLock.unlockRead(stamp); + } + } + return byteArray; + } + + @Override + public long[] toLongArray() { + long stamp = rwLock.tryOptimisticRead(); + long[] longArray = super.toLongArray(); + if (!rwLock.validate(stamp)) { + // Fallback to read lock + stamp = rwLock.readLock(); + try { + longArray = super.toLongArray(); + } finally { + rwLock.unlockRead(stamp); + } + } + return longArray; + } + + @Override + public void flip(int bitIndex) { + long stamp = rwLock.writeLock(); + try { + super.flip(bitIndex); + } finally { + rwLock.unlockWrite(stamp); + } + } + + @Override + public void flip(int fromIndex, int toIndex) { + long stamp = rwLock.writeLock(); + try { + super.flip(fromIndex, toIndex); + } finally { + rwLock.unlockWrite(stamp); + } + } + + @Override + public void set(int bitIndex, boolean value) { + long stamp = rwLock.writeLock(); + try { + super.set(bitIndex, value); + } finally { + rwLock.unlockWrite(stamp); + } + } + + @Override + public void set(int fromIndex, int toIndex, boolean value) { + long stamp = rwLock.writeLock(); + try { + super.set(fromIndex, toIndex, value); + } finally { + rwLock.unlockWrite(stamp); + } + } + + @Override + public BitSet get(int fromIndex, int toIndex) { + long stamp = rwLock.tryOptimisticRead(); + BitSet bitSet = super.get(fromIndex, toIndex); + if (!rwLock.validate(stamp)) { + // Fallback to read lock + stamp = rwLock.readLock(); + try { + bitSet = super.get(fromIndex, toIndex); + } finally { + rwLock.unlockRead(stamp); + } + } + return bitSet; + } + + @Override + public boolean intersects(BitSet set) { + long stamp = rwLock.writeLock(); + try { + return super.intersects(set); + } finally { + rwLock.unlockWrite(stamp); + } + } + + @Override + public void and(BitSet set) { + long stamp = rwLock.writeLock(); + try { + super.and(set); + } finally { + rwLock.unlockWrite(stamp); + } + } + + @Override + public void or(BitSet set) { + long stamp = rwLock.writeLock(); + try { + super.or(set); + } finally { + rwLock.unlockWrite(stamp); + } + } + + @Override + public void xor(BitSet set) { + long stamp = rwLock.writeLock(); + try { + super.xor(set); + } finally { + rwLock.unlockWrite(stamp); + } + } + + @Override + public void andNot(BitSet set) { + long stamp = rwLock.writeLock(); + try { + super.andNot(set); + } finally { + rwLock.unlockWrite(stamp); + } + } + + /** + * Returns the clone of the internal wrapped {@code BitSet}. + * This won't be a clone of the {@code ConcurrentBitSet} object. + * + * @return a clone of the internal wrapped {@code BitSet} + */ + @Override + public Object clone() { + long stamp = rwLock.tryOptimisticRead(); + RoaringBitSet clone = (RoaringBitSet) super.clone(); + if (!rwLock.validate(stamp)) { + // Fallback to read lock + stamp = rwLock.readLock(); + try { + clone = (RoaringBitSet) super.clone(); + } finally { + rwLock.unlockRead(stamp); + } + } + return clone; + } + + @Override + public String toString() { + long stamp = rwLock.tryOptimisticRead(); + String str = super.toString(); + if (!rwLock.validate(stamp)) { + // Fallback to read lock + stamp = rwLock.readLock(); + try { + str = super.toString(); + } finally { + rwLock.unlockRead(stamp); + } + } + return str; + } + + /** + * This operation is not supported on {@code ConcurrentBitSet}. + */ + @Override + public IntStream stream() { + throw new UnsupportedOperationException("stream is not supported"); + } + + public boolean equals(final Object o) { + long stamp = rwLock.tryOptimisticRead(); + boolean isEqual = super.equals(o); + if (!rwLock.validate(stamp)) { + // Fallback to read lock + stamp = rwLock.readLock(); + try { + isEqual = super.equals(o); + } finally { + rwLock.unlockRead(stamp); + } + } + return isEqual; + } + + public int hashCode() { + long stamp = rwLock.tryOptimisticRead(); + int hashCode = super.hashCode(); + if (!rwLock.validate(stamp)) { + // Fallback to read lock + stamp = rwLock.readLock(); + try { + hashCode = super.hashCode(); + } finally { + rwLock.unlockRead(stamp); + } + } + return hashCode; + } +} diff --git a/pulsar-common/src/main/proto/PulsarApi.proto b/pulsar-common/src/main/proto/PulsarApi.proto index afe193eeb7e9d..f56df6ae9d103 100644 --- a/pulsar-common/src/main/proto/PulsarApi.proto +++ b/pulsar-common/src/main/proto/PulsarApi.proto @@ -300,6 +300,7 @@ message FeatureFlags { optional bool supports_broker_entry_metadata = 2 [default = false]; optional bool supports_partial_producer = 3 [default = false]; optional bool supports_topic_watchers = 4 [default = false]; + optional bool supports_get_partitioned_metadata_without_auto_creation = 5 [default = false]; } message CommandConnected { @@ -413,6 +414,7 @@ message CommandPartitionedTopicMetadata { // to the proxy. optional string original_auth_data = 4; optional string original_auth_method = 5; + optional bool metadata_auto_creation_enabled = 6 [default = true]; } message CommandPartitionedTopicMetadataResponse { diff --git a/pulsar-common/src/test/java/org/apache/pulsar/common/naming/TopicNameTest.java b/pulsar-common/src/test/java/org/apache/pulsar/common/naming/TopicNameTest.java index 835045f9167dd..485bea3f1addb 100644 --- a/pulsar-common/src/test/java/org/apache/pulsar/common/naming/TopicNameTest.java +++ b/pulsar-common/src/test/java/org/apache/pulsar/common/naming/TopicNameTest.java @@ -267,6 +267,12 @@ public void testFromPersistenceNamingEncoding() { } catch (IllegalArgumentException e) { // Exception is expected. } + + // case5: local name with special characters e.g. a:b:c + String topicName = "persistent://tenant/namespace/a:b:c"; + String persistentNamingEncoding = "tenant/namespace/persistent/a%3Ab%3Ac"; + assertEquals(TopicName.get(topicName).getPersistenceNamingEncoding(), persistentNamingEncoding); + assertEquals(TopicName.fromPersistenceNamingEncoding(persistentNamingEncoding), topicName); } diff --git a/pulsar-functions/instance/src/main/java/org/apache/pulsar/functions/instance/ContextImpl.java b/pulsar-functions/instance/src/main/java/org/apache/pulsar/functions/instance/ContextImpl.java index d03f57e97205c..075e8bc9a764c 100644 --- a/pulsar-functions/instance/src/main/java/org/apache/pulsar/functions/instance/ContextImpl.java +++ b/pulsar-functions/instance/src/main/java/org/apache/pulsar/functions/instance/ContextImpl.java @@ -27,6 +27,7 @@ import java.nio.ByteBuffer; import java.util.Arrays; import java.util.Collection; +import java.util.Collections; import java.util.HashMap; import java.util.LinkedList; import java.util.List; @@ -40,14 +41,11 @@ import lombok.ToString; import org.apache.commons.lang3.StringUtils; import org.apache.pulsar.client.admin.PulsarAdmin; -import org.apache.pulsar.client.api.BatcherBuilder; import org.apache.pulsar.client.api.ClientBuilder; import org.apache.pulsar.client.api.CompressionType; import org.apache.pulsar.client.api.Consumer; import org.apache.pulsar.client.api.ConsumerBuilder; -import org.apache.pulsar.client.api.HashingScheme; import org.apache.pulsar.client.api.MessageId; -import org.apache.pulsar.client.api.MessageRoutingMode; import org.apache.pulsar.client.api.Producer; import org.apache.pulsar.client.api.PulsarClient; import org.apache.pulsar.client.api.PulsarClientException; @@ -55,7 +53,7 @@ import org.apache.pulsar.client.api.SubscriptionType; import org.apache.pulsar.client.api.TypedMessageBuilder; import org.apache.pulsar.client.impl.MultiTopicsConsumerImpl; -import org.apache.pulsar.client.impl.ProducerBuilderImpl; +import org.apache.pulsar.common.functions.ProducerConfig; import org.apache.pulsar.common.io.SinkConfig; import org.apache.pulsar.common.io.SourceConfig; import org.apache.pulsar.common.naming.TopicName; @@ -77,6 +75,7 @@ import org.apache.pulsar.functions.source.PulsarFunctionRecord; import org.apache.pulsar.functions.source.TopicSchema; import org.apache.pulsar.functions.utils.FunctionCommon; +import org.apache.pulsar.functions.utils.FunctionConfigUtils; import org.apache.pulsar.functions.utils.SinkConfigUtils; import org.apache.pulsar.functions.utils.SourceConfigUtils; import org.apache.pulsar.io.core.SinkContext; @@ -88,6 +87,8 @@ */ @ToString(exclude = {"pulsarAdmin"}) class ContextImpl implements Context, SinkContext, SourceContext, AutoCloseable { + private final ProducerBuilderFactory producerBuilderFactory; + private final Map producerProperties; private InstanceConfig config; private Logger logger; @@ -99,7 +100,6 @@ class ContextImpl implements Context, SinkContext, SourceContext, AutoCloseable private final PulsarAdmin pulsarAdmin; private Map> publishProducers; private ThreadLocal>> tlPublishProducers; - private ProducerBuilderImpl producerBuilder; private final TopicSchema topicSchema; @@ -151,27 +151,27 @@ public ContextImpl(InstanceConfig config, Logger logger, PulsarClient client, this.topicSchema = new TopicSchema(client, Thread.currentThread().getContextClassLoader()); this.statsManager = statsManager; - this.producerBuilder = (ProducerBuilderImpl) client.newProducer().blockIfQueueFull(true).enableBatching(true) - .batchingMaxPublishDelay(1, TimeUnit.MILLISECONDS); boolean useThreadLocalProducers = false; + Function.ProducerSpec producerSpec = config.getFunctionDetails().getSink().getProducerSpec(); + ProducerConfig producerConfig = null; if (producerSpec != null) { - if (producerSpec.getMaxPendingMessages() != 0) { - this.producerBuilder.maxPendingMessages(producerSpec.getMaxPendingMessages()); - } - if (producerSpec.getMaxPendingMessagesAcrossPartitions() != 0) { - this.producerBuilder - .maxPendingMessagesAcrossPartitions(producerSpec.getMaxPendingMessagesAcrossPartitions()); - } - if (producerSpec.getBatchBuilder() != null) { - if (producerSpec.getBatchBuilder().equals("KEY_BASED")) { - this.producerBuilder.batcherBuilder(BatcherBuilder.KEY_BASED); - } else { - this.producerBuilder.batcherBuilder(BatcherBuilder.DEFAULT); - } - } + producerConfig = FunctionConfigUtils.convertProducerSpecToProducerConfig(producerSpec); useThreadLocalProducers = producerSpec.getUseThreadLocalProducers(); } + producerBuilderFactory = new ProducerBuilderFactory(client, producerConfig, + Thread.currentThread().getContextClassLoader(), + // This is for backwards compatibility. The PR https://github.com/apache/pulsar/pull/19470 removed + // the default and made it configurable for the producers created in PulsarSink, but not in ContextImpl. + // This is to keep the default unchanged for the producers created in ContextImpl. + producerBuilder -> producerBuilder.compressionType(CompressionType.LZ4)); + producerProperties = Collections.unmodifiableMap(InstanceUtils.getProperties(componentType, + FunctionCommon.getFullyQualifiedName( + this.config.getFunctionDetails().getTenant(), + this.config.getFunctionDetails().getNamespace(), + this.config.getFunctionDetails().getName()), + this.config.getInstanceId())); + if (useThreadLocalProducers) { tlPublishProducers = new ThreadLocal<>(); } else { @@ -548,26 +548,9 @@ private Producer getProducer(String topicName, Schema schema) throws P } if (producer == null) { - - Producer newProducer = ((ProducerBuilderImpl) producerBuilder.clone()) - .schema(schema) - .blockIfQueueFull(true) - .enableBatching(true) - .batchingMaxPublishDelay(10, TimeUnit.MILLISECONDS) - .compressionType(CompressionType.LZ4) - .hashingScheme(HashingScheme.Murmur3_32Hash) // - .messageRoutingMode(MessageRoutingMode.CustomPartition) - .messageRouter(FunctionResultRouter.of()) - // set send timeout to be infinity to prevent potential deadlock with consumer - // that might happen when consumer is blocked due to unacked messages - .sendTimeout(0, TimeUnit.SECONDS) - .topic(topicName) - .properties(InstanceUtils.getProperties(componentType, - FunctionCommon.getFullyQualifiedName( - this.config.getFunctionDetails().getTenant(), - this.config.getFunctionDetails().getNamespace(), - this.config.getFunctionDetails().getName()), - this.config.getInstanceId())) + Producer newProducer = producerBuilderFactory + .createProducerBuilder(topicName, schema, null) + .properties(producerProperties) .create(); if (tlPublishProducers != null) { diff --git a/pulsar-functions/instance/src/main/java/org/apache/pulsar/functions/instance/ProducerBuilderFactory.java b/pulsar-functions/instance/src/main/java/org/apache/pulsar/functions/instance/ProducerBuilderFactory.java new file mode 100644 index 0000000000000..b08f7f3f2cb0f --- /dev/null +++ b/pulsar-functions/instance/src/main/java/org/apache/pulsar/functions/instance/ProducerBuilderFactory.java @@ -0,0 +1,159 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. The ASF licenses this file + * to you under the Apache License, Version 2.0 (the + * "License"); you may not use this file except in compliance + * with the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, + * software distributed under the License is distributed on an + * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY + * KIND, either express or implied. See the License for the + * specific language governing permissions and limitations + * under the License. + */ +package org.apache.pulsar.functions.instance; + +import static org.apache.commons.lang.StringUtils.isEmpty; +import com.google.common.annotations.VisibleForTesting; +import java.security.Security; +import java.util.concurrent.TimeUnit; +import java.util.function.Consumer; +import lombok.Builder; +import lombok.Data; +import lombok.extern.slf4j.Slf4j; +import org.apache.pulsar.client.api.BatcherBuilder; +import org.apache.pulsar.client.api.CompressionType; +import org.apache.pulsar.client.api.CryptoKeyReader; +import org.apache.pulsar.client.api.HashingScheme; +import org.apache.pulsar.client.api.MessageRoutingMode; +import org.apache.pulsar.client.api.ProducerBuilder; +import org.apache.pulsar.client.api.ProducerCryptoFailureAction; +import org.apache.pulsar.client.api.PulsarClient; +import org.apache.pulsar.client.api.Schema; +import org.apache.pulsar.common.functions.CryptoConfig; +import org.apache.pulsar.common.functions.ProducerConfig; +import org.apache.pulsar.functions.utils.CryptoUtils; +import org.bouncycastle.jce.provider.BouncyCastleProvider; + +/** + * This class is responsible for creating ProducerBuilders with the appropriate configurations to + * match the ProducerConfig provided. Producers are created in 2 locations in Pulsar Functions and Connectors + * and this class is used to unify the configuration of the producers without duplicating code. + */ +@Slf4j +public class ProducerBuilderFactory { + + private final PulsarClient client; + private final ProducerConfig producerConfig; + private final Consumer> defaultConfigurer; + private final Crypto crypto; + + public ProducerBuilderFactory(PulsarClient client, ProducerConfig producerConfig, ClassLoader functionClassLoader, + Consumer> defaultConfigurer) { + this.client = client; + this.producerConfig = producerConfig; + this.defaultConfigurer = defaultConfigurer; + try { + this.crypto = initializeCrypto(functionClassLoader); + } catch (ClassNotFoundException e) { + throw new RuntimeException("Unable to initialize crypto config " + producerConfig.getCryptoConfig(), e); + } + if (crypto == null) { + log.info("crypto key reader is not provided, not enabling end to end encryption"); + } + } + + public ProducerBuilder createProducerBuilder(String topic, Schema schema, String producerName) { + ProducerBuilder builder = client.newProducer(schema); + if (defaultConfigurer != null) { + defaultConfigurer.accept(builder); + } + builder.blockIfQueueFull(true) + .enableBatching(true) + .batchingMaxPublishDelay(10, TimeUnit.MILLISECONDS) + .hashingScheme(HashingScheme.Murmur3_32Hash) // + .messageRoutingMode(MessageRoutingMode.CustomPartition) + .messageRouter(FunctionResultRouter.of()) + // set send timeout to be infinity to prevent potential deadlock with consumer + // that might happen when consumer is blocked due to unacked messages + .sendTimeout(0, TimeUnit.SECONDS) + .topic(topic); + if (producerName != null) { + builder.producerName(producerName); + } + if (producerConfig != null) { + if (producerConfig.getCompressionType() != null) { + builder.compressionType(producerConfig.getCompressionType()); + } else { + // TODO: address this inconsistency. + // PR https://github.com/apache/pulsar/pull/19470 removed the default compression type of LZ4 + // from the top level. This default is only used if producer config is provided. + builder.compressionType(CompressionType.LZ4); + } + if (producerConfig.getMaxPendingMessages() != null && producerConfig.getMaxPendingMessages() != 0) { + builder.maxPendingMessages(producerConfig.getMaxPendingMessages()); + } + if (producerConfig.getMaxPendingMessagesAcrossPartitions() != null + && producerConfig.getMaxPendingMessagesAcrossPartitions() != 0) { + builder.maxPendingMessagesAcrossPartitions(producerConfig.getMaxPendingMessagesAcrossPartitions()); + } + if (producerConfig.getCryptoConfig() != null) { + builder.cryptoKeyReader(crypto.keyReader); + builder.cryptoFailureAction(crypto.failureAction); + for (String encryptionKeyName : crypto.getEncryptionKeys()) { + builder.addEncryptionKey(encryptionKeyName); + } + } + if (producerConfig.getBatchBuilder() != null) { + if (producerConfig.getBatchBuilder().equals("KEY_BASED")) { + builder.batcherBuilder(BatcherBuilder.KEY_BASED); + } else { + builder.batcherBuilder(BatcherBuilder.DEFAULT); + } + } + } + return builder; + } + + + @SuppressWarnings("unchecked") + @VisibleForTesting + Crypto initializeCrypto(ClassLoader functionClassLoader) throws ClassNotFoundException { + if (producerConfig == null + || producerConfig.getCryptoConfig() == null + || isEmpty(producerConfig.getCryptoConfig().getCryptoKeyReaderClassName())) { + return null; + } + + CryptoConfig cryptoConfig = producerConfig.getCryptoConfig(); + + // add provider only if it's not in the JVM + if (Security.getProvider(BouncyCastleProvider.PROVIDER_NAME) == null) { + Security.addProvider(new BouncyCastleProvider()); + } + + final String[] encryptionKeys = cryptoConfig.getEncryptionKeys(); + Crypto.CryptoBuilder bldr = Crypto.builder() + .failureAction(cryptoConfig.getProducerCryptoFailureAction()) + .encryptionKeys(encryptionKeys); + + bldr.keyReader(CryptoUtils.getCryptoKeyReaderInstance( + cryptoConfig.getCryptoKeyReaderClassName(), cryptoConfig.getCryptoKeyReaderConfig(), + functionClassLoader)); + + return bldr.build(); + } + + @Data + @Builder + private static class Crypto { + private CryptoKeyReader keyReader; + private ProducerCryptoFailureAction failureAction; + private String[] encryptionKeys; + } +} diff --git a/pulsar-functions/instance/src/main/java/org/apache/pulsar/functions/sink/PulsarSink.java b/pulsar-functions/instance/src/main/java/org/apache/pulsar/functions/sink/PulsarSink.java index 97a0ad0a2ce17..18e55e8e84de1 100644 --- a/pulsar-functions/instance/src/main/java/org/apache/pulsar/functions/sink/PulsarSink.java +++ b/pulsar-functions/instance/src/main/java/org/apache/pulsar/functions/sink/PulsarSink.java @@ -18,10 +18,8 @@ */ package org.apache.pulsar.functions.sink; -import static org.apache.commons.lang.StringUtils.isEmpty; import com.google.common.annotations.VisibleForTesting; import java.nio.charset.StandardCharsets; -import java.security.Security; import java.util.ArrayList; import java.util.Base64; import java.util.List; @@ -29,21 +27,12 @@ import java.util.Optional; import java.util.concurrent.CompletableFuture; import java.util.concurrent.ConcurrentHashMap; -import java.util.concurrent.TimeUnit; import java.util.function.Function; -import lombok.Builder; -import lombok.Data; import lombok.extern.slf4j.Slf4j; import org.apache.commons.lang3.StringUtils; -import org.apache.pulsar.client.api.BatcherBuilder; -import org.apache.pulsar.client.api.CompressionType; -import org.apache.pulsar.client.api.CryptoKeyReader; -import org.apache.pulsar.client.api.HashingScheme; import org.apache.pulsar.client.api.MessageId; -import org.apache.pulsar.client.api.MessageRoutingMode; import org.apache.pulsar.client.api.Producer; import org.apache.pulsar.client.api.ProducerBuilder; -import org.apache.pulsar.client.api.ProducerCryptoFailureAction; import org.apache.pulsar.client.api.PulsarClient; import org.apache.pulsar.client.api.PulsarClientException; import org.apache.pulsar.client.api.Schema; @@ -52,22 +41,18 @@ import org.apache.pulsar.client.api.schema.KeyValueSchema; import org.apache.pulsar.client.impl.schema.AutoConsumeSchema; import org.apache.pulsar.common.functions.ConsumerConfig; -import org.apache.pulsar.common.functions.CryptoConfig; import org.apache.pulsar.common.functions.FunctionConfig; -import org.apache.pulsar.common.functions.ProducerConfig; import org.apache.pulsar.common.schema.KeyValueEncodingType; import org.apache.pulsar.common.schema.SchemaType; import org.apache.pulsar.common.util.Reflections; import org.apache.pulsar.functions.api.Record; import org.apache.pulsar.functions.instance.AbstractSinkRecord; -import org.apache.pulsar.functions.instance.FunctionResultRouter; +import org.apache.pulsar.functions.instance.ProducerBuilderFactory; import org.apache.pulsar.functions.instance.stats.ComponentStatsManager; import org.apache.pulsar.functions.source.PulsarRecord; import org.apache.pulsar.functions.source.TopicSchema; -import org.apache.pulsar.functions.utils.CryptoUtils; import org.apache.pulsar.io.core.Sink; import org.apache.pulsar.io.core.SinkContext; -import org.bouncycastle.jce.provider.BouncyCastleProvider; @Slf4j public class PulsarSink implements Sink { @@ -82,6 +67,8 @@ public class PulsarSink implements Sink { PulsarSinkProcessor pulsarSinkProcessor; private final TopicSchema topicSchema; + private Schema schema; + private ProducerBuilderFactory producerBuilderFactory; private interface PulsarSinkProcessor { @@ -94,60 +81,6 @@ private interface PulsarSinkProcessor { abstract class PulsarSinkProcessorBase implements PulsarSinkProcessor { protected Map> publishProducers = new ConcurrentHashMap<>(); - protected Schema schema; - protected Crypto crypto; - - protected PulsarSinkProcessorBase(Schema schema, Crypto crypto) { - this.schema = schema; - this.crypto = crypto; - } - - public Producer createProducer(PulsarClient client, String topic, String producerName, Schema schema) - throws PulsarClientException { - ProducerBuilder builder = client.newProducer(schema) - .blockIfQueueFull(true) - .enableBatching(true) - .batchingMaxPublishDelay(10, TimeUnit.MILLISECONDS) - .hashingScheme(HashingScheme.Murmur3_32Hash) // - .messageRoutingMode(MessageRoutingMode.CustomPartition) - .messageRouter(FunctionResultRouter.of()) - // set send timeout to be infinity to prevent potential deadlock with consumer - // that might happen when consumer is blocked due to unacked messages - .sendTimeout(0, TimeUnit.SECONDS) - .topic(topic); - if (producerName != null) { - builder.producerName(producerName); - } - if (pulsarSinkConfig.getProducerConfig() != null) { - ProducerConfig producerConfig = pulsarSinkConfig.getProducerConfig(); - if (producerConfig.getCompressionType() != null) { - builder.compressionType(producerConfig.getCompressionType()); - } else { - builder.compressionType(CompressionType.LZ4); - } - if (producerConfig.getMaxPendingMessages() != 0) { - builder.maxPendingMessages(producerConfig.getMaxPendingMessages()); - } - if (producerConfig.getMaxPendingMessagesAcrossPartitions() != 0) { - builder.maxPendingMessagesAcrossPartitions(producerConfig.getMaxPendingMessagesAcrossPartitions()); - } - if (producerConfig.getCryptoConfig() != null) { - builder.cryptoKeyReader(crypto.keyReader); - builder.cryptoFailureAction(crypto.failureAction); - for (String encryptionKeyName : crypto.getEncryptionKeys()) { - builder.addEncryptionKey(encryptionKeyName); - } - } - if (producerConfig.getBatchBuilder() != null) { - if (producerConfig.getBatchBuilder().equals("KEY_BASED")) { - builder.batcherBuilder(BatcherBuilder.KEY_BASED); - } else { - builder.batcherBuilder(BatcherBuilder.DEFAULT); - } - } - } - return builder.properties(properties).create(); - } protected Producer getProducer(String destinationTopic, Schema schema) { return getProducer(destinationTopic, null, destinationTopic, schema); @@ -159,10 +92,9 @@ protected Producer getProducer(String producerId, String producerName, String log.info("Initializing producer {} on topic {} with schema {}", producerName, topicName, schema); Producer producer = createProducer( - client, topicName, - producerName, - schema != null ? schema : this.schema); + schema, producerName + ); log.info("Initialized producer {} on topic {} with schema {}: {} -> {}", producerName, topicName, schema, producerId, producer); return producer; @@ -218,13 +150,12 @@ public Function getPublishErrorHandler(AbstractSinkRecord re @VisibleForTesting class PulsarSinkAtMostOnceProcessor extends PulsarSinkProcessorBase { - public PulsarSinkAtMostOnceProcessor(Schema schema, Crypto crypto) { - super(schema, crypto); + public PulsarSinkAtMostOnceProcessor() { if (!(schema instanceof AutoConsumeSchema)) { // initialize default topic try { publishProducers.put(pulsarSinkConfig.getTopic(), - createProducer(client, pulsarSinkConfig.getTopic(), null, schema)); + createProducer(pulsarSinkConfig.getTopic(), schema, null)); } catch (PulsarClientException e) { log.error("Failed to create Producer while doing user publish", e); throw new RuntimeException(e); @@ -270,10 +201,6 @@ public void sendOutputMessage(TypedMessageBuilder msg, AbstractSinkRecord @VisibleForTesting class PulsarSinkAtLeastOnceProcessor extends PulsarSinkAtMostOnceProcessor { - public PulsarSinkAtLeastOnceProcessor(Schema schema, Crypto crypto) { - super(schema, crypto); - } - @Override public void sendOutputMessage(TypedMessageBuilder msg, AbstractSinkRecord record) { msg.sendAsync() @@ -284,11 +211,6 @@ public void sendOutputMessage(TypedMessageBuilder msg, AbstractSinkRecord @VisibleForTesting class PulsarSinkManualProcessor extends PulsarSinkAtMostOnceProcessor { - - public PulsarSinkManualProcessor(Schema schema, Crypto crypto) { - super(schema, crypto); - } - @Override public void sendOutputMessage(TypedMessageBuilder msg, AbstractSinkRecord record) { super.sendOutputMessage(msg, record); @@ -297,11 +219,6 @@ public void sendOutputMessage(TypedMessageBuilder msg, AbstractSinkRecord @VisibleForTesting class PulsarSinkEffectivelyOnceProcessor extends PulsarSinkProcessorBase { - - public PulsarSinkEffectivelyOnceProcessor(Schema schema, Crypto crypto) { - super(schema, crypto); - } - @Override public TypedMessageBuilder newMessage(AbstractSinkRecord record) { if (!record.getPartitionId().isPresent()) { @@ -359,30 +276,27 @@ public PulsarSink(PulsarClient client, PulsarSinkConfig pulsarSinkConfig, Map config, SinkContext sinkContext) throws Exception { log.info("Opening pulsar sink with config: {}", pulsarSinkConfig); - Schema schema = initializeSchema(); + schema = initializeSchema(); if (schema == null) { log.info("Since output type is null, not creating any real sink"); return; } - - Crypto crypto = initializeCrypto(); - if (crypto == null) { - log.info("crypto key reader is not provided, not enabling end to end encryption"); - } + producerBuilderFactory = + new ProducerBuilderFactory(client, pulsarSinkConfig.getProducerConfig(), functionClassLoader, null); FunctionConfig.ProcessingGuarantees processingGuarantees = this.pulsarSinkConfig.getProcessingGuarantees(); switch (processingGuarantees) { case ATMOST_ONCE: - this.pulsarSinkProcessor = new PulsarSinkAtMostOnceProcessor(schema, crypto); + this.pulsarSinkProcessor = new PulsarSinkAtMostOnceProcessor(); break; case ATLEAST_ONCE: - this.pulsarSinkProcessor = new PulsarSinkAtLeastOnceProcessor(schema, crypto); + this.pulsarSinkProcessor = new PulsarSinkAtLeastOnceProcessor(); break; case EFFECTIVELY_ONCE: - this.pulsarSinkProcessor = new PulsarSinkEffectivelyOnceProcessor(schema, crypto); + this.pulsarSinkProcessor = new PulsarSinkEffectivelyOnceProcessor(); break; case MANUAL: - this.pulsarSinkProcessor = new PulsarSinkManualProcessor(schema, crypto); + this.pulsarSinkProcessor = new PulsarSinkManualProcessor(); break; } } @@ -427,6 +341,16 @@ public void close() throws Exception { } } + Producer createProducer(String topic, Schema schema, String producerName) + throws PulsarClientException { + ProducerBuilder builder = + producerBuilderFactory.createProducerBuilder(topic, schema != null ? schema : this.schema, + producerName); + return builder + .properties(properties) + .create(); + } + @SuppressWarnings("unchecked") @VisibleForTesting Schema initializeSchema() throws ClassNotFoundException { @@ -461,39 +385,5 @@ Schema initializeSchema() throws ClassNotFoundException { } } - @SuppressWarnings("unchecked") - @VisibleForTesting - Crypto initializeCrypto() throws ClassNotFoundException { - if (pulsarSinkConfig.getProducerConfig() == null - || pulsarSinkConfig.getProducerConfig().getCryptoConfig() == null - || isEmpty(pulsarSinkConfig.getProducerConfig().getCryptoConfig().getCryptoKeyReaderClassName())) { - return null; - } - - CryptoConfig cryptoConfig = pulsarSinkConfig.getProducerConfig().getCryptoConfig(); - - // add provider only if it's not in the JVM - if (Security.getProvider(BouncyCastleProvider.PROVIDER_NAME) == null) { - Security.addProvider(new BouncyCastleProvider()); - } - - final String[] encryptionKeys = cryptoConfig.getEncryptionKeys(); - Crypto.CryptoBuilder bldr = Crypto.builder() - .failureAction(cryptoConfig.getProducerCryptoFailureAction()) - .encryptionKeys(encryptionKeys); - bldr.keyReader(CryptoUtils.getCryptoKeyReaderInstance( - cryptoConfig.getCryptoKeyReaderClassName(), cryptoConfig.getCryptoKeyReaderConfig(), - functionClassLoader)); - - return bldr.build(); - } - - @Data - @Builder - private static class Crypto { - private CryptoKeyReader keyReader; - private ProducerCryptoFailureAction failureAction; - private String[] encryptionKeys; - } } diff --git a/pulsar-functions/instance/src/main/resources/findbugsExclude.xml b/pulsar-functions/instance/src/main/resources/findbugsExclude.xml index 7fe247d2ab20a..40e3e91112328 100644 --- a/pulsar-functions/instance/src/main/resources/findbugsExclude.xml +++ b/pulsar-functions/instance/src/main/resources/findbugsExclude.xml @@ -49,7 +49,12 @@ - + + + + + + diff --git a/pulsar-functions/instance/src/test/java/org/apache/pulsar/functions/instance/ContextImplTest.java b/pulsar-functions/instance/src/test/java/org/apache/pulsar/functions/instance/ContextImplTest.java index 90f7df37fa196..108d8e4b66663 100644 --- a/pulsar-functions/instance/src/test/java/org/apache/pulsar/functions/instance/ContextImplTest.java +++ b/pulsar-functions/instance/src/test/java/org/apache/pulsar/functions/instance/ContextImplTest.java @@ -102,7 +102,9 @@ public void setup() throws PulsarClientException { client = mock(PulsarClientImpl.class); ConnectionPool connectionPool = mock(ConnectionPool.class); when(client.getCnxPool()).thenReturn(connectionPool); - when(client.newProducer()).thenReturn(new ProducerBuilderImpl(client, Schema.BYTES)); + when(client.newProducer()).thenAnswer(invocation -> new ProducerBuilderImpl(client, Schema.BYTES)); + when(client.newProducer(any())).thenAnswer( + invocation -> new ProducerBuilderImpl(client, invocation.getArgument(0))); when(client.createProducerAsync(any(ProducerConfigurationData.class), any(), any())) .thenReturn(CompletableFuture.completedFuture(producer)); when(client.getSchema(anyString())).thenReturn(CompletableFuture.completedFuture(Optional.empty())); diff --git a/pulsar-functions/instance/src/test/java/org/apache/pulsar/functions/instance/ProducerBuilderFactoryTest.java b/pulsar-functions/instance/src/test/java/org/apache/pulsar/functions/instance/ProducerBuilderFactoryTest.java new file mode 100644 index 0000000000000..42940f7e2dae3 --- /dev/null +++ b/pulsar-functions/instance/src/test/java/org/apache/pulsar/functions/instance/ProducerBuilderFactoryTest.java @@ -0,0 +1,178 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. The ASF licenses this file + * to you under the Apache License, Version 2.0 (the + * "License"); you may not use this file except in compliance + * with the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, + * software distributed under the License is distributed on an + * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY + * KIND, either express or implied. See the License for the + * specific language governing permissions and limitations + * under the License. + */ +package org.apache.pulsar.functions.instance; + +import static org.mockito.ArgumentMatchers.any; +import static org.mockito.ArgumentMatchers.anyBoolean; +import static org.mockito.ArgumentMatchers.anyInt; +import static org.mockito.ArgumentMatchers.anyLong; +import static org.mockito.ArgumentMatchers.anyString; +import static org.mockito.Mockito.doReturn; +import static org.mockito.Mockito.mock; +import static org.mockito.Mockito.verify; +import static org.mockito.Mockito.verifyNoMoreInteractions; +import static org.testng.Assert.assertEquals; +import static org.testng.Assert.assertNotNull; +import java.util.Map; +import java.util.concurrent.TimeUnit; +import org.apache.pulsar.client.api.BatcherBuilder; +import org.apache.pulsar.client.api.CompressionType; +import org.apache.pulsar.client.api.CryptoKeyReader; +import org.apache.pulsar.client.api.EncryptionKeyInfo; +import org.apache.pulsar.client.api.HashingScheme; +import org.apache.pulsar.client.api.MessageRoutingMode; +import org.apache.pulsar.client.api.ProducerBuilder; +import org.apache.pulsar.client.api.ProducerCryptoFailureAction; +import org.apache.pulsar.client.api.PulsarClient; +import org.apache.pulsar.client.api.Schema; +import org.apache.pulsar.common.functions.CryptoConfig; +import org.apache.pulsar.common.functions.ProducerConfig; +import org.mockito.internal.util.MockUtil; +import org.testng.annotations.AfterMethod; +import org.testng.annotations.BeforeMethod; +import org.testng.annotations.Test; + +public class ProducerBuilderFactoryTest { + private PulsarClient pulsarClient; + private ProducerBuilder producerBuilder; + + @BeforeMethod + public void setup() { + pulsarClient = mock(PulsarClient.class); + + producerBuilder = mock(ProducerBuilder.class); + doReturn(producerBuilder).when(producerBuilder).blockIfQueueFull(anyBoolean()); + doReturn(producerBuilder).when(producerBuilder).enableBatching(anyBoolean()); + doReturn(producerBuilder).when(producerBuilder).batchingMaxPublishDelay(anyLong(), any()); + doReturn(producerBuilder).when(producerBuilder).compressionType(any()); + doReturn(producerBuilder).when(producerBuilder).hashingScheme(any()); + doReturn(producerBuilder).when(producerBuilder).messageRoutingMode(any()); + doReturn(producerBuilder).when(producerBuilder).messageRouter(any()); + doReturn(producerBuilder).when(producerBuilder).topic(anyString()); + doReturn(producerBuilder).when(producerBuilder).producerName(anyString()); + doReturn(producerBuilder).when(producerBuilder).property(anyString(), anyString()); + doReturn(producerBuilder).when(producerBuilder).properties(any()); + doReturn(producerBuilder).when(producerBuilder).sendTimeout(anyInt(), any()); + + doReturn(producerBuilder).when(pulsarClient).newProducer(); + doReturn(producerBuilder).when(pulsarClient).newProducer(any()); + } + + @AfterMethod + public void tearDown() { + MockUtil.resetMock(pulsarClient); + pulsarClient = null; + MockUtil.resetMock(producerBuilder); + producerBuilder = null; + TestCryptoKeyReader.LAST_INSTANCE = null; + } + + @Test + public void testCreateProducerBuilder() { + ProducerBuilderFactory builderFactory = new ProducerBuilderFactory(pulsarClient, null, null, null); + builderFactory.createProducerBuilder("topic", Schema.STRING, "producerName"); + verifyCommon(); + verifyNoMoreInteractions(producerBuilder); + } + + private void verifyCommon() { + verify(pulsarClient).newProducer(Schema.STRING); + verify(producerBuilder).blockIfQueueFull(true); + verify(producerBuilder).enableBatching(true); + verify(producerBuilder).batchingMaxPublishDelay(10, TimeUnit.MILLISECONDS); + verify(producerBuilder).hashingScheme(HashingScheme.Murmur3_32Hash); + verify(producerBuilder).messageRoutingMode(MessageRoutingMode.CustomPartition); + verify(producerBuilder).messageRouter(FunctionResultRouter.of()); + verify(producerBuilder).sendTimeout(0, TimeUnit.SECONDS); + verify(producerBuilder).topic("topic"); + verify(producerBuilder).producerName("producerName"); + } + + @Test + public void testCreateProducerBuilderWithDefaultConfigurer() { + ProducerBuilderFactory builderFactory = new ProducerBuilderFactory(pulsarClient, null, null, + builder -> builder.property("key", "value")); + builderFactory.createProducerBuilder("topic", Schema.STRING, "producerName"); + verifyCommon(); + verify(producerBuilder).property("key", "value"); + verifyNoMoreInteractions(producerBuilder); + } + + @Test + public void testCreateProducerBuilderWithSimpleProducerConfig() { + ProducerConfig producerConfig = new ProducerConfig(); + producerConfig.setBatchBuilder("KEY_BASED"); + ProducerBuilderFactory builderFactory = new ProducerBuilderFactory(pulsarClient, producerConfig, null, null); + builderFactory.createProducerBuilder("topic", Schema.STRING, "producerName"); + verifyCommon(); + verify(producerBuilder).compressionType(CompressionType.LZ4); + verify(producerBuilder).batcherBuilder(BatcherBuilder.KEY_BASED); + verifyNoMoreInteractions(producerBuilder); + } + + @Test + public void testCreateProducerBuilderWithAdvancedProducerConfig() { + ProducerConfig producerConfig = new ProducerConfig(); + producerConfig.setBatchBuilder("KEY_BASED"); + producerConfig.setCompressionType(CompressionType.SNAPPY); + producerConfig.setMaxPendingMessages(5000); + producerConfig.setMaxPendingMessagesAcrossPartitions(50000); + CryptoConfig cryptoConfig = new CryptoConfig(); + cryptoConfig.setProducerCryptoFailureAction(ProducerCryptoFailureAction.FAIL); + cryptoConfig.setEncryptionKeys(new String[]{"key1", "key2"}); + cryptoConfig.setCryptoKeyReaderConfig(Map.of("key", "value")); + cryptoConfig.setCryptoKeyReaderClassName(TestCryptoKeyReader.class.getName()); + producerConfig.setCryptoConfig(cryptoConfig); + ProducerBuilderFactory builderFactory = new ProducerBuilderFactory(pulsarClient, producerConfig, null, null); + builderFactory.createProducerBuilder("topic", Schema.STRING, "producerName"); + verifyCommon(); + verify(producerBuilder).compressionType(CompressionType.SNAPPY); + verify(producerBuilder).batcherBuilder(BatcherBuilder.KEY_BASED); + verify(producerBuilder).maxPendingMessages(5000); + verify(producerBuilder).maxPendingMessagesAcrossPartitions(50000); + TestCryptoKeyReader lastInstance = TestCryptoKeyReader.LAST_INSTANCE; + assertNotNull(lastInstance); + assertEquals(lastInstance.configs, cryptoConfig.getCryptoKeyReaderConfig()); + verify(producerBuilder).cryptoKeyReader(lastInstance); + verify(producerBuilder).cryptoFailureAction(ProducerCryptoFailureAction.FAIL); + verify(producerBuilder).addEncryptionKey("key1"); + verify(producerBuilder).addEncryptionKey("key2"); + verifyNoMoreInteractions(producerBuilder); + } + + public static class TestCryptoKeyReader implements CryptoKeyReader { + static TestCryptoKeyReader LAST_INSTANCE; + Map configs; + public TestCryptoKeyReader(Map configs) { + this.configs = configs; + assert LAST_INSTANCE == null; + LAST_INSTANCE = this; + } + + @Override + public EncryptionKeyInfo getPublicKey(String keyName, Map metadata) { + throw new UnsupportedOperationException(); + } + + @Override + public EncryptionKeyInfo getPrivateKey(String keyName, Map metadata) { + throw new UnsupportedOperationException(); + } + } +} \ No newline at end of file diff --git a/pulsar-functions/runtime/src/main/java/org/apache/pulsar/functions/runtime/RuntimeUtils.java b/pulsar-functions/runtime/src/main/java/org/apache/pulsar/functions/runtime/RuntimeUtils.java index 0214b18fb2326..78347948688dd 100644 --- a/pulsar-functions/runtime/src/main/java/org/apache/pulsar/functions/runtime/RuntimeUtils.java +++ b/pulsar-functions/runtime/src/main/java/org/apache/pulsar/functions/runtime/RuntimeUtils.java @@ -361,12 +361,26 @@ public static List getCmd(InstanceConfig instanceConfig, instanceConfig.getFunctionDetails().getName(), shardId)); + // Needed for optimized Netty direct byte buffer support args.add("-Dio.netty.tryReflectionSetAccessible=true"); + // Handle possible shaded Netty versions + args.add("-Dorg.apache.pulsar.shade.io.netty.tryReflectionSetAccessible=true"); + args.add("-Dio.grpc.netty.shaded.io.netty.tryReflectionSetAccessible=true"); + + if (SystemUtils.isJavaVersionAtLeast(JavaVersion.JAVA_11)) { + // Needed for optimized Netty direct byte buffer support + args.add("--add-opens"); + args.add("java.base/java.nio=ALL-UNNAMED"); + args.add("--add-opens"); + args.add("java.base/jdk.internal.misc=ALL-UNNAMED"); + } - // Needed for netty.DnsResolverUtil on JDK9+ if (SystemUtils.isJavaVersionAtLeast(JavaVersion.JAVA_9)) { + // Needed for optimized checksum calculation when com.scurrilous.circe.checksum.Java9IntHash + // is used. That gets used when the native library libcirce-checksum is not available or cannot + // be loaded. args.add("--add-opens"); - args.add("java.base/sun.net=ALL-UNNAMED"); + args.add("java.base/java.util.zip=ALL-UNNAMED"); } if (instanceConfig.getAdditionalJavaRuntimeArguments() != null) { diff --git a/pulsar-functions/runtime/src/test/java/org/apache/pulsar/functions/runtime/kubernetes/KubernetesRuntimeTest.java b/pulsar-functions/runtime/src/test/java/org/apache/pulsar/functions/runtime/kubernetes/KubernetesRuntimeTest.java index 02f3c0d23fb17..6ed9849412910 100644 --- a/pulsar-functions/runtime/src/test/java/org/apache/pulsar/functions/runtime/kubernetes/KubernetesRuntimeTest.java +++ b/pulsar-functions/runtime/src/test/java/org/apache/pulsar/functions/runtime/kubernetes/KubernetesRuntimeTest.java @@ -441,14 +441,14 @@ private void verifyJavaInstance(InstanceConfig config, String depsDir, boolean s if (null != depsDir) { extraDepsEnv = " -Dpulsar.functions.extra.dependencies.dir=" + depsDir; classpath = classpath + ":" + depsDir + "/*"; - totalArgs = 46; - portArg = 33; - metricsPortArg = 35; + totalArgs = 52; + portArg = 39; + metricsPortArg = 41; } else { extraDepsEnv = ""; - portArg = 32; - metricsPortArg = 34; - totalArgs = 45; + portArg = 38; + metricsPortArg = 40; + totalArgs = 51; } if (secretsAttached) { totalArgs += 4; @@ -479,7 +479,11 @@ private void verifyJavaInstance(InstanceConfig config, String depsDir, boolean s + "-Dpulsar.function.log.dir=" + logDirectory + "/" + FunctionCommon.getFullyQualifiedName(config.getFunctionDetails()) + " -Dpulsar.function.log.file=" + config.getFunctionDetails().getName() + "-$SHARD_ID" + " -Dio.netty.tryReflectionSetAccessible=true" - + " --add-opens java.base/sun.net=ALL-UNNAMED" + + " -Dorg.apache.pulsar.shade.io.netty.tryReflectionSetAccessible=true" + + " -Dio.grpc.netty.shaded.io.netty.tryReflectionSetAccessible=true" + + " --add-opens java.base/java.nio=ALL-UNNAMED" + + " --add-opens java.base/jdk.internal.misc=ALL-UNNAMED" + + " --add-opens java.base/java.util.zip=ALL-UNNAMED" + " -Xmx" + RESOURCES.getRam() + " org.apache.pulsar.functions.instance.JavaInstanceMain" + " --jar " + jarLocation @@ -1306,7 +1310,7 @@ private void assertMetricsPortConfigured(Map functionRuntimeFact .contains("--metrics_port 0")); } } - + @Test public void testDeleteStatefulSetWithTranslatedKubernetesLabelChars() throws Exception { InstanceConfig config = createJavaInstanceConfig(FunctionDetails.Runtime.JAVA, false); @@ -1315,22 +1319,22 @@ public void testDeleteStatefulSetWithTranslatedKubernetesLabelChars() throws Exc CoreV1Api coreApi = mock(CoreV1Api.class); AppsV1Api appsApi = mock(AppsV1Api.class); - + Call successfulCall = mock(Call.class); Response okResponse = mock(Response.class); when(okResponse.code()).thenReturn(HttpURLConnection.HTTP_OK); when(okResponse.isSuccessful()).thenReturn(true); when(okResponse.message()).thenReturn(""); when(successfulCall.execute()).thenReturn(okResponse); - + final String expectedFunctionNamePrefix = String.format("pf-%s-%s-%s", "c-tenant", "c-ns", "c-fn"); - + factory = createKubernetesRuntimeFactory(null, 10, 1.0, 1.0); factory.setCoreClient(coreApi); factory.setAppsClient(appsApi); ArgumentMatcher hasTranslatedFunctionName = (String t) -> t.startsWith(expectedFunctionNamePrefix); - + when(appsApi.deleteNamespacedStatefulSetCall( argThat(hasTranslatedFunctionName), anyString(), isNull(), isNull(), anyInt(), isNull(), anyString(), any(), isNull())).thenReturn(successfulCall); @@ -1342,14 +1346,14 @@ public void testDeleteStatefulSetWithTranslatedKubernetesLabelChars() throws Exc V1PodList podList = mock(V1PodList.class); when(podList.getItems()).thenReturn(Collections.emptyList()); - + String expectedLabels = String.format("tenant=%s,namespace=%s,name=%s", "c-tenant", "c-ns", "c-fn"); - + when(coreApi.listNamespacedPod(anyString(), isNull(), isNull(), isNull(), isNull(), eq(expectedLabels), isNull(), isNull(), isNull(), isNull(), isNull())).thenReturn(podList); - KubernetesRuntime kr = factory.createContainer(config, "/test/code", "code.yml", "/test/transforms", "transform.yml", Long.MIN_VALUE); + KubernetesRuntime kr = factory.createContainer(config, "/test/code", "code.yml", "/test/transforms", "transform.yml", Long.MIN_VALUE); kr.deleteStatefulSet(); - + verify(coreApi).listNamespacedPod(anyString(), isNull(), isNull(), isNull(), isNull(), eq(expectedLabels), isNull(), isNull(), isNull(), isNull(), isNull()); } diff --git a/pulsar-functions/runtime/src/test/java/org/apache/pulsar/functions/runtime/process/ProcessRuntimeTest.java b/pulsar-functions/runtime/src/test/java/org/apache/pulsar/functions/runtime/process/ProcessRuntimeTest.java index f63f24dc25624..365704ea0b4ed 100644 --- a/pulsar-functions/runtime/src/test/java/org/apache/pulsar/functions/runtime/process/ProcessRuntimeTest.java +++ b/pulsar-functions/runtime/src/test/java/org/apache/pulsar/functions/runtime/process/ProcessRuntimeTest.java @@ -297,7 +297,7 @@ private void verifyJavaInstance(InstanceConfig config, Path depsDir, String webS String extraDepsEnv; int portArg; int metricsPortArg; - int totalArgCount = 48; + int totalArgCount = 54; if (webServiceUrl != null && config.isExposePulsarAdminClientEnabled()) { totalArgCount += 3; } @@ -305,13 +305,13 @@ private void verifyJavaInstance(InstanceConfig config, Path depsDir, String webS assertEquals(args.size(), totalArgCount); extraDepsEnv = " -Dpulsar.functions.extra.dependencies.dir=" + depsDir; classpath = classpath + ":" + depsDir + "/*"; - portArg = 31; - metricsPortArg = 33; + portArg = 37; + metricsPortArg = 39; } else { assertEquals(args.size(), totalArgCount-1); extraDepsEnv = ""; - portArg = 30; - metricsPortArg = 32; + portArg = 36; + metricsPortArg = 38; } if (webServiceUrl != null && config.isExposePulsarAdminClientEnabled()) { portArg += 3; @@ -328,7 +328,11 @@ private void verifyJavaInstance(InstanceConfig config, Path depsDir, String webS + "-Dpulsar.function.log.dir=" + logDirectory + "/functions/" + FunctionCommon.getFullyQualifiedName(config.getFunctionDetails()) + " -Dpulsar.function.log.file=" + config.getFunctionDetails().getName() + "-" + config.getInstanceId() + " -Dio.netty.tryReflectionSetAccessible=true" - + " --add-opens java.base/sun.net=ALL-UNNAMED" + + " -Dorg.apache.pulsar.shade.io.netty.tryReflectionSetAccessible=true" + + " -Dio.grpc.netty.shaded.io.netty.tryReflectionSetAccessible=true" + + " --add-opens java.base/java.nio=ALL-UNNAMED" + + " --add-opens java.base/jdk.internal.misc=ALL-UNNAMED" + + " --add-opens java.base/java.util.zip=ALL-UNNAMED" + " org.apache.pulsar.functions.instance.JavaInstanceMain" + " --jar " + userJarFile + " --transform_function_jar " + userJarFile diff --git a/pulsar-functions/utils/src/main/java/org/apache/pulsar/functions/utils/FunctionConfigUtils.java b/pulsar-functions/utils/src/main/java/org/apache/pulsar/functions/utils/FunctionConfigUtils.java index ee59317daf755..3c96837e4374e 100644 --- a/pulsar-functions/utils/src/main/java/org/apache/pulsar/functions/utils/FunctionConfigUtils.java +++ b/pulsar-functions/utils/src/main/java/org/apache/pulsar/functions/utils/FunctionConfigUtils.java @@ -250,29 +250,7 @@ public static FunctionDetails convert(FunctionConfig functionConfig, ExtractedFu sinkSpecBuilder.setTypeClassName(functionConfig.getOutputTypeClassName()); } if (functionConfig.getProducerConfig() != null) { - ProducerConfig producerConf = functionConfig.getProducerConfig(); - Function.ProducerSpec.Builder pbldr = Function.ProducerSpec.newBuilder(); - if (producerConf.getMaxPendingMessages() != null) { - pbldr.setMaxPendingMessages(producerConf.getMaxPendingMessages()); - } - if (producerConf.getMaxPendingMessagesAcrossPartitions() != null) { - pbldr.setMaxPendingMessagesAcrossPartitions(producerConf.getMaxPendingMessagesAcrossPartitions()); - } - if (producerConf.getUseThreadLocalProducers() != null) { - pbldr.setUseThreadLocalProducers(producerConf.getUseThreadLocalProducers()); - } - if (producerConf.getCryptoConfig() != null) { - pbldr.setCryptoSpec(CryptoUtils.convert(producerConf.getCryptoConfig())); - } - if (producerConf.getBatchBuilder() != null) { - pbldr.setBatchBuilder(producerConf.getBatchBuilder()); - } - if (producerConf.getCompressionType() != null) { - pbldr.setCompressionType(convertFromCompressionType(producerConf.getCompressionType())); - } else { - pbldr.setCompressionType(Function.CompressionType.LZ4); - } - sinkSpecBuilder.setProducerSpec(pbldr.build()); + sinkSpecBuilder.setProducerSpec(convertProducerConfigToProducerSpec(functionConfig.getProducerConfig())); } if (functionConfig.getBatchBuilder() != null) { Function.ProducerSpec.Builder builder = sinkSpecBuilder.getProducerSpec() != null @@ -463,23 +441,8 @@ public static FunctionConfig convertFromDetails(FunctionDetails functionDetails) functionConfig.setOutputSchemaType(functionDetails.getSink().getSchemaType()); } if (functionDetails.getSink().getProducerSpec() != null) { - Function.ProducerSpec spec = functionDetails.getSink().getProducerSpec(); - ProducerConfig producerConfig = new ProducerConfig(); - if (spec.getMaxPendingMessages() != 0) { - producerConfig.setMaxPendingMessages(spec.getMaxPendingMessages()); - } - if (spec.getMaxPendingMessagesAcrossPartitions() != 0) { - producerConfig.setMaxPendingMessagesAcrossPartitions(spec.getMaxPendingMessagesAcrossPartitions()); - } - if (spec.hasCryptoSpec()) { - producerConfig.setCryptoConfig(CryptoUtils.convertFromSpec(spec.getCryptoSpec())); - } - if (spec.getBatchBuilder() != null) { - producerConfig.setBatchBuilder(spec.getBatchBuilder()); - } - producerConfig.setUseThreadLocalProducers(spec.getUseThreadLocalProducers()); - producerConfig.setCompressionType(convertFromFunctionDetailsCompressionType(spec.getCompressionType())); - functionConfig.setProducerConfig(producerConfig); + functionConfig.setProducerConfig( + convertProducerSpecToProducerConfig(functionDetails.getSink().getProducerSpec())); } if (!isEmpty(functionDetails.getLogTopic())) { functionConfig.setLogTopic(functionDetails.getLogTopic()); @@ -544,6 +507,50 @@ public static FunctionConfig convertFromDetails(FunctionDetails functionDetails) return functionConfig; } + public static Function.ProducerSpec convertProducerConfigToProducerSpec(ProducerConfig producerConf) { + Function.ProducerSpec.Builder builder = Function.ProducerSpec.newBuilder(); + if (producerConf.getMaxPendingMessages() != null) { + builder.setMaxPendingMessages(producerConf.getMaxPendingMessages()); + } + if (producerConf.getMaxPendingMessagesAcrossPartitions() != null) { + builder.setMaxPendingMessagesAcrossPartitions(producerConf.getMaxPendingMessagesAcrossPartitions()); + } + if (producerConf.getUseThreadLocalProducers() != null) { + builder.setUseThreadLocalProducers(producerConf.getUseThreadLocalProducers()); + } + if (producerConf.getCryptoConfig() != null) { + builder.setCryptoSpec(CryptoUtils.convert(producerConf.getCryptoConfig())); + } + if (producerConf.getBatchBuilder() != null) { + builder.setBatchBuilder(producerConf.getBatchBuilder()); + } + if (producerConf.getCompressionType() != null) { + builder.setCompressionType(convertFromCompressionType(producerConf.getCompressionType())); + } else { + builder.setCompressionType(Function.CompressionType.LZ4); + } + return builder.build(); + } + + public static ProducerConfig convertProducerSpecToProducerConfig(Function.ProducerSpec spec) { + ProducerConfig producerConfig = new ProducerConfig(); + if (spec.getMaxPendingMessages() != 0) { + producerConfig.setMaxPendingMessages(spec.getMaxPendingMessages()); + } + if (spec.getMaxPendingMessagesAcrossPartitions() != 0) { + producerConfig.setMaxPendingMessagesAcrossPartitions(spec.getMaxPendingMessagesAcrossPartitions()); + } + if (spec.hasCryptoSpec()) { + producerConfig.setCryptoConfig(CryptoUtils.convertFromSpec(spec.getCryptoSpec())); + } + if (spec.getBatchBuilder() != null) { + producerConfig.setBatchBuilder(spec.getBatchBuilder()); + } + producerConfig.setUseThreadLocalProducers(spec.getUseThreadLocalProducers()); + producerConfig.setCompressionType(convertFromFunctionDetailsCompressionType(spec.getCompressionType())); + return producerConfig; + } + public static void inferMissingArguments(FunctionConfig functionConfig, boolean forwardSourceMessagePropertyEnabled) { if (StringUtils.isEmpty(functionConfig.getName())) { diff --git a/pulsar-functions/utils/src/main/java/org/apache/pulsar/functions/utils/SourceConfigUtils.java b/pulsar-functions/utils/src/main/java/org/apache/pulsar/functions/utils/SourceConfigUtils.java index a6430bbea4585..2239f0fcbc2b7 100644 --- a/pulsar-functions/utils/src/main/java/org/apache/pulsar/functions/utils/SourceConfigUtils.java +++ b/pulsar-functions/utils/src/main/java/org/apache/pulsar/functions/utils/SourceConfigUtils.java @@ -19,10 +19,10 @@ package org.apache.pulsar.functions.utils; import static org.apache.commons.lang3.StringUtils.isEmpty; -import static org.apache.pulsar.functions.utils.FunctionCommon.convertFromCompressionType; -import static org.apache.pulsar.functions.utils.FunctionCommon.convertFromFunctionDetailsCompressionType; import static org.apache.pulsar.functions.utils.FunctionCommon.convertProcessingGuarantee; import static org.apache.pulsar.functions.utils.FunctionCommon.getSourceType; +import static org.apache.pulsar.functions.utils.FunctionConfigUtils.convertProducerConfigToProducerSpec; +import static org.apache.pulsar.functions.utils.FunctionConfigUtils.convertProducerSpecToProducerConfig; import com.fasterxml.jackson.core.type.TypeReference; import com.google.gson.Gson; import com.google.gson.reflect.TypeToken; @@ -39,7 +39,6 @@ import net.bytebuddy.description.type.TypeDescription; import net.bytebuddy.pool.TypePool; import org.apache.commons.lang3.StringUtils; -import org.apache.pulsar.common.functions.ProducerConfig; import org.apache.pulsar.common.functions.Resources; import org.apache.pulsar.common.io.BatchSourceConfig; import org.apache.pulsar.common.io.ConnectorDefinition; @@ -149,29 +148,7 @@ public static FunctionDetails convert(SourceConfig sourceConfig, ExtractedSource } if (sourceConfig.getProducerConfig() != null) { - ProducerConfig conf = sourceConfig.getProducerConfig(); - Function.ProducerSpec.Builder pbldr = Function.ProducerSpec.newBuilder(); - if (conf.getMaxPendingMessages() != null) { - pbldr.setMaxPendingMessages(conf.getMaxPendingMessages()); - } - if (conf.getMaxPendingMessagesAcrossPartitions() != null) { - pbldr.setMaxPendingMessagesAcrossPartitions(conf.getMaxPendingMessagesAcrossPartitions()); - } - if (conf.getUseThreadLocalProducers() != null) { - pbldr.setUseThreadLocalProducers(conf.getUseThreadLocalProducers()); - } - if (conf.getCryptoConfig() != null) { - pbldr.setCryptoSpec(CryptoUtils.convert(conf.getCryptoConfig())); - } - if (conf.getBatchBuilder() != null) { - pbldr.setBatchBuilder(conf.getBatchBuilder()); - } - if (conf.getCompressionType() != null) { - pbldr.setCompressionType(convertFromCompressionType(conf.getCompressionType())); - } else { - pbldr.setCompressionType(Function.CompressionType.LZ4); - } - sinkSpecBuilder.setProducerSpec(pbldr.build()); + sinkSpecBuilder.setProducerSpec(convertProducerConfigToProducerSpec(sourceConfig.getProducerConfig())); } if (sourceConfig.getBatchBuilder() != null) { @@ -256,23 +233,7 @@ public static SourceConfig convertFromDetails(FunctionDetails functionDetails) { sourceConfig.setSerdeClassName(sinkSpec.getSerDeClassName()); } if (sinkSpec.getProducerSpec() != null) { - Function.ProducerSpec spec = sinkSpec.getProducerSpec(); - ProducerConfig producerConfig = new ProducerConfig(); - if (spec.getMaxPendingMessages() != 0) { - producerConfig.setMaxPendingMessages(spec.getMaxPendingMessages()); - } - if (spec.getMaxPendingMessagesAcrossPartitions() != 0) { - producerConfig.setMaxPendingMessagesAcrossPartitions(spec.getMaxPendingMessagesAcrossPartitions()); - } - if (spec.hasCryptoSpec()) { - producerConfig.setCryptoConfig(CryptoUtils.convertFromSpec(spec.getCryptoSpec())); - } - if (spec.getBatchBuilder() != null) { - producerConfig.setBatchBuilder(spec.getBatchBuilder()); - } - producerConfig.setUseThreadLocalProducers(spec.getUseThreadLocalProducers()); - producerConfig.setCompressionType(convertFromFunctionDetailsCompressionType(spec.getCompressionType())); - sourceConfig.setProducerConfig(producerConfig); + sourceConfig.setProducerConfig(convertProducerSpecToProducerConfig(sinkSpec.getProducerSpec())); } if (functionDetails.hasResources()) { Resources resources = new Resources(); diff --git a/pulsar-functions/utils/src/test/java/org/apache/pulsar/functions/utils/FunctionConfigUtilsTest.java b/pulsar-functions/utils/src/test/java/org/apache/pulsar/functions/utils/FunctionConfigUtilsTest.java index 954eef44a7366..cf4e7dd92a8f7 100644 --- a/pulsar-functions/utils/src/test/java/org/apache/pulsar/functions/utils/FunctionConfigUtilsTest.java +++ b/pulsar-functions/utils/src/test/java/org/apache/pulsar/functions/utils/FunctionConfigUtilsTest.java @@ -35,9 +35,12 @@ import java.util.Map; import lombok.extern.slf4j.Slf4j; import org.apache.pulsar.client.api.CompressionType; +import org.apache.pulsar.client.api.ConsumerCryptoFailureAction; +import org.apache.pulsar.client.api.ProducerCryptoFailureAction; import org.apache.pulsar.client.api.SubscriptionInitialPosition; import org.apache.pulsar.client.impl.schema.JSONSchema; import org.apache.pulsar.common.functions.ConsumerConfig; +import org.apache.pulsar.common.functions.CryptoConfig; import org.apache.pulsar.common.functions.FunctionConfig; import org.apache.pulsar.common.functions.ProducerConfig; import org.apache.pulsar.common.functions.Resources; @@ -667,4 +670,36 @@ public void testPoolMessages() { convertedConfig = FunctionConfigUtils.convertFromDetails(functionDetails); assertTrue(convertedConfig.getInputSpecs().get("test-input").isPoolMessages()); } + + @Test + public void testConvertProducerSpecToProducerConfigAndBackToProducerSpec() { + // given + Function.ProducerSpec producerSpec = Function.ProducerSpec.newBuilder() + .setBatchBuilder("KEY_BASED") + .setCompressionType(Function.CompressionType.ZSTD) + .setCryptoSpec(Function.CryptoSpec.newBuilder() + .addProducerEncryptionKeyName("key1") + .addProducerEncryptionKeyName("key2") + .setConsumerCryptoFailureAction(Function.CryptoSpec.FailureAction.DISCARD) + .setProducerCryptoFailureAction(Function.CryptoSpec.FailureAction.SEND) + .setCryptoKeyReaderClassName("ReaderClassName") + .setCryptoKeyReaderConfig("{\"key\":\"value\"}") + .build()) + .build(); + // when + ProducerConfig producerConfig = FunctionConfigUtils.convertProducerSpecToProducerConfig(producerSpec); + // then + assertEquals(producerConfig.getBatchBuilder(), "KEY_BASED"); + assertEquals(producerConfig.getCompressionType(), CompressionType.ZSTD); + CryptoConfig cryptoConfig = producerConfig.getCryptoConfig(); + assertEquals(cryptoConfig.getProducerCryptoFailureAction(), ProducerCryptoFailureAction.SEND); + assertEquals(cryptoConfig.getConsumerCryptoFailureAction(), ConsumerCryptoFailureAction.DISCARD); + assertEquals(cryptoConfig.getEncryptionKeys(), new String[]{"key1", "key2"}); + assertEquals(cryptoConfig.getCryptoKeyReaderClassName(), "ReaderClassName"); + // and when + // converted back to producer spec + Function.ProducerSpec producerSpec2 = FunctionConfigUtils.convertProducerConfigToProducerSpec(producerConfig); + // then + assertEquals(producerSpec2, producerSpec); + } } diff --git a/pulsar-metadata/src/main/java/org/apache/pulsar/metadata/bookkeeper/BKCluster.java b/pulsar-metadata/src/main/java/org/apache/pulsar/metadata/bookkeeper/BKCluster.java index c2f3f72ec21c0..fe2b981ffe995 100644 --- a/pulsar-metadata/src/main/java/org/apache/pulsar/metadata/bookkeeper/BKCluster.java +++ b/pulsar-metadata/src/main/java/org/apache/pulsar/metadata/bookkeeper/BKCluster.java @@ -30,6 +30,7 @@ import java.util.Iterator; import java.util.List; import java.util.Map; +import java.util.concurrent.atomic.AtomicBoolean; import java.util.regex.Matcher; import java.util.regex.Pattern; import lombok.Getter; @@ -49,8 +50,8 @@ import org.apache.bookkeeper.replication.AutoRecoveryMain; import org.apache.bookkeeper.server.conf.BookieConfiguration; import org.apache.bookkeeper.util.IOUtils; -import org.apache.bookkeeper.util.PortManager; import org.apache.commons.io.FileUtils; +import org.apache.pulsar.common.util.PortManager; import org.apache.pulsar.metadata.api.MetadataStoreConfig; import org.apache.pulsar.metadata.api.extended.MetadataStoreExtended; @@ -74,6 +75,9 @@ public class BKCluster implements AutoCloseable { protected final ServerConfiguration baseConf; protected final ClientConfiguration baseClientConf; + private final List lockedPorts = new ArrayList<>(); + private final AtomicBoolean closed = new AtomicBoolean(false); + public static class BKClusterConf { private ServerConfiguration baseServerConfiguration; @@ -148,20 +152,24 @@ private BKCluster(BKClusterConf bkClusterConf) throws Exception { @Override public void close() throws Exception { - // stop bookkeeper service - try { - stopBKCluster(); - } catch (Exception e) { - log.error("Got Exception while trying to stop BKCluster", e); - } - // cleanup temp dirs - try { - cleanupTempDirs(); - } catch (Exception e) { - log.error("Got Exception while trying to cleanupTempDirs", e); - } + if (closed.compareAndSet(false, true)) { + // stop bookkeeper service + try { + stopBKCluster(); + } catch (Exception e) { + log.error("Got Exception while trying to stop BKCluster", e); + } + lockedPorts.forEach(PortManager::releaseLockedPort); + lockedPorts.clear(); + // cleanup temp dirs + try { + cleanupTempDirs(); + } catch (Exception e) { + log.error("Got Exception while trying to cleanupTempDirs", e); + } - this.store.close(); + this.store.close(); + } } private File createTempDir(String prefix, String suffix) throws IOException { @@ -224,12 +232,14 @@ private ServerConfiguration newServerConfiguration(int index) throws Exception { } if (clusterConf.clearOldData && dataDir.exists()) { + log.info("Wiping Bookie data directory at {}", dataDir.getAbsolutePath()); cleanDirectory(dataDir); } int port; if (baseConf.isEnableLocalTransport() || !baseConf.getAllowEphemeralPorts() || clusterConf.bkPort == 0) { - port = PortManager.nextFreePort(); + port = PortManager.nextLockedFreePort(); + lockedPorts.add(port); } else { // bk 4.15 cookie validation finds the same ip:port in case of port 0 // and 2nd bookie's cookie validation fails @@ -399,4 +409,8 @@ private static ServerConfiguration setLoopbackInterfaceAndAllowLoopback(ServerCo serverConf.setAllowLoopback(true); return serverConf; } + + public boolean isClosed() { + return closed.get(); + } } diff --git a/pulsar-metadata/src/main/java/org/apache/pulsar/metadata/impl/AbstractMetadataStore.java b/pulsar-metadata/src/main/java/org/apache/pulsar/metadata/impl/AbstractMetadataStore.java index b520818fb28d6..d099d79d05c4d 100644 --- a/pulsar-metadata/src/main/java/org/apache/pulsar/metadata/impl/AbstractMetadataStore.java +++ b/pulsar-metadata/src/main/java/org/apache/pulsar/metadata/impl/AbstractMetadataStore.java @@ -89,7 +89,9 @@ public abstract class AbstractMetadataStore implements MetadataStoreExtended, Co protected abstract CompletableFuture existsFromStore(String path); protected AbstractMetadataStore(String metadataStoreName) { - this.executor = new ScheduledThreadPoolExecutor(1, new DefaultThreadFactory(metadataStoreName)); + this.executor = new ScheduledThreadPoolExecutor(1, + new DefaultThreadFactory( + StringUtils.isNotBlank(metadataStoreName) ? metadataStoreName : getClass().getSimpleName())); registerListener(this); this.childrenCache = Caffeine.newBuilder() diff --git a/pulsar-metadata/src/test/java/org/apache/bookkeeper/replication/BookKeeperClusterTestCase.java b/pulsar-metadata/src/test/java/org/apache/bookkeeper/replication/BookKeeperClusterTestCase.java index c681a1f0764ee..9a8e3ef5a2d4f 100644 --- a/pulsar-metadata/src/test/java/org/apache/bookkeeper/replication/BookKeeperClusterTestCase.java +++ b/pulsar-metadata/src/test/java/org/apache/bookkeeper/replication/BookKeeperClusterTestCase.java @@ -238,7 +238,7 @@ protected void startZKCluster() throws Exception { zkc = zkUtil.getZooKeeperClient(); metadataStore = new FaultInjectionMetadataStore( MetadataStoreExtended.create(zkUtil.getZooKeeperConnectString(), - MetadataStoreConfig.builder().build())); + MetadataStoreConfig.builder().metadataStoreName("metastore-" + getClass().getSimpleName()).build())); } /** diff --git a/pulsar-proxy/src/main/java/org/apache/pulsar/proxy/server/DefaultLookupProxyHandler.java b/pulsar-proxy/src/main/java/org/apache/pulsar/proxy/server/DefaultLookupProxyHandler.java index 15163dd7f6063..c976e44c05cdf 100644 --- a/pulsar-proxy/src/main/java/org/apache/pulsar/proxy/server/DefaultLookupProxyHandler.java +++ b/pulsar-proxy/src/main/java/org/apache/pulsar/proxy/server/DefaultLookupProxyHandler.java @@ -251,7 +251,7 @@ private void handlePartitionMetadataResponse(CommandPartitionedTopicMetadata par // Connected to backend broker long requestId = proxyConnection.newRequestId(); ByteBuf command; - command = Commands.newPartitionMetadataRequest(topicName.toString(), requestId); + command = Commands.newPartitionMetadataRequest(topicName.toString(), requestId, true); clientCnx.newLookup(command, requestId).whenComplete((r, t) -> { if (t != null) { log.warn("[{}] failed to get Partitioned metadata : {}", topicName.toString(), diff --git a/pulsar-sql/presto-distribution/LICENSE b/pulsar-sql/presto-distribution/LICENSE index 9242877df015f..7ca0a9fe76274 100644 --- a/pulsar-sql/presto-distribution/LICENSE +++ b/pulsar-sql/presto-distribution/LICENSE @@ -255,6 +255,7 @@ The Apache Software License, Version 2.0 - netty-tcnative-classes-2.0.65.Final.jar - netty-transport-4.1.108.Final.jar - netty-transport-classes-epoll-4.1.108.Final.jar + - netty-transport-native-epoll-4.1.108.Final-linux-aarch_64.jar - netty-transport-native-epoll-4.1.108.Final-linux-x86_64.jar - netty-transport-native-unix-common-4.1.108.Final.jar - netty-transport-native-unix-common-4.1.108.Final-linux-x86_64.jar @@ -483,7 +484,8 @@ The Apache Software License, Version 2.0 - stream-2.9.5.jar * High Performance Primitive Collections for Java - hppc-0.9.1.jar - + * RoaringBitmap + - RoaringBitmap-1.1.0.jar Protocol Buffers License * Protocol Buffers diff --git a/pulsar-transaction/coordinator/src/test/java/org/apache/pulsar/transaction/coordinator/test/MockedBookKeeperTestCase.java b/pulsar-transaction/coordinator/src/test/java/org/apache/pulsar/transaction/coordinator/test/MockedBookKeeperTestCase.java index e0b10ca0280d2..ac5aa3bd8927e 100644 --- a/pulsar-transaction/coordinator/src/test/java/org/apache/pulsar/transaction/coordinator/test/MockedBookKeeperTestCase.java +++ b/pulsar-transaction/coordinator/src/test/java/org/apache/pulsar/transaction/coordinator/test/MockedBookKeeperTestCase.java @@ -71,7 +71,9 @@ public MockedBookKeeperTestCase(int numBookies) { public void setUp(Method method) throws Exception { LOG.info(">>>>>> starting {}", method); metadataStore = new FaultInjectionMetadataStore(MetadataStoreExtended.create("memory:local", - MetadataStoreConfig.builder().build())); + MetadataStoreConfig.builder() + .metadataStoreName("metastore-" + method.getName()) + .build())); try { // start bookkeeper service startBookKeeper();