|
38 | 38 | import java.util.concurrent.ScheduledFuture;
|
39 | 39 | import java.util.concurrent.TimeUnit;
|
40 | 40 | import java.util.concurrent.atomic.AtomicReference;
|
| 41 | +import java.util.function.Function; |
41 | 42 | import java.util.stream.Collectors;
|
42 | 43 | import lombok.Getter;
|
43 | 44 | import lombok.extern.slf4j.Slf4j;
|
|
84 | 85 | import org.apache.pulsar.broker.loadbalance.extensions.strategy.LeastResourceUsageWithWeight;
|
85 | 86 | import org.apache.pulsar.broker.loadbalance.impl.LoadManagerShared;
|
86 | 87 | import org.apache.pulsar.broker.loadbalance.impl.SimpleResourceAllocationPolicies;
|
| 88 | +import org.apache.pulsar.broker.namespace.NamespaceEphemeralData; |
87 | 89 | import org.apache.pulsar.client.admin.PulsarAdminException;
|
88 | 90 | import org.apache.pulsar.common.naming.NamespaceBundle;
|
89 | 91 | import org.apache.pulsar.common.naming.NamespaceBundleSplitAlgorithm;
|
@@ -365,56 +367,99 @@ public CompletableFuture<Optional<BrokerLookupData>> assign(Optional<ServiceUnit
|
365 | 367 |
|
366 | 368 | final String bundle = serviceUnit.toString();
|
367 | 369 |
|
368 |
| - CompletableFuture<Optional<BrokerLookupData>> future = lookupRequests.computeIfAbsent(bundle, k -> { |
| 370 | + return dedupeLookupRequest(bundle, k -> { |
369 | 371 | final CompletableFuture<Optional<String>> owner;
|
370 | 372 | // Assign the bundle to channel owner if is internal topic, to avoid circular references.
|
371 | 373 | if (topic.isPresent() && isInternalTopic(topic.get().toString())) {
|
372 | 374 | owner = serviceUnitStateChannel.getChannelOwnerAsync();
|
373 | 375 | } else {
|
374 |
| - owner = serviceUnitStateChannel.getOwnerAsync(bundle).thenCompose(broker -> { |
375 |
| - // If the bundle not assign yet, select and publish assign event to channel. |
376 |
| - if (broker.isEmpty()) { |
377 |
| - return this.selectAsync(serviceUnit).thenCompose(brokerOpt -> { |
378 |
| - if (brokerOpt.isPresent()) { |
379 |
| - assignCounter.incrementSuccess(); |
380 |
| - log.info("Selected new owner broker: {} for bundle: {}.", brokerOpt.get(), bundle); |
381 |
| - return serviceUnitStateChannel.publishAssignEventAsync(bundle, brokerOpt.get()) |
382 |
| - .thenApply(Optional::of); |
383 |
| - } else { |
384 |
| - throw new IllegalStateException( |
385 |
| - "Failed to select the new owner broker for bundle: " + bundle); |
386 |
| - } |
387 |
| - }); |
| 376 | + owner = getOwnerAsync(serviceUnit, bundle, false).thenApply(Optional::ofNullable); |
| 377 | + } |
| 378 | + return getBrokerLookupData(owner, bundle); |
| 379 | + }); |
| 380 | + } |
| 381 | + |
| 382 | + private CompletableFuture<String> getOwnerAsync( |
| 383 | + ServiceUnitId serviceUnit, String bundle, boolean ownByLocalBrokerIfAbsent) { |
| 384 | + return serviceUnitStateChannel.getOwnerAsync(bundle).thenCompose(broker -> { |
| 385 | + // If the bundle not assign yet, select and publish assign event to channel. |
| 386 | + if (broker.isEmpty()) { |
| 387 | + CompletableFuture<Optional<String>> selectedBroker; |
| 388 | + if (ownByLocalBrokerIfAbsent) { |
| 389 | + String brokerId = this.brokerRegistry.getBrokerId(); |
| 390 | + selectedBroker = CompletableFuture.completedFuture(Optional.of(brokerId)); |
| 391 | + } else { |
| 392 | + selectedBroker = this.selectAsync(serviceUnit); |
| 393 | + } |
| 394 | + return selectedBroker.thenCompose(brokerOpt -> { |
| 395 | + if (brokerOpt.isPresent()) { |
| 396 | + assignCounter.incrementSuccess(); |
| 397 | + log.info("Selected new owner broker: {} for bundle: {}.", brokerOpt.get(), bundle); |
| 398 | + return serviceUnitStateChannel.publishAssignEventAsync(bundle, brokerOpt.get()); |
388 | 399 | }
|
389 |
| - assignCounter.incrementSkip(); |
390 |
| - // Already assigned, return it. |
391 |
| - return CompletableFuture.completedFuture(broker); |
| 400 | + throw new IllegalStateException( |
| 401 | + "Failed to select the new owner broker for bundle: " + bundle); |
392 | 402 | });
|
393 | 403 | }
|
| 404 | + assignCounter.incrementSkip(); |
| 405 | + // Already assigned, return it. |
| 406 | + return CompletableFuture.completedFuture(broker.get()); |
| 407 | + }); |
| 408 | + } |
394 | 409 |
|
395 |
| - return owner.thenCompose(broker -> { |
396 |
| - if (broker.isEmpty()) { |
397 |
| - String errorMsg = String.format( |
398 |
| - "Failed to get or assign the owner for bundle:%s", bundle); |
399 |
| - log.error(errorMsg); |
400 |
| - throw new IllegalStateException(errorMsg); |
401 |
| - } |
402 |
| - return CompletableFuture.completedFuture(broker.get()); |
403 |
| - }).thenCompose(broker -> this.getBrokerRegistry().lookupAsync(broker).thenCompose(brokerLookupData -> { |
404 |
| - if (brokerLookupData.isEmpty()) { |
405 |
| - String errorMsg = String.format( |
406 |
| - "Failed to look up a broker registry:%s for bundle:%s", broker, bundle); |
407 |
| - log.error(errorMsg); |
408 |
| - throw new IllegalStateException(errorMsg); |
409 |
| - } |
410 |
| - return CompletableFuture.completedFuture(brokerLookupData); |
411 |
| - })); |
| 410 | + private CompletableFuture<Optional<BrokerLookupData>> getBrokerLookupData( |
| 411 | + CompletableFuture<Optional<String>> owner, |
| 412 | + String bundle) { |
| 413 | + return owner.thenCompose(broker -> { |
| 414 | + if (broker.isEmpty()) { |
| 415 | + String errorMsg = String.format( |
| 416 | + "Failed to get or assign the owner for bundle:%s", bundle); |
| 417 | + log.error(errorMsg); |
| 418 | + throw new IllegalStateException(errorMsg); |
| 419 | + } |
| 420 | + return CompletableFuture.completedFuture(broker.get()); |
| 421 | + }).thenCompose(broker -> this.getBrokerRegistry().lookupAsync(broker).thenCompose(brokerLookupData -> { |
| 422 | + if (brokerLookupData.isEmpty()) { |
| 423 | + String errorMsg = String.format( |
| 424 | + "Failed to look up a broker registry:%s for bundle:%s", broker, bundle); |
| 425 | + log.error(errorMsg); |
| 426 | + throw new IllegalStateException(errorMsg); |
| 427 | + } |
| 428 | + return CompletableFuture.completedFuture(brokerLookupData); |
| 429 | + })); |
| 430 | + } |
| 431 | + |
| 432 | + /** |
| 433 | + * Method to get the current owner of the <code>NamespaceBundle</code> |
| 434 | + * or set the local broker as the owner if absent. |
| 435 | + * |
| 436 | + * @param namespaceBundle the <code>NamespaceBundle</code> |
| 437 | + * @return The ephemeral node data showing the current ownership info in <code>ServiceUnitStateChannel</code> |
| 438 | + */ |
| 439 | + public CompletableFuture<NamespaceEphemeralData> tryAcquiringOwnership(NamespaceBundle namespaceBundle) { |
| 440 | + log.info("Try acquiring ownership for bundle: {} - {}.", namespaceBundle, brokerRegistry.getBrokerId()); |
| 441 | + final String bundle = namespaceBundle.toString(); |
| 442 | + return dedupeLookupRequest(bundle, k -> { |
| 443 | + final CompletableFuture<String> owner = |
| 444 | + this.getOwnerAsync(namespaceBundle, bundle, true); |
| 445 | + return getBrokerLookupData(owner.thenApply(Optional::ofNullable), bundle); |
| 446 | + }).thenApply(brokerLookupData -> { |
| 447 | + if (brokerLookupData.isEmpty()) { |
| 448 | + throw new IllegalStateException( |
| 449 | + "Failed to get the broker lookup data for bundle: " + bundle); |
| 450 | + } |
| 451 | + return brokerLookupData.get().toNamespaceEphemeralData(); |
412 | 452 | });
|
| 453 | + } |
| 454 | + |
| 455 | + private CompletableFuture<Optional<BrokerLookupData>> dedupeLookupRequest( |
| 456 | + String key, Function<String, CompletableFuture<Optional<BrokerLookupData>>> provider) { |
| 457 | + CompletableFuture<Optional<BrokerLookupData>> future = lookupRequests.computeIfAbsent(key, provider); |
413 | 458 | future.whenComplete((r, t) -> {
|
414 | 459 | if (t != null) {
|
415 | 460 | assignCounter.incrementFailure();
|
416 | 461 | }
|
417 |
| - lookupRequests.remove(bundle); |
| 462 | + lookupRequests.remove(key); |
418 | 463 | }
|
419 | 464 | );
|
420 | 465 | return future;
|
|
0 commit comments