Skip to content

Commit

Permalink
[CLI] Add a broker tool for operations of a specific broker (apache#5768
Browse files Browse the repository at this point in the history
)

*Motivation*

Add a tool to collect load report on a specific broker.
This is used for troubleshooting problems in a production cluster.

*Modifications*

Add a broker tool for operations of a specific broker.
  • Loading branch information
sijie authored Nov 30, 2019
1 parent e4fc609 commit 035743f
Show file tree
Hide file tree
Showing 9 changed files with 252 additions and 28 deletions.
3 changes: 3 additions & 0 deletions bin/pulsar
Original file line number Diff line number Diff line change
Expand Up @@ -144,6 +144,7 @@ where command is one of:
initialize-transaction-coordinator-metadata One-time transaction coordinator metadata initialization
compact-topic Run compaction against a topic
zookeeper-shell Open a ZK shell client
broker-tool CLI to operate a specific broker
tokens Utility to create authentication tokens
help This help message
Expand Down Expand Up @@ -339,6 +340,8 @@ elif [ $COMMAND == "initialize-transaction-coordinator-metadata" ]; then
exec $JAVA $OPTS org.apache.pulsar.PulsarTransactionCoordinatorMetadataSetup $@
elif [ $COMMAND == "zookeeper-shell" ]; then
exec $JAVA $OPTS org.apache.zookeeper.ZooKeeperMain $@
elif [ $COMMAND == "broker-tool" ]; then
exec $JAVA $OPTS org.apache.pulsar.broker.tools.BrokerTool $@
elif [ $COMMAND == "compact-topic" ]; then
exec $JAVA $OPTS org.apache.pulsar.compaction.CompactorTool --broker-conf $PULSAR_BROKER_CONF $@
elif [ $COMMAND == "sql" ]; then
Expand Down
6 changes: 6 additions & 0 deletions pom.xml
Original file line number Diff line number Diff line change
Expand Up @@ -333,6 +333,12 @@ flexible messaging model and an intuitive client API.</description>
<version>${bookkeeper.version}</version>
</dependency>

<dependency>
<groupId>org.apache.bookkeeper</groupId>
<artifactId>bookkeeper-tools-framework</artifactId>
<version>${bookkeeper.version}</version>
</dependency>

<!-- reflection libs -->
<dependency>
<groupId>org.reflections</groupId>
Expand Down
5 changes: 5 additions & 0 deletions pulsar-broker/pom.xml
Original file line number Diff line number Diff line change
Expand Up @@ -102,6 +102,11 @@
<version>${project.version}</version>
</dependency>

<dependency>
<groupId>org.apache.bookkeeper</groupId>
<artifactId>bookkeeper-tools-framework</artifactId>
</dependency>

<dependency>
<groupId>${project.groupId}</groupId>
<artifactId>pulsar-broker-common</artifactId>
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -18,20 +18,21 @@
*/
package org.apache.pulsar.broker.loadbalance;

import java.io.IOException;

import org.apache.pulsar.policies.data.loadbalancer.SystemResourceUsage;

/**
* Class that will return the broker host usage.
*
*
*/
public interface BrokerHostUsage {
/**
* Returns the host usage information in the following format -
* Returns the host usage information.
*
* @return Broker host usage in the json string format
*/
SystemResourceUsage getBrokerHostUsage();

/**
* Calculate the host usage information.
*/
void calculateBrokerHostUsage();
}
Original file line number Diff line number Diff line change
Expand Up @@ -20,40 +20,40 @@

import com.sun.management.OperatingSystemMXBean;

import java.lang.management.ManagementFactory;
import java.util.concurrent.ScheduledExecutorService;
import java.util.concurrent.TimeUnit;
import org.apache.pulsar.broker.PulsarService;
import org.apache.pulsar.broker.loadbalance.BrokerHostUsage;
import org.apache.pulsar.policies.data.loadbalancer.ResourceUsage;
import org.apache.pulsar.policies.data.loadbalancer.SystemResourceUsage;
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;

import java.io.IOException;
import java.lang.management.ManagementFactory;
import java.util.concurrent.TimeUnit;

/**
* Class that will return the broker host usage.
*/
public class GenericBrokerHostUsageImpl implements BrokerHostUsage {
// The interval for host usage check command
private static final int CPU_CHECK_MILLIS = 1000;
private static final Logger LOG = LoggerFactory.getLogger(GenericBrokerHostUsageImpl.class);
private final int hostUsageCheckIntervalMin;
private long lastCollection;
private double totalCpuLimit;
private double cpuUsageSum = 0d;
private int cpuUsageCount = 0;
private OperatingSystemMXBean systemBean;
private SystemResourceUsage usage;

public GenericBrokerHostUsageImpl(PulsarService pulsar) {
this.hostUsageCheckIntervalMin = pulsar.getConfiguration().getLoadBalancerHostUsageCheckIntervalMinutes();
this(
pulsar.getConfiguration().getLoadBalancerHostUsageCheckIntervalMinutes(),
pulsar.getLoadManagerExecutor()
);
}

public GenericBrokerHostUsageImpl(int hostUsageCheckIntervalMin,
ScheduledExecutorService executorService) {
this.systemBean = (OperatingSystemMXBean) ManagementFactory.getOperatingSystemMXBean();
this.lastCollection = 0L;
this.usage = new SystemResourceUsage();
this.totalCpuLimit = getTotalCpuLimit();
pulsar.getLoadManagerExecutor().scheduleAtFixedRate(this::checkCpuLoad, 0, CPU_CHECK_MILLIS, TimeUnit.MILLISECONDS);
pulsar.getLoadManagerExecutor().scheduleAtFixedRate(this::calculateBrokerHostUsage, 0, hostUsageCheckIntervalMin, TimeUnit.MINUTES);
executorService.scheduleAtFixedRate(this::checkCpuLoad, 0, CPU_CHECK_MILLIS, TimeUnit.MILLISECONDS);
executorService.scheduleAtFixedRate(this::doCalculateBrokerHostUsage, 0, hostUsageCheckIntervalMin, TimeUnit.MINUTES);
}

@Override
Expand All @@ -66,7 +66,13 @@ private void checkCpuLoad() {
cpuUsageCount++;
}

private void calculateBrokerHostUsage() {
@Override
public void calculateBrokerHostUsage() {
checkCpuLoad();
doCalculateBrokerHostUsage();
}

void doCalculateBrokerHostUsage() {
SystemResourceUsage usage = new SystemResourceUsage();
usage.setCpu(getCpuUsage());
usage.setMemory(getMemUsage());
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -18,6 +18,7 @@
*/
package org.apache.pulsar.broker.loadbalance.impl;

import com.sun.management.OperatingSystemMXBean;
import java.io.IOException;
import java.lang.management.ManagementFactory;
import java.nio.file.Files;
Expand All @@ -27,6 +28,7 @@
import java.util.Collections;
import java.util.List;
import java.util.Optional;
import java.util.concurrent.ScheduledExecutorService;
import java.util.concurrent.TimeUnit;
import java.util.stream.Collectors;
import java.util.stream.Stream;
Expand All @@ -38,16 +40,11 @@
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;

import com.sun.management.OperatingSystemMXBean;

/**
* Class that will return the broker host usage.
*
*
*/
public class LinuxBrokerHostUsageImpl implements BrokerHostUsage {
// The interval for host usage check command
private final int hostUsageCheckIntervalMin;
private long lastCollection;
private double lastTotalNicUsageTx;
private double lastTotalNicUsageRx;
Expand All @@ -60,12 +57,21 @@ public class LinuxBrokerHostUsageImpl implements BrokerHostUsage {
private static final Logger LOG = LoggerFactory.getLogger(LinuxBrokerHostUsageImpl.class);

public LinuxBrokerHostUsageImpl(PulsarService pulsar) {
this.hostUsageCheckIntervalMin = pulsar.getConfiguration().getLoadBalancerHostUsageCheckIntervalMinutes();
this(
pulsar.getConfiguration().getLoadBalancerHostUsageCheckIntervalMinutes(),
pulsar.getConfiguration().getLoadBalancerOverrideBrokerNicSpeedGbps(),
pulsar.getLoadManagerExecutor()
);
}

public LinuxBrokerHostUsageImpl(int hostUsageCheckIntervalMin,
Optional<Double> overrideBrokerNicSpeedGbps,
ScheduledExecutorService executorService) {
this.systemBean = (OperatingSystemMXBean) ManagementFactory.getOperatingSystemMXBean();
this.lastCollection = 0L;
this.usage = new SystemResourceUsage();
this.overrideBrokerNicSpeedGbps = pulsar.getConfiguration().getLoadBalancerOverrideBrokerNicSpeedGbps();
pulsar.getLoadManagerExecutor().scheduleAtFixedRate(this::calculateBrokerHostUsage, 0,
this.overrideBrokerNicSpeedGbps = overrideBrokerNicSpeedGbps;
executorService.scheduleAtFixedRate(this::calculateBrokerHostUsage, 0,
hostUsageCheckIntervalMin, TimeUnit.MINUTES);
}

Expand All @@ -74,7 +80,8 @@ public SystemResourceUsage getBrokerHostUsage() {
return usage;
}

private void calculateBrokerHostUsage() {
@Override
public void calculateBrokerHostUsage() {
List<String> nics = getNics();
double totalNicLimit = getTotalNicLimitKbps(nics);
double totalNicUsageTx = getTotalNicUsageTxKb(nics);
Expand Down
Original file line number Diff line number Diff line change
@@ -0,0 +1,47 @@
/**
* 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.tools;

import org.apache.bookkeeper.tools.framework.Cli;
import org.apache.bookkeeper.tools.framework.CliFlags;
import org.apache.bookkeeper.tools.framework.CliSpec;

/**
* <b>broker-tool</b> is used for operations on a specific broker.
*/
public class BrokerTool {

public static final String NAME = "broker-tool";

public static void main(String[] args) {
CliSpec.Builder<CliFlags> specBuilder = CliSpec.newBuilder()
.withName(NAME)
.withUsage(NAME + " [flags] [commands]")
.withDescription(NAME + " is used for operations on a specific broker")
.withFlags(new CliFlags())
.withConsole(System.out)
.addCommand(new LoadReportCommand());

CliSpec<CliFlags> spec = specBuilder.build();

int retCode = Cli.runCli(spec, args);
Runtime.getRuntime().exit(retCode);
}

}
Original file line number Diff line number Diff line change
@@ -0,0 +1,127 @@
/**
* 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.tools;

import com.beust.jcommander.Parameter;
import java.util.Optional;
import java.util.concurrent.Executors;
import java.util.concurrent.ScheduledExecutorService;
import java.util.concurrent.TimeUnit;
import org.apache.bookkeeper.tools.framework.Cli;
import org.apache.bookkeeper.tools.framework.CliCommand;
import org.apache.bookkeeper.tools.framework.CliFlags;
import org.apache.bookkeeper.tools.framework.CliSpec;
import org.apache.commons.lang3.SystemUtils;
import org.apache.pulsar.broker.loadbalance.BrokerHostUsage;
import org.apache.pulsar.broker.loadbalance.impl.GenericBrokerHostUsageImpl;
import org.apache.pulsar.broker.loadbalance.impl.LinuxBrokerHostUsageImpl;
import org.apache.pulsar.broker.tools.LoadReportCommand.Flags;
import org.apache.pulsar.policies.data.loadbalancer.ResourceUsage;
import org.apache.pulsar.policies.data.loadbalancer.SystemResourceUsage;

/**
* The command to collect the load report of a specific broker.
*/
public class LoadReportCommand extends CliCommand<CliFlags, Flags> {

private static final String NAME = "load-report";
private static final String DESC = "Collect the load report of a specific broker";

/**
* The CLI flags of load report command.
*/
public static class Flags extends CliFlags {

@Parameter(
names = {
"-i", "--interval-ms"
},
description = "Interval to collect load report, in milliseconds"
)
public int intervalMilliseconds = 100;

}

public LoadReportCommand() {
super(CliSpec.<Flags>newBuilder()
.withName(NAME)
.withDescription(DESC)
.withFlags(new Flags())
.build());
}

@Override
public Boolean apply(CliFlags globalFlags, String[] args) {
CliSpec<Flags> newSpec = CliSpec.newBuilder(spec)
.withRunFunc(cmdFlags -> apply(cmdFlags))
.build();
return 0 == Cli.runCli(newSpec, args);
}

private boolean apply(Flags flags) {

boolean isLinux = SystemUtils.IS_OS_LINUX;
spec.console().println("OS ARCH: " + SystemUtils.OS_ARCH);
spec.console().println("OS NAME: " + SystemUtils.OS_NAME);
spec.console().println("OS VERSION: " + SystemUtils.OS_VERSION);
spec.console().println("Linux: " + isLinux);
spec.console().println("--------------------------------------");
spec.console().println();
spec.console().println("Load Report Interval : " + flags.intervalMilliseconds + " ms");
spec.console().println();
spec.console().println("--------------------------------------");
spec.console().println();

ScheduledExecutorService scheduler = Executors.newSingleThreadScheduledExecutor();
BrokerHostUsage hostUsage;
try {
if (isLinux) {
hostUsage = new LinuxBrokerHostUsageImpl(
Integer.MAX_VALUE, Optional.empty(), scheduler
);
} else {
hostUsage = new GenericBrokerHostUsageImpl(
Integer.MAX_VALUE, scheduler
);
}

hostUsage.calculateBrokerHostUsage();
try {
TimeUnit.MILLISECONDS.sleep(flags.intervalMilliseconds);
} catch (InterruptedException e) {
}
hostUsage.calculateBrokerHostUsage();
SystemResourceUsage usage = hostUsage.getBrokerHostUsage();

printResourceUsage("CPU", usage.cpu);
printResourceUsage("Memory", usage.memory);
printResourceUsage("Direct Memory", usage.directMemory);
printResourceUsage("Bandwidth In", usage.bandwidthIn);
printResourceUsage("Bandwidth Out", usage.bandwidthOut);

return true;
} finally {
scheduler.shutdown();
}
}

private void printResourceUsage(String name, ResourceUsage usage) {
spec.console().println(name + " : usage = " + usage.usage + ", limit = " + usage.limit);
}
}
Loading

0 comments on commit 035743f

Please sign in to comment.