forked from apache/pulsar
-
Notifications
You must be signed in to change notification settings - Fork 0
Commit
This commit does not belong to any branch on this repository, and may belong to a fork outside of the repository.
Introduce auto bundle split and unloading of split bundle in ModularL…
…oadManager (apache#857)
- Loading branch information
Showing
24 changed files
with
479 additions
and
47 deletions.
There are no files selected for viewing
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
40 changes: 40 additions & 0 deletions
40
pulsar-broker/src/main/java/org/apache/pulsar/broker/loadbalance/BundleSplitStrategy.java
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
Original file line number | Diff line number | Diff line change |
---|---|---|
@@ -0,0 +1,40 @@ | ||
/** | ||
* 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; | ||
|
||
import java.util.Set; | ||
|
||
import org.apache.pulsar.broker.PulsarService; | ||
|
||
/** | ||
* Load Manager component which determines what bundles should be split into two bundles. | ||
*/ | ||
public interface BundleSplitStrategy { | ||
/** | ||
* Determines which bundles, if any, should be split. | ||
* | ||
* @param loadData | ||
* Load data to base decisions on (does not have benefit of preallocated data since this may not be the | ||
* leader broker). | ||
* @param pulsar | ||
* Service to use. | ||
* @return A set of the bundles that should be split. | ||
*/ | ||
Set<String> findBundlesToSplit(LoadData loadData, PulsarService pulsar); | ||
} |
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
111 changes: 111 additions & 0 deletions
111
...ar-broker/src/main/java/org/apache/pulsar/broker/loadbalance/impl/BundleSplitterTask.java
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
Original file line number | Diff line number | Diff line change |
---|---|---|
@@ -0,0 +1,111 @@ | ||
/** | ||
* 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.impl; | ||
|
||
import java.util.HashSet; | ||
import java.util.Map; | ||
import java.util.Set; | ||
|
||
import org.slf4j.Logger; | ||
import org.slf4j.LoggerFactory; | ||
|
||
import org.apache.pulsar.broker.LocalBrokerData; | ||
//import org.apache.pulsar.broker.MessageData; | ||
import org.apache.pulsar.broker.PulsarService; | ||
import org.apache.pulsar.broker.ServiceConfiguration; | ||
import org.apache.pulsar.broker.TimeAverageMessageData; | ||
import org.apache.pulsar.broker.loadbalance.BundleSplitStrategy; | ||
import org.apache.pulsar.broker.loadbalance.LoadData; | ||
import org.apache.pulsar.common.naming.NamespaceName; | ||
import org.apache.pulsar.policies.data.loadbalancer.NamespaceBundleStats; | ||
|
||
/** | ||
* Determines which bundles should be split based on various thresholds. | ||
*/ | ||
public class BundleSplitterTask implements BundleSplitStrategy { | ||
private static final Logger log = LoggerFactory.getLogger(BundleSplitStrategy.class); | ||
private final Set<String> bundleCache; | ||
|
||
/** | ||
* Construct a BundleSplitterTask. | ||
* | ||
* @param pulsar | ||
* Service to construct from. | ||
*/ | ||
public BundleSplitterTask(final PulsarService pulsar) { | ||
bundleCache = new HashSet<>(); | ||
} | ||
|
||
/** | ||
* Determines which bundles should be split based on various thresholds. | ||
* | ||
* @param loadData | ||
* Load data to base decisions on (does not have benefit of preallocated data since this may not be the | ||
* leader broker). | ||
* @param localData | ||
* Local data for the broker we are splitting on. | ||
* @param pulsar | ||
* Service to use. | ||
* @return All bundles who have exceeded configured thresholds in number of topics, number of sessions, total | ||
* message rates, or total throughput. | ||
*/ | ||
@Override | ||
public Set<String> findBundlesToSplit(final LoadData loadData, final PulsarService pulsar) { | ||
bundleCache.clear(); | ||
final ServiceConfiguration conf = pulsar.getConfiguration(); | ||
int maxBundleCount = conf.getLoadBalancerNamespaceMaximumBundles(); | ||
long maxBundleTopics = conf.getLoadBalancerNamespaceBundleMaxTopics(); | ||
long maxBundleSessions = conf.getLoadBalancerNamespaceBundleMaxSessions(); | ||
long maxBundleMsgRate = conf.getLoadBalancerNamespaceBundleMaxMsgRate(); | ||
long maxBundleBandwidth = conf.getLoadBalancerNamespaceBundleMaxBandwidthMbytes() * LoadManagerShared.MIBI; | ||
loadData.getBrokerData().forEach((broker, brokerData) -> { | ||
LocalBrokerData localData = brokerData.getLocalData(); | ||
for (final Map.Entry<String, NamespaceBundleStats> entry : localData.getLastStats().entrySet()) { | ||
final String bundle = entry.getKey(); | ||
final NamespaceBundleStats stats = entry.getValue(); | ||
double totalMessageRate = 0; | ||
double totalMessageThroughput = 0; | ||
// Attempt to consider long-term message data, otherwise effectively ignore. | ||
if (loadData.getBundleData().containsKey(bundle)) { | ||
final TimeAverageMessageData longTermData = loadData.getBundleData().get(bundle).getLongTermData(); | ||
totalMessageRate = longTermData.totalMsgRate(); | ||
totalMessageThroughput = longTermData.totalMsgThroughput(); | ||
} | ||
if (stats.topics > maxBundleTopics || stats.consumerCount + stats.producerCount > maxBundleSessions | ||
|| totalMessageRate > maxBundleMsgRate || totalMessageThroughput > maxBundleBandwidth) { | ||
final String namespace = LoadManagerShared.getNamespaceNameFromBundleName(bundle); | ||
try { | ||
final int bundleCount = pulsar.getNamespaceService() | ||
.getBundleCount(new NamespaceName(namespace)); | ||
if (bundleCount < maxBundleCount) { | ||
bundleCache.add(bundle); | ||
} else { | ||
log.warn( | ||
"Could not split namespace bundle {} because namespace {} has too many bundles: {}", | ||
bundle, namespace, bundleCount); | ||
} | ||
} catch (Exception e) { | ||
log.warn("Error while getting bundle count for namespace {}", namespace, e); | ||
} | ||
} | ||
} | ||
}); | ||
return bundleCache; | ||
} | ||
} |
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
Oops, something went wrong.