You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@pulsar.apache.org by "Technoboy- (via GitHub)" <gi...@apache.org> on 2023/02/01 05:44:39 UTC

[GitHub] [pulsar] Technoboy- commented on a diff in pull request #18810: [improve][broker] PIP-192: Implement broker registry for new load manager

Technoboy- commented on code in PR #18810:
URL: https://github.com/apache/pulsar/pull/18810#discussion_r1092786983


##########
pulsar-broker/src/main/java/org/apache/pulsar/broker/loadbalance/extensions/BrokerRegistryImpl.java:
##########
@@ -0,0 +1,245 @@
+/*
+ * 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.loadbalance.extensions;
+
+import com.google.common.annotations.VisibleForTesting;
+import com.google.common.collect.Lists;
+import java.util.ArrayList;
+import java.util.List;
+import java.util.Map;
+import java.util.Optional;
+import java.util.concurrent.CompletableFuture;
+import java.util.concurrent.CompletionException;
+import java.util.concurrent.ConcurrentHashMap;
+import java.util.concurrent.ExecutionException;
+import java.util.concurrent.RejectedExecutionException;
+import java.util.concurrent.ScheduledExecutorService;
+import java.util.concurrent.TimeUnit;
+import java.util.concurrent.TimeoutException;
+import java.util.function.BiConsumer;
+import lombok.extern.slf4j.Slf4j;
+import org.apache.pulsar.broker.PulsarServerException;
+import org.apache.pulsar.broker.PulsarService;
+import org.apache.pulsar.broker.ServiceConfiguration;
+import org.apache.pulsar.broker.loadbalance.extensions.data.BrokerLookupData;
+import org.apache.pulsar.common.util.FutureUtil;
+import org.apache.pulsar.metadata.api.MetadataStoreException;
+import org.apache.pulsar.metadata.api.Notification;
+import org.apache.pulsar.metadata.api.NotificationType;
+import org.apache.pulsar.metadata.api.coordination.LockManager;
+import org.apache.pulsar.metadata.api.coordination.ResourceLock;
+
+/**
+ * The broker registry impl, base on the LockManager.
+ */
+@Slf4j
+public class BrokerRegistryImpl implements BrokerRegistry {
+
+    protected static final String LOOKUP_DATA_PATH = "/loadbalance/brokers";
+
+    private final PulsarService pulsar;
+
+    private final ServiceConfiguration conf;
+
+    private final BrokerLookupData brokerLookupData;
+
+    private final LockManager<BrokerLookupData> brokerLookupDataLockManager;
+
+    private final String brokerId;
+
+    private final ScheduledExecutorService scheduler;
+
+    private final List<BiConsumer<String, NotificationType>> listeners;
+
+    private volatile ResourceLock<BrokerLookupData> brokerLookupDataLock;
+
+    protected enum State {
+        Init,
+        Started,
+        Registered,
+        Closed
+    }
+
+    private State state;
+
+    public BrokerRegistryImpl(PulsarService pulsar) {
+        this.pulsar = pulsar;
+        this.conf = pulsar.getConfiguration();
+        this.brokerLookupDataLockManager = pulsar.getCoordinationService().getLockManager(BrokerLookupData.class);
+        this.scheduler = pulsar.getLoadManagerExecutor();
+        this.listeners = new ArrayList<>();
+        this.brokerId = pulsar.getLookupServiceAddress();
+        this.brokerLookupData = new BrokerLookupData(
+                pulsar.getSafeWebServiceAddress(),
+                pulsar.getWebServiceAddressTls(),
+                pulsar.getBrokerServiceUrl(),
+                pulsar.getBrokerServiceUrlTls(),
+                pulsar.getAdvertisedListeners(),
+                pulsar.getProtocolDataToAdvertise(),
+                pulsar.getConfiguration().isEnablePersistentTopics(),
+                pulsar.getConfiguration().isEnableNonPersistentTopics(),
+                pulsar.getBrokerVersion());
+        this.state = State.Init;
+    }
+
+    @Override
+    public synchronized void start() throws PulsarServerException {
+        if (this.state != State.Init) {
+            return;
+        }
+        pulsar.getLocalMetadataStore().registerListener(this::handleMetadataStoreNotification);
+        try {
+            this.state = State.Started;
+            this.register();
+        } catch (MetadataStoreException e) {
+            throw new PulsarServerException(e);
+        }
+    }
+
+    @Override
+    public boolean isStarted() {
+        return this.state == State.Started || this.state == State.Registered;
+    }
+
+    @Override
+    public synchronized void register() throws MetadataStoreException {
+        if (this.state == State.Started) {
+            try {
+                this.brokerLookupDataLock = brokerLookupDataLockManager.acquireLock(keyPath(brokerId), brokerLookupData)
+                        .get(conf.getMetadataStoreOperationTimeoutSeconds(), TimeUnit.SECONDS);
+                this.state = State.Registered;
+            } catch (InterruptedException | ExecutionException | TimeoutException e) {
+                throw MetadataStoreException.unwrap(e);
+            }
+        }
+    }
+
+    @Override
+    public synchronized void unregister() throws MetadataStoreException {
+        if (this.state == State.Registered) {
+            try {
+                this.brokerLookupDataLock.release()
+                        .get(conf.getMetadataStoreOperationTimeoutSeconds(), TimeUnit.SECONDS);
+                this.state = State.Started;
+            } catch (CompletionException | InterruptedException | ExecutionException | TimeoutException e) {
+                throw MetadataStoreException.unwrap(e);
+            }
+        }
+    }
+
+    @Override
+    public String getBrokerId() {
+        return this.brokerId;
+    }
+
+    @Override
+    public CompletableFuture<List<String>> getAvailableBrokersAsync() {
+        if (!this.isStarted()) {
+            return FutureUtil.failedFuture(new IllegalStateException("The registry already closed."));
+        }
+        return brokerLookupDataLockManager.listLocks(LOOKUP_DATA_PATH).thenApply(Lists::newArrayList);
+    }
+
+    @Override
+    public CompletableFuture<Optional<BrokerLookupData>> lookupAsync(String broker) {
+        if (!this.isStarted()) {
+            return FutureUtil.failedFuture(new IllegalStateException("The registry already closed."));
+        }
+        return brokerLookupDataLockManager.readLock(keyPath(broker));
+    }
+
+    public CompletableFuture<Map<String, BrokerLookupData>> getAvailableBrokerLookupDataAsync() {
+        if (!this.isStarted()) {
+            return FutureUtil.failedFuture(new IllegalStateException("The registry already closed."));
+        }

Review Comment:
   Duplicate codes. Please add a private method named `checkState()` for line 168~170.
   And replace them for `lookupAsync`, `getAvailableBrokersAsync`



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: commits-unsubscribe@pulsar.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org