This is an automated email from the ASF dual-hosted git repository.
abhishekrb pushed a commit to branch master
in repository https://gitbox.apache.org/repos/asf/druid.git
The following commit(s) were added to refs/heads/master by this push:
new 13b5492e342 Implement flushPeriod to expire metrics past TTL for
prometheus-emitter (#18598)
13b5492e342 is described below
commit 13b5492e342ca074d471d83f50405145398692de
Author: aho135 <[email protected]>
AuthorDate: Sun Oct 12 11:34:49 2025 -0700
Implement flushPeriod to expire metrics past TTL for prometheus-emitter
(#18598)
Partially addresses #14638. This implementation adds flushPeriod to the
exporter strategy in the prometheus-emitter, which when configured, will stop
emitting the metric if it has not been updated within the TTL. This does not
account for scenarios where the metric is being emitted but with different
label values. This PR can be enhanced in the future to track all label
combinations.
---
docs/development/extensions-contrib/prometheus.md | 2 +-
.../emitter/prometheus/DimensionsAndCollector.java | 31 +++++++-
.../apache/druid/emitter/prometheus/Metrics.java | 11 ++-
.../emitter/prometheus/PrometheusEmitter.java | 56 ++++++++++++--
.../prometheus/PrometheusEmitterConfig.java | 19 +++--
.../druid/emitter/prometheus/MetricsTest.java | 22 ++++--
.../prometheus/PrometheusEmitterConfigTest.java | 11 ++-
.../emitter/prometheus/PrometheusEmitterTest.java | 88 ++++++++++++++++++++++
website/.spelling | 3 +
9 files changed, 215 insertions(+), 28 deletions(-)
diff --git a/docs/development/extensions-contrib/prometheus.md
b/docs/development/extensions-contrib/prometheus.md
index 2114eb2d23b..d5660e2e54b 100644
--- a/docs/development/extensions-contrib/prometheus.md
+++ b/docs/development/extensions-contrib/prometheus.md
@@ -44,7 +44,7 @@ All the configuration parameters for the Prometheus emitter
are under `druid.emi
| `druid.emitter.prometheus.addHostAsLabel` | Flag to include the hostname
as a prometheus label.
| no | false
|
| `druid.emitter.prometheus.addServiceAsLabel` | Flag to include the druid
service name (e.g. `druid/broker`, `druid/coordinator`, etc.) as a prometheus
label.
| no | false
|
| `druid.emitter.prometheus.pushGatewayAddress` | Pushgateway address.
Required if using `pushgateway` strategy.
| no | none
|
-| `druid.emitter.prometheus.flushPeriod` | Emit metrics to Pushgateway
every `flushPeriod` seconds. Required if `pushgateway` strategy is used.
| no | 15
|
+| `druid.emitter.prometheus.flushPeriod` | When using the `pushgateway`
strategy metrics are emitted every `flushPeriod` seconds. <br/>When using the
`exporter` strategy this configures the metric TTL such that if the metric
value is not updated within `flushPeriod` seconds then it will stop being
emitted. Note that unique label combinations per metric are currently not
subject to TTL expiration. It is recommended to set this to at least 3 *
`scrape_interval`. | Required if `pushg [...]
| `druid.emitter.prometheus.extraLabels` | JSON key-value pairs for
additional labels on all metrics. Keys (label names) must match the regex
`[a-zA-Z_:][a-zA-Z0-9_:]*`. Example: `{"cluster_name": "druid_cluster1", "env":
"staging"}`. | no | none
|
| `druid.emitter.prometheus.deletePushGatewayMetricsOnShutdown` | Flag to
delete metrics from Pushgateway on task shutdown. Works only if `pushgateway`
strategy is used. This feature allows to delete a stale metrics from batch
executed tasks. Otherwise, the Pushgateway will store these stale metrics
indefinitely as there is [no time to live
mechanism](https://github.com/prometheus/pushgateway/issues/117), using the
memory to hold data that was already scraped by Prometheus. | no | false |
| `druid.emitter.prometheus.waitForShutdownDelay` | Time in milliseconds to
wait for peon tasks to delete metrics from the Pushgateway on shutdown (e.g.
60_000). Applicable only when `pushgateway` strategy is used and
`deletePushGatewayMetricsOnShutdown` is set to true. There is no guarantee that
a peon task will delete metrics from the gateway if the configured delay is
more than the [Peon's
`druid.indexer.task.gracefulShutdownTimeout`](https://druid.apache.org/docs/latest/configuration
[...]
diff --git
a/extensions-contrib/prometheus-emitter/src/main/java/org/apache/druid/emitter/prometheus/DimensionsAndCollector.java
b/extensions-contrib/prometheus-emitter/src/main/java/org/apache/druid/emitter/prometheus/DimensionsAndCollector.java
index 9fbf22f643a..73be1eed205 100644
---
a/extensions-contrib/prometheus-emitter/src/main/java/org/apache/druid/emitter/prometheus/DimensionsAndCollector.java
+++
b/extensions-contrib/prometheus-emitter/src/main/java/org/apache/druid/emitter/prometheus/DimensionsAndCollector.java
@@ -20,20 +20,30 @@
package org.apache.druid.emitter.prometheus;
import io.prometheus.client.SimpleCollector;
+import org.apache.druid.java.util.common.Stopwatch;
+import org.apache.druid.java.util.common.logger.Logger;
+import org.joda.time.Duration;
+
+import javax.annotation.Nullable;
public class DimensionsAndCollector
{
+ private static final Logger log = new Logger(DimensionsAndCollector.class);
private final String[] dimensions;
private final SimpleCollector collector;
private final double conversionFactor;
private final double[] histogramBuckets;
+ private final Stopwatch updateTimer;
+ private final Duration ttlSeconds;
- DimensionsAndCollector(String[] dimensions, SimpleCollector collector,
double conversionFactor, double[] histogramBuckets)
+ DimensionsAndCollector(String[] dimensions, SimpleCollector collector,
double conversionFactor, double[] histogramBuckets, @Nullable Integer
ttlSeconds)
{
this.dimensions = dimensions;
this.collector = collector;
this.conversionFactor = conversionFactor;
this.histogramBuckets = histogramBuckets;
+ this.updateTimer = Stopwatch.createStarted();
+ this.ttlSeconds = ttlSeconds != null ?
Duration.standardSeconds(ttlSeconds) : null;
}
public String[] getDimensions()
@@ -55,4 +65,23 @@ public class DimensionsAndCollector
{
return histogramBuckets;
}
+
+ public void resetLastUpdateTime()
+ {
+ updateTimer.restart();
+ }
+
+ public long getMillisSinceLastUpdate()
+ {
+ return updateTimer.millisElapsed();
+ }
+
+ public boolean isExpired()
+ {
+ if (ttlSeconds == null) {
+ log.error("Invalid usage of isExpired(), TTL has not been set");
+ return false;
+ }
+ return updateTimer.hasElapsed(ttlSeconds);
+ }
}
diff --git
a/extensions-contrib/prometheus-emitter/src/main/java/org/apache/druid/emitter/prometheus/Metrics.java
b/extensions-contrib/prometheus-emitter/src/main/java/org/apache/druid/emitter/prometheus/Metrics.java
index 00df3c46cdb..4edb76e7d84 100644
---
a/extensions-contrib/prometheus-emitter/src/main/java/org/apache/druid/emitter/prometheus/Metrics.java
+++
b/extensions-contrib/prometheus-emitter/src/main/java/org/apache/druid/emitter/prometheus/Metrics.java
@@ -63,8 +63,15 @@ public class Metrics
}
}
- public Metrics(String namespace, String path, boolean isAddHostAsLabel,
boolean isAddServiceAsLabel, Map<String, String> extraLabels)
+ public Metrics(PrometheusEmitterConfig config)
{
+ String namespace = config.getNamespace();
+ String path = config.getDimensionMapPath();
+ boolean isAddHostAsLabel = config.isAddHostAsLabel();
+ boolean isAddServiceAsLabel = config.isAddServiceAsLabel();
+ Map<String, String> extraLabels = config.getExtraLabels();
+ Integer ttlSeconds = config.getFlushPeriod();
+
Map<String, DimensionsAndCollector> parsedRegisteredMetrics = new
HashMap<>();
Map<String, Metric> metrics = readConfig(path);
@@ -117,7 +124,7 @@ public class Metrics
}
if (collector != null) {
- parsedRegisteredMetrics.put(name, new
DimensionsAndCollector(dimensions, collector, metric.conversionFactor,
metric.histogramBuckets));
+ parsedRegisteredMetrics.put(name, new
DimensionsAndCollector(dimensions, collector, metric.conversionFactor,
metric.histogramBuckets, ttlSeconds));
}
}
this.registeredMetrics =
Collections.unmodifiableMap(parsedRegisteredMetrics);
diff --git
a/extensions-contrib/prometheus-emitter/src/main/java/org/apache/druid/emitter/prometheus/PrometheusEmitter.java
b/extensions-contrib/prometheus-emitter/src/main/java/org/apache/druid/emitter/prometheus/PrometheusEmitter.java
index 4426ac30a96..87583b1ef82 100644
---
a/extensions-contrib/prometheus-emitter/src/main/java/org/apache/druid/emitter/prometheus/PrometheusEmitter.java
+++
b/extensions-contrib/prometheus-emitter/src/main/java/org/apache/druid/emitter/prometheus/PrometheusEmitter.java
@@ -69,16 +69,9 @@ public class PrometheusEmitter implements Emitter
{
this.config = config;
this.strategy = config.getStrategy();
- metrics = new Metrics(
- config.getNamespace(),
- config.getDimensionMapPath(),
- config.isAddHostAsLabel(),
- config.isAddServiceAsLabel(),
- config.getExtraLabels()
- );
+ metrics = new Metrics(config);
}
-
@Override
public void start()
{
@@ -93,6 +86,17 @@ public class PrometheusEmitter implements Emitter
} else {
log.error("HTTPServer is already started");
}
+ // Start TTL scheduler if TTL is configured
+ if (config.getFlushPeriod() != null) {
+ exec = ScheduledExecutors.fixed(1, "PrometheusTTLExecutor-%s");
+ exec.scheduleAtFixedRate(
+ this::cleanUpStaleMetrics,
+ config.getFlushPeriod(),
+ config.getFlushPeriod(),
+ TimeUnit.SECONDS
+ );
+ log.info("Started TTL scheduler with TTL of [%d] seconds.",
config.getFlushPeriod());
+ }
} else if (strategy.equals(PrometheusEmitterConfig.Strategy.pushgateway)) {
String address = config.getPushGatewayAddress();
if (address.startsWith("https") || address.startsWith("http")) {
@@ -167,11 +171,14 @@ public class PrometheusEmitter implements Emitter
if (metric.getCollector() instanceof Counter) {
((Counter)
metric.getCollector()).labels(labelValues).inc(value.doubleValue());
+ metric.resetLastUpdateTime();
} else if (metric.getCollector() instanceof Gauge) {
((Gauge)
metric.getCollector()).labels(labelValues).set(value.doubleValue());
+ metric.resetLastUpdateTime();
} else if (metric.getCollector() instanceof Histogram) {
((Histogram) metric.getCollector()).labels(labelValues)
.observe(value.doubleValue() /
metric.getConversionFactor());
+ metric.resetLastUpdateTime();
} else {
log.error("Unrecognized metric type [%s]",
metric.getCollector().getClass());
}
@@ -208,6 +215,9 @@ public class PrometheusEmitter implements Emitter
public void close()
{
if (strategy.equals(PrometheusEmitterConfig.Strategy.exporter)) {
+ if (exec != null) {
+ exec.shutdownNow();
+ }
if (server != null) {
server.close();
}
@@ -247,6 +257,11 @@ public class PrometheusEmitter implements Emitter
return server;
}
+ public Metrics getMetrics()
+ {
+ return metrics;
+ }
+
public PushGateway getPushGateway()
{
return pushGateway;
@@ -256,4 +271,29 @@ public class PrometheusEmitter implements Emitter
{
this.pushGateway = pushGateway;
}
+
+ /**
+ * Cleans up stale metrics that have not been updated within the configured
TTL.
+ * This method is called periodically by the TTL scheduler when using the
'exporter' strategy with
+ * a configured flushPeriod.
+ */
+ private void cleanUpStaleMetrics()
+ {
+ if (config.getFlushPeriod() == null) {
+ return;
+ }
+
+ Map<String, DimensionsAndCollector> map = metrics.getRegisteredMetrics();
+ for (Map.Entry<String, DimensionsAndCollector> entry : map.entrySet()) {
+ DimensionsAndCollector metric = entry.getValue();
+ if (metric.isExpired()) {
+ log.debug(
+ "Metric [%s] has expired (last updated [%d] ms ago)",
+ entry.getKey(),
+ metric.getMillisSinceLastUpdate()
+ );
+ metric.getCollector().clear();
+ }
+ }
+ }
}
diff --git
a/extensions-contrib/prometheus-emitter/src/main/java/org/apache/druid/emitter/prometheus/PrometheusEmitterConfig.java
b/extensions-contrib/prometheus-emitter/src/main/java/org/apache/druid/emitter/prometheus/PrometheusEmitterConfig.java
index bddea2a3848..56def5c4c2d 100644
---
a/extensions-contrib/prometheus-emitter/src/main/java/org/apache/druid/emitter/prometheus/PrometheusEmitterConfig.java
+++
b/extensions-contrib/prometheus-emitter/src/main/java/org/apache/druid/emitter/prometheus/PrometheusEmitterConfig.java
@@ -86,7 +86,7 @@ public class PrometheusEmitterConfig
@JsonProperty("pushGatewayAddress") @Nullable String pushGatewayAddress,
@JsonProperty("addHostAsLabel") boolean addHostAsLabel,
@JsonProperty("addServiceAsLabel") boolean addServiceAsLabel,
- @JsonProperty("flushPeriod") Integer flushPeriod,
+ @JsonProperty("flushPeriod") @Nullable Integer flushPeriod,
@JsonProperty("extraLabels") @Nullable Map<String, String> extraLabels,
@JsonProperty("deletePushGatewayMetricsOnShutdown") @Nullable Boolean
deletePushGatewayMetricsOnShutdown,
@JsonProperty("waitForShutdownDelay") @Nullable Long waitForShutdownDelay
@@ -99,11 +99,20 @@ public class PrometheusEmitterConfig
Preconditions.checkArgument(port != null, "For `exporter` strategy, port
must be specified.");
} else if (this.strategy == Strategy.pushgateway) {
Preconditions.checkArgument(pushGatewayAddress != null, "For
`pushgateway` strategy, pushGatewayAddress must be specified.");
- if (Objects.nonNull(flushPeriod)) {
- Preconditions.checkArgument(flushPeriod > 0, "flushPeriod must be
greater than 0.");
- } else {
- flushPeriod = 15;
+ }
+ if (Objects.nonNull(flushPeriod)) {
+ if (flushPeriod <= 0) {
+ throw DruidException.forPersona(DruidException.Persona.OPERATOR)
+ .ofCategory(DruidException.Category.INVALID_INPUT)
+ .build(
+ StringUtils.format(
+ "Invalid value for flushPeriod[%s]
specified, flushPeriod must be > 0.",
+ flushPeriod
+ )
+ );
}
+ } else if (this.strategy == Strategy.pushgateway) {
+ flushPeriod = 15;
}
this.dimensionMapPath = dimensionMapPath;
this.port = port;
diff --git
a/extensions-contrib/prometheus-emitter/src/test/java/org/apache/druid/emitter/prometheus/MetricsTest.java
b/extensions-contrib/prometheus-emitter/src/test/java/org/apache/druid/emitter/prometheus/MetricsTest.java
index ef6a1d51775..7139e446ab4 100644
---
a/extensions-contrib/prometheus-emitter/src/test/java/org/apache/druid/emitter/prometheus/MetricsTest.java
+++
b/extensions-contrib/prometheus-emitter/src/test/java/org/apache/druid/emitter/prometheus/MetricsTest.java
@@ -20,6 +20,7 @@
package org.apache.druid.emitter.prometheus;
import io.prometheus.client.Histogram;
+import org.apache.druid.error.DruidException;
import org.apache.druid.java.util.common.ISE;
import org.junit.Assert;
import org.junit.Test;
@@ -32,7 +33,8 @@ public class MetricsTest
@Test
public void testMetricsConfiguration()
{
- Metrics metrics = new Metrics("test", null, true, true, null);
+ PrometheusEmitterConfig config = new PrometheusEmitterConfig(null, "test",
null, null, null, true, true, null, null, null, null);
+ Metrics metrics = new Metrics(config);
DimensionsAndCollector dimensionsAndCollector =
metrics.getByName("query/time", "historical");
Assert.assertNotNull(dimensionsAndCollector);
String[] dimensions = dimensionsAndCollector.getDimensions();
@@ -59,7 +61,8 @@ public class MetricsTest
Map<String, String> extraLabels = new HashMap<>();
extraLabels.put("extra_label", "value");
- Metrics metrics = new Metrics("test_2", null, true, true, extraLabels);
+ PrometheusEmitterConfig config = new PrometheusEmitterConfig(null,
"test_2", null, null, null, true, true, null, extraLabels, null, null);
+ Metrics metrics = new Metrics(config);
DimensionsAndCollector dimensionsAndCollector =
metrics.getByName("query/time", "historical");
Assert.assertNotNull(dimensionsAndCollector);
String[] dimensions = dimensionsAndCollector.getDimensions();
@@ -89,11 +92,11 @@ public class MetricsTest
extraLabels.put("extra label", "value");
// Expect an exception thrown by Prometheus code due to invalid metric
label
- Exception exception = Assert.assertThrows(IllegalArgumentException.class,
() -> {
- new Metrics("test_3", null, true, true, extraLabels);
+ Exception exception = Assert.assertThrows(DruidException.class, () -> {
+ new Metrics(new PrometheusEmitterConfig(null, "test_3", null, null,
null, true, true, null, extraLabels, null, null));
});
- String expectedMessage = "Invalid metric label name: extra label";
+ String expectedMessage = "Invalid metric label name [extra label]. Label
names must conform to the pattern [[a-zA-Z_:][a-zA-Z0-9_:]*].";
String actualMessage = exception.getMessage();
Assert.assertTrue(actualMessage.contains(expectedMessage));
@@ -102,7 +105,8 @@ public class MetricsTest
@Test
public void testMetricsConfigurationWithNonExistentMetric()
{
- Metrics metrics = new Metrics("test_4", null, true, true, null);
+ PrometheusEmitterConfig config = new PrometheusEmitterConfig(null,
"test_4", null, null, null, true, true, null, null, null, null);
+ Metrics metrics = new Metrics(config);
DimensionsAndCollector nonExistentDimsCollector =
metrics.getByName("non/existent", "historical");
Assert.assertNull(nonExistentDimsCollector);
}
@@ -110,8 +114,9 @@ public class MetricsTest
@Test
public void testMetricsConfigurationWithUnSupportedType()
{
+ PrometheusEmitterConfig config = new PrometheusEmitterConfig(null,
"test_5", "src/test/resources/defaultInvalidMetricsTest.json", null, null,
true, true, null, null, null, null);
ISE iseException = Assert.assertThrows(ISE.class, () -> {
- new Metrics("test_5",
"src/test/resources/defaultInvalidMetricsTest.json", true, true, null);
+ new Metrics(config);
});
Assert.assertEquals("Failed to parse metric configuration",
iseException.getMessage());
}
@@ -119,7 +124,8 @@ public class MetricsTest
@Test
public void testMetricsConfigurationWithTimerHistogramBuckets()
{
- Metrics metrics = new Metrics("test_6",
"src/test/resources/defaultMetricsTest.json", true, true, null);
+ PrometheusEmitterConfig config = new PrometheusEmitterConfig(null,
"test_6", "src/test/resources/defaultMetricsTest.json", null, null, true, true,
null, null, null, null);
+ Metrics metrics = new Metrics(config);
DimensionsAndCollector dimensionsAndCollector =
metrics.getByName("query/time", "historical");
Assert.assertNotNull(dimensionsAndCollector);
String[] dimensions = dimensionsAndCollector.getDimensions();
diff --git
a/extensions-contrib/prometheus-emitter/src/test/java/org/apache/druid/emitter/prometheus/PrometheusEmitterConfigTest.java
b/extensions-contrib/prometheus-emitter/src/test/java/org/apache/druid/emitter/prometheus/PrometheusEmitterConfigTest.java
index b0b75cea4f2..962e3653f74 100644
---
a/extensions-contrib/prometheus-emitter/src/test/java/org/apache/druid/emitter/prometheus/PrometheusEmitterConfigTest.java
+++
b/extensions-contrib/prometheus-emitter/src/test/java/org/apache/druid/emitter/prometheus/PrometheusEmitterConfigTest.java
@@ -90,10 +90,15 @@ public class PrometheusEmitterConfigTest
@Test
public void testInvalidFlushPeriod()
{
- IllegalArgumentException illegalArgumentException =
Assert.assertThrows(IllegalArgumentException.class, () -> {
- new
PrometheusEmitterConfig(PrometheusEmitterConfig.Strategy.pushgateway, null,
null, null, "localhost:9091", false, false, 0, null, null, null);
+ DruidException druidException = Assert.assertThrows(DruidException.class,
() -> {
+ new
PrometheusEmitterConfig(PrometheusEmitterConfig.Strategy.pushgateway, null,
null, null, "localhost:9091", false, false, -1, null, null, null);
+ });
+ Assert.assertEquals("Invalid value for flushPeriod[-1] specified,
flushPeriod must be > 0.", druidException.getMessage());
+
+ druidException = Assert.assertThrows(DruidException.class, () -> {
+ new PrometheusEmitterConfig(PrometheusEmitterConfig.Strategy.exporter,
null, null, 0, null, false, false, 0, null, null, null);
});
- Assert.assertEquals("flushPeriod must be greater than 0.",
illegalArgumentException.getMessage());
+ Assert.assertEquals("Invalid value for flushPeriod[0] specified,
flushPeriod must be > 0.", druidException.getMessage());
}
@Test
diff --git
a/extensions-contrib/prometheus-emitter/src/test/java/org/apache/druid/emitter/prometheus/PrometheusEmitterTest.java
b/extensions-contrib/prometheus-emitter/src/test/java/org/apache/druid/emitter/prometheus/PrometheusEmitterTest.java
index 98a2a21df3b..69da07ff46b 100644
---
a/extensions-contrib/prometheus-emitter/src/test/java/org/apache/druid/emitter/prometheus/PrometheusEmitterTest.java
+++
b/extensions-contrib/prometheus-emitter/src/test/java/org/apache/druid/emitter/prometheus/PrometheusEmitterTest.java
@@ -33,6 +33,7 @@ import org.junit.Test;
import java.io.IOException;
import java.util.HashMap;
import java.util.Map;
+import java.util.concurrent.TimeUnit;
import static org.easymock.EasyMock.anyObject;
import static org.easymock.EasyMock.anyString;
@@ -430,4 +431,91 @@ public class PrometheusEmitterTest
EasyMock.verify(mockPushGateway);
}
+
+ @Test
+ public void testMetricTtlExpiration()
+ {
+ int flushPeriod = 3;
+ PrometheusEmitterConfig config = new
PrometheusEmitterConfig(PrometheusEmitterConfig.Strategy.exporter, "test",
null, 0, null, false, true, flushPeriod, null, false, null);
+ PrometheusEmitter emitter = new PrometheusEmitter(config);
+ emitter.start();
+
+ ServiceMetricEvent event = ServiceMetricEvent.builder()
+
.setMetric("segment/loadQueue/count", 10)
+
.build(ImmutableMap.of("service", "historical", "host", "druid.test.cn"));
+ emitter.emit(event);
+
+ // Get the metrics and check that it's not expired initially
+ Map<String, DimensionsAndCollector> registeredMetrics =
emitter.getMetrics().getRegisteredMetrics();
+ DimensionsAndCollector testMetric =
registeredMetrics.get("segment/loadQueue/count");
+
+ Assert.assertNotNull("Test metric should be registered", testMetric);
+ Assert.assertFalse(
+ "Metric should not be expired initially",
+ testMetric.isExpired()
+ );
+
+ // Wait for the metric to expire (ttl + 1 second buffer)
+ try {
+ Thread.sleep(TimeUnit.SECONDS.toMillis(flushPeriod) + 1000);
+ }
+ catch (InterruptedException e) {
+ Thread.currentThread().interrupt();
+ }
+
+ Assert.assertTrue(
+ "Metric should be expired after TTL",
+ testMetric.isExpired()
+ );
+ emitter.close();
+ }
+
+ @Test
+ public void testMetricTtlUpdate()
+ {
+ int flushPeriod = 3;
+ PrometheusEmitterConfig config = new
PrometheusEmitterConfig(PrometheusEmitterConfig.Strategy.exporter, "test",
null, 0, null, false, true, flushPeriod, null, false, null);
+ PrometheusEmitter emitter = new PrometheusEmitter(config);
+ emitter.start();
+
+ ServiceMetricEvent event = ServiceMetricEvent.builder()
+
.setMetric("segment/loadQueue/count", 10)
+
.build(ImmutableMap.of("service", "historical", "host", "druid.test.cn"));
+ emitter.emit(event);
+
+ // Get the metrics and check that it's not expired initially
+ Map<String, DimensionsAndCollector> registeredMetrics =
emitter.getMetrics().getRegisteredMetrics();
+ DimensionsAndCollector testMetric =
registeredMetrics.get("segment/loadQueue/count");
+
+ Assert.assertNotNull(
+ "Test metric should be registered",
+ testMetric
+ );
+ Assert.assertFalse(
+ "Metric should not be expired initially",
+ testMetric.isExpired()
+ );
+
+ // Wait for a little, but not long enough for the metric to expire
+ long waitTime = TimeUnit.SECONDS.toMillis(flushPeriod) / 5;
+ try {
+ Thread.sleep(waitTime);
+ }
+ catch (InterruptedException e) {
+ Thread.currentThread().interrupt();
+ }
+
+ Assert.assertFalse(
+ "Metric should not be expired",
+ testMetric.isExpired()
+ );
+ emitter.emit(event);
+
+ long timeSinceLastUpdate = testMetric.getMillisSinceLastUpdate();
+ Assert.assertTrue(
+ "Update time should have been refreshed",
+ timeSinceLastUpdate < waitTime
+ );
+ emitter.close();
+ }
}
diff --git a/website/.spelling b/website/.spelling
index 7aefe34bcba..a79ddea2f62 100644
--- a/website/.spelling
+++ b/website/.spelling
@@ -2458,3 +2458,6 @@ PropertyKeyName
PropertyValueType
tableSpec
TableSpec
+
+- ../docs/development/extensions-contrib/prometheus.md
+TTL
---------------------------------------------------------------------
To unsubscribe, e-mail: [email protected]
For additional commands, e-mail: [email protected]