Skip to content

Commit

Permalink
[METRICS] Migrate topic collector to balancer, partitioiner (#1765)
Browse files Browse the repository at this point in the history
  • Loading branch information
chinghongfang authored May 23, 2023
1 parent 9613d6b commit 28ec2c6
Show file tree
Hide file tree
Showing 5 changed files with 257 additions and 40 deletions.
71 changes: 49 additions & 22 deletions app/src/main/java/org/astraea/app/web/WebService.java
Original file line number Diff line number Diff line change
Expand Up @@ -24,24 +24,31 @@
import java.util.Collection;
import java.util.List;
import java.util.Map;
import java.util.concurrent.CompletionStage;
import java.util.concurrent.CompletableFuture;
import java.util.concurrent.ConcurrentLinkedQueue;
import java.util.concurrent.TimeUnit;
import java.util.function.BiConsumer;
import java.util.function.Function;
import java.util.function.Supplier;
import java.util.stream.Collectors;
import org.astraea.app.argument.DurationField;
import org.astraea.app.argument.IntegerMapField;
import org.astraea.app.argument.NonNegativeIntegerField;
import org.astraea.common.Configuration;
import org.astraea.common.Utils;
import org.astraea.common.admin.Admin;
import org.astraea.common.admin.Broker;
import org.astraea.common.admin.NodeInfo;
import org.astraea.common.metrics.JndiClient;
import org.astraea.common.metrics.MBeanClient;
import org.astraea.common.metrics.collector.MetricSensor;
import org.astraea.common.metrics.collector.MetricStore;

public class WebService implements AutoCloseable {
public static final String METRIC_STORE_KEY = "metric.store";
public static final String METRIC_STORE_LOCAL = "local";
public static final String METRIC_STORE_TOPIC = "topic";
public static final String BOOTSTRAP_SERVERS_KEY = "bootstrap.servers";

private final HttpServer server;
private final Admin admin;
Expand All @@ -51,32 +58,48 @@ public WebService(
Admin admin,
int port,
Function<Integer, Integer> brokerIdToJmxPort,
Duration beanExpiration) {
Duration beanExpiration,
Configuration config) {
this.admin = admin;
Supplier<CompletionStage<Map<Integer, MBeanClient>>> clientSupplier =
Supplier<Map<MetricSensor, BiConsumer<Integer, Exception>>> sensorsSupplier =
() ->
admin
.brokers()
.thenApply(
brokers ->
brokers.stream()
.collect(
Collectors.toUnmodifiableMap(
NodeInfo::id,
b ->
JndiClient.of(b.host(), brokerIdToJmxPort.apply(b.id())))));
sensors.metricSensors().stream()
.distinct()
.collect(
Collectors.toUnmodifiableMap(Function.identity(), ignored -> (id, ee) -> {}));

List<MetricStore.Receiver> receivers =
switch (config.string(METRIC_STORE_KEY).orElse(METRIC_STORE_LOCAL)) {
case METRIC_STORE_LOCAL -> {
Function<List<Broker>, Map<Integer, MBeanClient>> asBeanClientMap =
brokers ->
brokers.stream()
.collect(
Collectors.toUnmodifiableMap(
NodeInfo::id,
b -> JndiClient.of(b.host(), brokerIdToJmxPort.apply(b.id()))));
yield List.of(
MetricStore.Receiver.local(() -> admin.brokers().thenApply(asBeanClientMap)));
}
case METRIC_STORE_TOPIC -> List.of(
MetricStore.Receiver.topic(config.requireString(BOOTSTRAP_SERVERS_KEY)),
MetricStore.Receiver.local(
() -> CompletableFuture.completedStage(Map.of(-1, JndiClient.local()))));
default -> throw new IllegalArgumentException(
"unknown metric store type: "
+ config.string(METRIC_STORE_KEY)
+ ". use "
+ METRIC_STORE_LOCAL
+ " or "
+ METRIC_STORE_TOPIC);
};
var metricStore =
MetricStore.builder()
.beanExpiration(beanExpiration)
.receivers(List.of(MetricStore.Receiver.local(clientSupplier)))
.sensorsSupplier(
() ->
sensors.metricSensors().stream()
.distinct()
.collect(
Collectors.toUnmodifiableMap(
Function.identity(), ignored -> (id, ee) -> {})))
.receivers(receivers)
.sensorsSupplier(sensorsSupplier)
.build();

server = Utils.packException(() -> HttpServer.create(new InetSocketAddress(port), 0));
server.createContext("/topics", to(new TopicHandler(admin)));
server.createContext("/groups", to(new GroupHandler(admin)));
Expand Down Expand Up @@ -109,7 +132,11 @@ public static void main(String[] args) throws Exception {
throw new IllegalArgumentException("you must define either --jmx.port or --jmx.ports");
try (var service =
new WebService(
Admin.of(arg.configs()), arg.port, arg::jmxPortMapping, arg.beanExpiration)) {
Admin.of(arg.configs()),
arg.port,
arg::jmxPortMapping,
arg.beanExpiration,
new Configuration(arg.configs()))) {
if (arg.ttl == null) {
System.out.println("enter ctrl + c to terminate web service");
TimeUnit.MILLISECONDS.sleep(Long.MAX_VALUE);
Expand Down
4 changes: 3 additions & 1 deletion app/src/test/java/org/astraea/app/web/TopicHandlerTest.java
Original file line number Diff line number Diff line change
Expand Up @@ -27,6 +27,7 @@
import java.util.concurrent.ThreadLocalRandom;
import java.util.stream.Collectors;
import java.util.stream.IntStream;
import org.astraea.common.Configuration;
import org.astraea.common.Utils;
import org.astraea.common.admin.Admin;
import org.astraea.common.admin.TopicPartition;
Expand Down Expand Up @@ -69,7 +70,8 @@ void testWithWebService() {
Admin.of(SERVICE.bootstrapServers()),
0,
id -> SERVICE.jmxServiceURL().getPort(),
Duration.ofMillis(5))) {
Duration.ofMillis(5),
Configuration.EMPTY)) {
Response<TopicHandler.Topics> response =
HttpExecutor.builder()
.build()
Expand Down
86 changes: 85 additions & 1 deletion app/src/test/java/org/astraea/app/web/WebServiceTest.java
Original file line number Diff line number Diff line change
Expand Up @@ -17,13 +17,18 @@
package org.astraea.app.web;

import java.time.Duration;
import java.util.Map;
import java.util.concurrent.ThreadLocalRandom;
import java.util.concurrent.atomic.AtomicInteger;
import org.astraea.app.argument.Argument;
import org.astraea.common.Configuration;
import org.astraea.common.admin.Admin;
import org.astraea.common.metrics.collector.MetricStore;
import org.junit.jupiter.api.Assertions;
import org.junit.jupiter.api.Test;
import org.junit.jupiter.api.Timeout;
import org.mockito.Mockito;
import org.mockito.stubbing.Answer;

public class WebServiceTest {

Expand All @@ -40,7 +45,9 @@ void testArgument() {
@Timeout(10)
@Test
void testClose() {
var web = new WebService(Mockito.mock(Admin.class), 0, id -> -1, Duration.ofMillis(5));
var web =
new WebService(
Mockito.mock(Admin.class), 0, id -> -1, Duration.ofMillis(5), Configuration.EMPTY);
web.close();
}

Expand Down Expand Up @@ -84,4 +91,81 @@ void testJmxPort() {
Assertions.assertThrows(
IllegalArgumentException.class, () -> noDefaultArgument.jmxPortMapping(4));
}

@Test
void testMetricStoreConfiguration() {
try (var mockedReceiver = Mockito.mockStatic(MetricStore.Receiver.class)) {
var topicReceiverCount = new AtomicInteger(0);
var localReceiverCount = new AtomicInteger(0);
mockedReceiver
.when(() -> MetricStore.Receiver.topic(Mockito.any()))
.then(
(Answer<MetricStore.Receiver>)
invocation -> {
topicReceiverCount.incrementAndGet();
return Mockito.mock(MetricStore.Receiver.class);
});
mockedReceiver
.when(() -> MetricStore.Receiver.local(Mockito.any()))
.then(
(Answer<MetricStore.Receiver>)
invocation -> {
localReceiverCount.incrementAndGet();
return Mockito.mock(MetricStore.Receiver.class);
});
// Test default metric store configuration
try (var web =
new WebService(
Mockito.mock(Admin.class), 0, id -> -1, Duration.ofMillis(5), Configuration.EMPTY)) {

Assertions.assertEquals(1, localReceiverCount.get());
Assertions.assertEquals(0, topicReceiverCount.get());
}
localReceiverCount.set(0);
topicReceiverCount.set(0);
// Test local metric store configuration
try (var web =
new WebService(
Mockito.mock(Admin.class),
0,
id -> -1,
Duration.ofMillis(5),
new Configuration(
Map.of(WebService.METRIC_STORE_KEY, WebService.METRIC_STORE_LOCAL)))) {

Assertions.assertEquals(1, localReceiverCount.get());
Assertions.assertEquals(0, topicReceiverCount.get());
}
localReceiverCount.set(0);
topicReceiverCount.set(0);
// Test topic metric store configuration
try (var web =
new WebService(
Mockito.mock(Admin.class),
0,
id -> -1,
Duration.ofMillis(5),
new Configuration(
Map.of(
WebService.METRIC_STORE_KEY,
WebService.METRIC_STORE_TOPIC,
WebService.BOOTSTRAP_SERVERS_KEY,
"ignore")))) {

// topic collector may create local receiver to receive local jmx metric
Assertions.assertEquals(1, topicReceiverCount.get());
}

// Test invalid metric store configuration
Assertions.assertThrows(
IllegalArgumentException.class,
() ->
new WebService(
Mockito.mock(Admin.class),
0,
id -> -1,
Duration.ofMillis(5),
new Configuration(Map.of(WebService.METRIC_STORE_KEY, "unknown"))));
}
}
}
Original file line number Diff line number Diff line change
Expand Up @@ -23,6 +23,7 @@
import java.util.Map;
import java.util.NoSuchElementException;
import java.util.Optional;
import java.util.concurrent.CompletableFuture;
import java.util.concurrent.CompletionStage;
import java.util.function.Function;
import java.util.function.Supplier;
Expand All @@ -38,6 +39,7 @@
import org.astraea.common.metrics.JndiClient;
import org.astraea.common.metrics.MBeanClient;
import org.astraea.common.metrics.collector.MetricStore;
import org.astraea.common.producer.ProducerConfigs;

/**
* this partitioner scores the nodes by multiples cost functions. Each function evaluate the target
Expand All @@ -55,6 +57,9 @@
* `org.astraea.cost.ThroughputCost=1,org.astraea.cost.broker.BrokerOutputCost=1`.
*/
public class StrictCostPartitioner extends Partitioner {
public static final String METRIC_STORE_KEY = "metric.store";
public static final String METRIC_STORE_TOPIC = "topic";
public static final String METRIC_STORE_LOCAL = "local";
static final int ROUND_ROBIN_LENGTH = 400;
static final String JMX_PORT = "jmx.port";
static final String ROUND_ROBIN_LEASE_KEY = "round.robin.lease";
Expand Down Expand Up @@ -140,27 +145,47 @@ public void configure(Configuration config) {
.string(ROUND_ROBIN_LEASE_KEY)
.map(Utils::toDuration)
.ifPresent(d -> this.roundRobinLease = d);
Supplier<CompletionStage<Map<Integer, MBeanClient>>> clientSupplier =
() ->
admin
.brokers()
.thenApply(
brokers -> {
var map = new HashMap<Integer, JndiClient>();
brokers.forEach(
b ->
map.put(
b.id(), JndiClient.of(b.host(), jmxPortGetter.apply(b.id()))));
// add local client to fetch consumer metrics
map.put(-1, JndiClient.local());
return Collections.unmodifiableMap(map);
});

List<MetricStore.Receiver> receivers =
switch (config.string(METRIC_STORE_KEY).orElse(METRIC_STORE_LOCAL)) {
case METRIC_STORE_TOPIC -> List.of(
MetricStore.Receiver.topic(
config.requireString(ProducerConfigs.BOOTSTRAP_SERVERS_CONFIG)),
MetricStore.Receiver.local(
() -> CompletableFuture.completedStage(Map.of(-1, JndiClient.local()))));
case METRIC_STORE_LOCAL -> {
Supplier<CompletionStage<Map<Integer, MBeanClient>>> clientSupplier =
() ->
admin
.brokers()
.thenApply(
brokers -> {
var map = new HashMap<Integer, JndiClient>();
brokers.forEach(
b ->
map.put(
b.id(),
JndiClient.of(b.host(), jmxPortGetter.apply(b.id()))));
// add local client to fetch consumer metrics
map.put(-1, JndiClient.local());
return Collections.unmodifiableMap(map);
});
yield List.of(MetricStore.Receiver.local(clientSupplier));
}
default -> throw new IllegalArgumentException(
"unknown metric store type: "
+ config.string(METRIC_STORE_KEY)
+ ". Use "
+ METRIC_STORE_TOPIC
+ " or "
+ METRIC_STORE_LOCAL);
};
metricStore =
MetricStore.builder()
.receivers(List.of(MetricStore.Receiver.local(clientSupplier)))
.receivers(receivers)
.sensorsSupplier(() -> Map.of(this.costFunction.metricSensor(), (integer, e) -> {}))
.build();

this.roundRobinKeeper = RoundRobinKeeper.of(ROUND_ROBIN_LENGTH, roundRobinLease);
}

Expand Down
Loading

0 comments on commit 28ec2c6

Please sign in to comment.