diff --git a/pulsar-broker-common/src/main/java/org/apache/pulsar/broker/resources/TenantResources.java b/pulsar-broker-common/src/main/java/org/apache/pulsar/broker/resources/TenantResources.java index 5336a9351a80e9..87f48f7e682a4d 100644 --- a/pulsar-broker-common/src/main/java/org/apache/pulsar/broker/resources/TenantResources.java +++ b/pulsar-broker-common/src/main/java/org/apache/pulsar/broker/resources/TenantResources.java @@ -140,9 +140,9 @@ public CompletableFuture> getListOfNamespacesAsync(String tenant) { return ret; } })).reduce(CompletableFuture.completedFuture(new ArrayList<>()), - (accumulator, n) -> accumulator.thenCompose(namespaces -> n.thenCompose(m -> { + (accumulator, n) -> accumulator.thenCompose(namespaces -> n.thenApply(m -> { namespaces.addAll(m); - return CompletableFuture.completedFuture(namespaces); + return namespaces; })))); } diff --git a/pulsar-broker/src/main/java/org/apache/pulsar/broker/admin/impl/NamespacesBase.java b/pulsar-broker/src/main/java/org/apache/pulsar/broker/admin/impl/NamespacesBase.java index cea220593b6fa9..39a4eb13060832 100644 --- a/pulsar-broker/src/main/java/org/apache/pulsar/broker/admin/impl/NamespacesBase.java +++ b/pulsar-broker/src/main/java/org/apache/pulsar/broker/admin/impl/NamespacesBase.java @@ -135,11 +135,10 @@ protected CompletableFuture internalCreateNamespace(Policies policies) { .thenCompose(__ -> validatePoliciesReadOnlyAccessAsync()) .thenAccept(__ -> validatePolicies(namespaceName, policies)) .thenCompose(__ -> { - CompletableFuture ret = CompletableFuture.completedFuture(null); int maxNamespacesPerTenant = pulsar().getConfiguration().getMaxNamespacesPerTenant(); // no distributed locks are added here.In a concurrent scenario, the threshold will be exceeded. if (maxNamespacesPerTenant > 0) { - ret = tenantResources().getListOfNamespacesAsync(namespaceName.getTenant()) + return tenantResources().getListOfNamespacesAsync(namespaceName.getTenant()) .thenAccept(namespaces -> { if (namespaces != null && namespaces.size() > maxNamespacesPerTenant) { throw new RestException(Status.PRECONDITION_FAILED, @@ -148,7 +147,7 @@ protected CompletableFuture internalCreateNamespace(Policies policies) { } }); } - return ret; + return CompletableFuture.completedFuture(null); }) .thenCompose(__ -> namespaceResources().createPoliciesAsync(namespaceName, policies)) .thenAccept(__ -> log.info("[{}] Created namespace {}", clientAppId(), namespaceName));