Skip to content

Commit

Permalink
Add ability to filter brokers based on their build version (apache#397)
Browse files Browse the repository at this point in the history
  • Loading branch information
Brad McMillen authored and merlimat committed May 14, 2017
1 parent 1c12849 commit ca5a445
Show file tree
Hide file tree
Showing 17 changed files with 513 additions and 14 deletions.
4 changes: 4 additions & 0 deletions conf/broker.conf
Original file line number Diff line number Diff line change
Expand Up @@ -80,6 +80,10 @@ clientLibraryVersionCheckAllowUnversioned=true
# to service discovery health checks
statusFilePath=

# If true, (and ModularLoadManagerImpl is being used), the load manager will attempt to
* use only brokers running the latest software version (to minimize impact to bundles)
preferLaterVersions=false;

### --- Authentication --- ###

# Enable TLS
Expand Down
6 changes: 6 additions & 0 deletions pom.xml
Original file line number Diff line number Diff line change
Expand Up @@ -405,6 +405,12 @@ flexible messaging model and an intuitive client API.</description>
<version>${athenz.version}</version>
</dependency>

<dependency>
<groupId>com.github.zafarkhaja</groupId>
<artifactId>java-semver</artifactId>
<version>0.9.0</version>
</dependency>

<dependency>
<groupId>org.apache.spark</groupId>
<artifactId>spark-streaming_2.10</artifactId>
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -264,6 +264,10 @@ public class ServiceConfiguration implements PulsarConfiguration {
// Name of load manager to use
@FieldContext(dynamic = true)
private String loadManagerClassName = "com.yahoo.pulsar.broker.loadbalance.impl.SimpleLoadManagerImpl";
// If true, (and ModularLoadManagerImpl is being used), the load manager will attempt to
// use only brokers running the latest software version (to minimize impact to bundles)
@FieldContext(dynamic = true)
private boolean preferLaterVersions = false;

public String getZookeeperServers() {
return zookeeperServers;
Expand Down Expand Up @@ -979,4 +983,12 @@ public String getLoadManagerClassName() {
public void setLoadManagerClassName(String loadManagerClassName) {
this.loadManagerClassName = loadManagerClassName;
}

public boolean isPreferLaterVersions() {
return preferLaterVersions;
}

public void setPreferLaterVersions(boolean preferLaterVersions) {
this.preferLaterVersions = preferLaterVersions;
}
}
12 changes: 11 additions & 1 deletion pulsar-broker/pom.xml
Original file line number Diff line number Diff line change
Expand Up @@ -31,7 +31,6 @@
<name>Pulsar Broker</name>

<dependencies>

<dependency>
<groupId>commons-codec</groupId>
<artifactId>commons-codec</artifactId>
Expand Down Expand Up @@ -198,6 +197,11 @@
<artifactId>gson</artifactId>
</dependency>

<dependency>
<groupId>com.github.zafarkhaja</groupId>
<artifactId>java-semver</artifactId>
</dependency>

<dependency>
<groupId>org.mockito</groupId>
<artifactId>mockito-core</artifactId>
Expand Down Expand Up @@ -266,5 +270,11 @@
</executions>
</plugin>
</plugins>
<resources>
<resource>
<directory>src/main/resources</directory>
<filtering>true</filtering>
</resource>
</resources>
</build>
</project>
Original file line number Diff line number Diff line change
Expand Up @@ -71,6 +71,9 @@ public class LocalBrokerData extends JSONWritable implements ServiceLookupData {
// The bundles lost since the last invocation of update.
private Set<String> lastBundleLosses;

// The version string that this broker is running, obtained from the Maven build artifact in the POM
private String brokerVersionString;

// For JSON only.
public LocalBrokerData() {
this(null, null, null, null);
Expand Down Expand Up @@ -337,6 +340,14 @@ public void setMsgRateOut(double msgRateOut) {
this.msgRateOut = msgRateOut;
}

public void setBrokerVersionString(String brokerVersionString) {
this.brokerVersionString = brokerVersionString;
}

public String getBrokerVersionString() {
return brokerVersionString;
}

@Override
public String getWebServiceUrl() {
return webServiceUrl;
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -28,6 +28,7 @@
import java.util.concurrent.locks.ReentrantLock;
import java.util.function.Supplier;

import com.yahoo.pulsar.utils.PulsarBrokerVersionStringUtils;
import org.apache.bookkeeper.mledger.ManagedLedgerFactory;
import org.apache.bookkeeper.util.OrderedSafeExecutor;
import org.apache.bookkeeper.util.ZkUtils;
Expand Down Expand Up @@ -111,6 +112,7 @@ public class PulsarService implements AutoCloseable {
private final String webServiceAddressTls;
private final String brokerServiceUrl;
private final String brokerServiceUrlTls;
private final String brokerVersion;

private final MessagingServiceShutdownHook shutdownService;

Expand All @@ -133,6 +135,7 @@ public PulsarService(ServiceConfiguration config) {
this.webServiceAddressTls = webAddressTls(config);
this.brokerServiceUrl = brokerUrl(config);
this.brokerServiceUrlTls = brokerUrlTls(config);
this.brokerVersion = PulsarBrokerVersionStringUtils.getNormalizedVersionString();
this.config = config;
this.shutdownService = new MessagingServiceShutdownHook(this);
loadManagerExecutor = Executors.newSingleThreadScheduledExecutor();
Expand Down Expand Up @@ -256,7 +259,7 @@ public void start() throws PulsarServerException {
// needs load management service
this.startNamespaceService();

LOG.info("Starting Pulsar Broker service");
LOG.info("Starting Pulsar Broker service; version: '{}'", ( brokerVersion != null ? brokerVersion : "unknown" ) );
brokerService.start();

this.webService = new WebService(this);
Expand Down Expand Up @@ -648,4 +651,7 @@ public AtomicReference<LoadManager> getLoadManager() {
return loadManager;
}

public String getBrokerVersion() {
return brokerVersion;
}
}
Original file line number Diff line number Diff line change
Expand Up @@ -18,6 +18,7 @@
import java.util.Set;

import com.yahoo.pulsar.broker.BundleData;
import com.yahoo.pulsar.broker.PulsarService;
import com.yahoo.pulsar.broker.ServiceConfiguration;

/**
Expand All @@ -40,6 +41,9 @@ public interface BrokerFilter {
* The load data from the leader broker.
* @param conf
* The service configuration.
* @throws BrokerFilterException
* There was an error in the pipeline and the brokers should be reset to their original value
*/
void filter(Set<String> brokers, BundleData bundleToAssign, LoadData loadData, ServiceConfiguration conf);
}
public void filter(Set<String> brokers, BundleData bundleToAssign, LoadData loadData, ServiceConfiguration conf)
throws BrokerFilterException;
}
Original file line number Diff line number Diff line change
@@ -0,0 +1,30 @@
/**
* Copyright 2016 Yahoo Inc.
*
* Licensed 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 com.yahoo.pulsar.broker.loadbalance;

public class BrokerFilterBadVersionException extends BrokerFilterException {
public BrokerFilterBadVersionException(String msg) {
super(msg);
}

public BrokerFilterBadVersionException(Throwable t) {
super(t);
}

public BrokerFilterBadVersionException(String msg, Throwable t) {
super(msg, t);
}
}
Original file line number Diff line number Diff line change
@@ -0,0 +1,30 @@
/**
* Copyright 2016 Yahoo Inc.
*
* Licensed 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 com.yahoo.pulsar.broker.loadbalance;

public class BrokerFilterException extends Exception {
public BrokerFilterException(String msg) {
super(msg);
}

public BrokerFilterException(Throwable t) {
super(t);
}

public BrokerFilterException(String msg, Throwable t) {
super(msg, t);
}
}
Original file line number Diff line number Diff line change
@@ -0,0 +1,152 @@
/**
* Copyright 2016 Yahoo Inc.
*
* Licensed 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 com.yahoo.pulsar.broker.loadbalance.impl;

import java.util.*;
import java.util.regex.Matcher;
import java.util.regex.Pattern;

import com.yahoo.pulsar.broker.PulsarService;
import com.yahoo.pulsar.broker.loadbalance.BrokerFilterBadVersionException;
import org.apache.bookkeeper.mledger.ManagedLedgerException;
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;
import com.github.zafarkhaja.semver.Version;

import com.yahoo.pulsar.broker.BrokerData;
import com.yahoo.pulsar.broker.loadbalance.BrokerFilter;
import com.yahoo.pulsar.broker.BundleData;
import com.yahoo.pulsar.broker.ServiceConfiguration;
import com.yahoo.pulsar.broker.loadbalance.LoadData;

public class BrokerVersionFilter implements BrokerFilter {

private static final Logger LOG = LoggerFactory.getLogger(BrokerVersionFilter.class);

/**
* Get the most recent broker version number from the load reports of all the running brokers. The version
* number is from the build artifact in the pom and got added to the package when it was built by Maven
*
* @param brokers
* The brokers to choose the latest version string from.
* @param loadData
* The load data from the leader broker (contains the load reports which in turn contain the version string).
* @return The most recent broker version
* @throws BrokerFilterBadVersionException
* If the most recent version is undefined (e.g., a bad broker version was encountered or a broker
* does not have a version string in its load report.
*/
public Version getLatestVersionNumber(Set<String> brokers, LoadData loadData) throws BrokerFilterBadVersionException {
if ( null == brokers ) {
throw new BrokerFilterBadVersionException("Unable to determine latest version since broker set was null");
}
if ( brokers.size() == 0 ) {
throw new BrokerFilterBadVersionException("Unable to determine latest version since broker set was empty");
}
if ( null == loadData ) {
throw new BrokerFilterBadVersionException("Unable to determine latest version since loadData was null");
}

Version latestVersion = null;
for ( String broker : brokers ) {
BrokerData data = loadData.getBrokerData().get(broker);
if (null == data) {
LOG.warn("No broker data for broker [{}]; disabling PreferLaterVersions feature", broker);
// trigger the ModularLoadManager to reset all the brokers to the original set
throw new BrokerFilterBadVersionException("No broker data for broker \"" + broker + "\"");
}

String brokerVersion = data.getLocalData().getBrokerVersionString();
if (null == brokerVersion || brokerVersion.length() == 0) {
LOG.warn("No version string in load report for broker [{}]; disabling PreferLaterVersions feature", broker);
// trigger the ModularLoadManager to reset all the brokers to the original set
throw new BrokerFilterBadVersionException("No version string in load report for broker \"" + broker + "\"");
}

Version brokerVersionVersion = null;
try {
brokerVersionVersion = Version.valueOf(brokerVersion);
} catch (Exception x) {
LOG.warn("Invalid version string in load report for broker [{}]: [{}]; disabling PreferLaterVersions feature", broker, brokerVersion);
// trigger the ModularLoadManager to reset all the brokers to the original set
throw new BrokerFilterBadVersionException("Invalid version string in load report for broker \"" + broker + "\": \"" + brokerVersion + "\")");
}

if ( null == latestVersion ) {
latestVersion = brokerVersionVersion;
} else if (Version.BUILD_AWARE_ORDER.compare(latestVersion, brokerVersionVersion) < 0) {
latestVersion = brokerVersionVersion;
}
}

if ( null == latestVersion ) {
throw new BrokerFilterBadVersionException("Unable to determine latest broker version");
}

return latestVersion;
}

/**
* From the given set of available broker candidates, filter those using the version numbers.
*
* @param brokers
* The currently available brokers that have not already been filtered.
* @param bundleToAssign
* The data for the bundle to assign.
* @param loadData
* The load data from the leader broker.
* @param conf
* The service configuration.
*/
public void filter(Set<String> brokers, BundleData bundleToAssign, LoadData loadData, ServiceConfiguration conf)
throws BrokerFilterBadVersionException {

if ( !conf.isPreferLaterVersions()) {
return;
}

com.github.zafarkhaja.semver.Version latestVersion = null;
try {
latestVersion = getLatestVersionNumber(brokers, loadData);
LOG.info("Latest broker version found was [{}]", latestVersion);
} catch ( Exception x ) {
LOG.warn("Disabling PreferLaterVersions feature; reason: " + x.getMessage());
throw new BrokerFilterBadVersionException("Cannot determine newest broker version: " + x.getMessage());
}

int numBrokersLatestVersion=0;
int numBrokersOlderVersion=0;
Iterator<String> brokerIterator = brokers.iterator();
while ( brokerIterator.hasNext() ) {
String broker = brokerIterator.next();
BrokerData data = loadData.getBrokerData().get(broker);
String brokerVersion = data.getLocalData().getBrokerVersionString();
com.github.zafarkhaja.semver.Version brokerVersionVersion = Version.valueOf(brokerVersion);

if ( brokerVersionVersion.equals(latestVersion) ) {
LOG.debug("Broker [{}] is running the latest version ([{}])", broker, brokerVersion);
++numBrokersLatestVersion;
} else {
LOG.info("Broker [{}] is running an older version ([{}]); latest version is [{}]", broker, brokerVersion, latestVersion);
++numBrokersOlderVersion;
brokerIterator.remove();
}
}
if ( numBrokersOlderVersion == 0 ) {
LOG.info("All {} brokers are running the latest version [{}]", numBrokersLatestVersion, latestVersion);
}
}
}
Loading

0 comments on commit ca5a445

Please sign in to comment.