kfaraz commented on code in PR #18742: URL: https://github.com/apache/druid/pull/18742#discussion_r2526539911
########## server/src/main/java/org/apache/druid/server/metrics/StorageMonitor.java: ########## @@ -0,0 +1,107 @@ +/* + * 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.druid.server.metrics; + +import com.google.inject.Inject; +import org.apache.druid.discovery.NodeRole; +import org.apache.druid.guice.annotations.LoadScope; +import org.apache.druid.java.util.emitter.service.ServiceEmitter; +import org.apache.druid.java.util.emitter.service.ServiceMetricEvent; +import org.apache.druid.java.util.metrics.AbstractMonitor; +import org.apache.druid.segment.loading.SegmentCacheManager; +import org.apache.druid.segment.loading.StorageLocationStats; +import org.apache.druid.segment.loading.StorageStats; +import org.apache.druid.segment.loading.VirtualStorageLocationStats; + +import java.util.Map; + +@LoadScope(roles = { + NodeRole.BROKER_JSON_NAME, + NodeRole.HISTORICAL_JSON_NAME, + NodeRole.INDEXER_JSON_NAME, + NodeRole.PEON_JSON_NAME +}) +public class StorageMonitor extends AbstractMonitor +{ + public static final String LOCATION_DIMENSION = "location"; + public static final String LOAD_COUNT = "storage/static/count"; + public static final String LOAD_BYTES = "storage/static/bytes"; + public static final String DROP_COUNT = "storage/drop/count"; + public static final String DROP_BYTES = "storage/drop/bytes"; + public static final String VSF_HIT_COUNT = "storage/virtual/hit/count"; + public static final String VSF_LOAD_COUNT = "storage/virtual/load/count"; + public static final String VSF_LOAD_BYTES = "storage/virtual/load/bytes"; + public static final String VSF_EVICT_COUNT = "storage/virtual/evict/count"; + public static final String VSF_EVICT_BYTES = "storage/virtual/evict/bytes"; + public static final String VSF_REJECT_COUNT = "storage/virtual/reject/count"; + + private final SegmentCacheManager cacheManager; + + @Inject + public StorageMonitor( + SegmentCacheManager cacheManager + ) + { + this.cacheManager = cacheManager; + } + + @Override + public boolean doMonitor(ServiceEmitter emitter) + { + final StorageStats stats = cacheManager.getStorageStats(); + + if (stats != null) { + for (Map.Entry<String, StorageLocationStats> location : stats.getLocationStats().entrySet()) { + final StorageLocationStats staticStats = location.getValue(); + final ServiceMetricEvent.Builder builder = new ServiceMetricEvent.Builder() + .setDimension(LOCATION_DIMENSION, location.getKey()); + builder.setMetric(LOAD_COUNT, staticStats.getLoadCount()); + emitter.emit(builder); + builder.setMetric(LOAD_BYTES, staticStats.getLoadBytes()); + emitter.emit(builder); + builder.setMetric(DROP_COUNT, staticStats.getDropCount()); + emitter.emit(builder); + builder.setMetric(DROP_BYTES, staticStats.getDropBytes()); + emitter.emit(builder); + } + + for (Map.Entry<String, VirtualStorageLocationStats> location : stats.getVirtualLocationStats().entrySet()) { + final VirtualStorageLocationStats weakStats = location.getValue(); + final ServiceMetricEvent.Builder builder = new ServiceMetricEvent.Builder().setDimension( + LOCATION_DIMENSION, + location.getKey() + ); + builder.setMetric(VSF_HIT_COUNT, weakStats.getHitCount()); + emitter.emit(builder); Review Comment: Nit: Maybe combine these lines for brevity (unless you prefer the current syntax more) ```suggestion emitter.emit(builder.setMetric(VSF_HIT_COUNT, weakStats.getHitCount())); ``` ########## server/src/main/java/org/apache/druid/server/metrics/StorageMonitor.java: ########## @@ -0,0 +1,107 @@ +/* + * 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.druid.server.metrics; + +import com.google.inject.Inject; +import org.apache.druid.discovery.NodeRole; +import org.apache.druid.guice.annotations.LoadScope; +import org.apache.druid.java.util.emitter.service.ServiceEmitter; +import org.apache.druid.java.util.emitter.service.ServiceMetricEvent; +import org.apache.druid.java.util.metrics.AbstractMonitor; +import org.apache.druid.segment.loading.SegmentCacheManager; +import org.apache.druid.segment.loading.StorageLocationStats; +import org.apache.druid.segment.loading.StorageStats; +import org.apache.druid.segment.loading.VirtualStorageLocationStats; + +import java.util.Map; + +@LoadScope(roles = { + NodeRole.BROKER_JSON_NAME, + NodeRole.HISTORICAL_JSON_NAME, + NodeRole.INDEXER_JSON_NAME, + NodeRole.PEON_JSON_NAME +}) +public class StorageMonitor extends AbstractMonitor +{ + public static final String LOCATION_DIMENSION = "location"; Review Comment: Should we move this to `DruidMetrics` class? ########## server/src/main/java/org/apache/druid/segment/loading/StorageLocationStats.java: ########## @@ -0,0 +1,31 @@ +/* + * 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.druid.segment.loading; + +public interface StorageLocationStats +{ + long getLoadCount(); + + long getLoadBytes(); Review Comment: Does this represent the load events that have occurred since the last emission period? Or the number/bytes of segments currently loaded at a location? The latter seems like a useful metric too. We already have the `segment/used` and `segment/usedPercent` metrics emitted by the `HistoricalMetricsMonitor`, but they are not keyed on the location as a dimension. ########## embedded-tests/src/test/java/org/apache/druid/testing/embedded/query/QueryVirtualStorageTest.java: ########## @@ -152,88 +158,141 @@ void testQueryPartials() "select count(*) from \"%s\" WHERE __time >= TIMESTAMP '2015-09-12 14:00:00' and __time < TIMESTAMP '2015-09-12 19:00:00'", "select count(*) from \"%s\" WHERE __time >= TIMESTAMP '2015-09-12 19:00:00' and __time < TIMESTAMP '2015-09-13 00:00:00'" }; - final long[] expectedResults = new long[] { - 9770, - 10524, - 10267, - 8683 - }; + final long[] expectedResults = new long[]{9770, 10524, 10267, 8683}; + final long[] expectedLoads = new long[]{8L, 6L, 5L, 5L}; + + + LatchableEmitter emitter = historical.latchableEmitter(); + // sleep to clear out the pipe to get zerod out storage monitor metrics and then flush (which clears out the + // internal events stores in test emitter) + sleepForStorageMonitor(); + emitter.flush(); + + emitter.waitForAnyEventWithMetricName(StorageMonitor.VSF_LOAD_COUNT); + long beforeLoads = getMetricTotal(emitter, StorageMonitor.VSF_LOAD_COUNT); + // confirm flushed + Assertions.assertEquals(0, beforeLoads); + // run the queries in order Assertions.assertEquals(expectedResults[0], Long.parseLong(cluster.runSql(queries[0], dataSource))); - assertMetrics(1, 8L); + assertQueryMetrics(1, expectedLoads[0]); Assertions.assertEquals(expectedResults[1], Long.parseLong(cluster.runSql(queries[1], dataSource))); - assertMetrics(2, 6L); + assertQueryMetrics(2, expectedLoads[1]); Assertions.assertEquals(expectedResults[2], Long.parseLong(cluster.runSql(queries[2], dataSource))); - assertMetrics(3, 5L); + assertQueryMetrics(3, expectedLoads[2]); Assertions.assertEquals(expectedResults[3], Long.parseLong(cluster.runSql(queries[3], dataSource))); - assertMetrics(4, 5L); + assertQueryMetrics(4, expectedLoads[3]); + sleepForStorageMonitor(); + emitter.waitForAnyEventWithMetricName(StorageMonitor.VSF_LOAD_COUNT); + long firstLoads = getMetricTotal(emitter, StorageMonitor.VSF_LOAD_COUNT); + Assertions.assertTrue(firstLoads >= 24, "expected " + 24 + " but only got " + firstLoads); + + long expectedTotalHits = 0; + long expectedTotalLoad = 0; for (int i = 0; i < 1000; i++) { int nextQuery = ThreadLocalRandom.current().nextInt(queries.length); Assertions.assertEquals(expectedResults[nextQuery], Long.parseLong(cluster.runSql(queries[nextQuery], dataSource))); - assertMetrics(i + 5, null); + assertQueryMetrics(i + 5, null); + long actualLoads = getMetricLatestEvent(emitter, DefaultQueryMetrics.QUERY_ON_DEMAND_LOAD_COUNT, i + 5); + expectedTotalLoad += actualLoads; + expectedTotalHits += (expectedLoads[nextQuery] - actualLoads); } + + sleepForStorageMonitor(); + + emitter.waitForAnyEventWithMetricName(StorageMonitor.VSF_HIT_COUNT); Review Comment: Nit: Do we need the new `waitForAnyEventWithMetricName` method? Does the original syntax seem verbose? ```suggestion emitter.waitForEvent(event -> event.hasMetricName(StorageMonitor.VSF_HIT_COUNT)); ``` Just wondering if we there is something we can do to make the syntax more dev-friendly for the general case too, since matching on just the metric name doesn't seem like a common use case. ########## server/src/main/java/org/apache/druid/server/metrics/StorageMonitor.java: ########## @@ -0,0 +1,107 @@ +/* + * 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.druid.server.metrics; + +import com.google.inject.Inject; +import org.apache.druid.discovery.NodeRole; +import org.apache.druid.guice.annotations.LoadScope; +import org.apache.druid.java.util.emitter.service.ServiceEmitter; +import org.apache.druid.java.util.emitter.service.ServiceMetricEvent; +import org.apache.druid.java.util.metrics.AbstractMonitor; +import org.apache.druid.segment.loading.SegmentCacheManager; +import org.apache.druid.segment.loading.StorageLocationStats; +import org.apache.druid.segment.loading.StorageStats; +import org.apache.druid.segment.loading.VirtualStorageLocationStats; + +import java.util.Map; + +@LoadScope(roles = { + NodeRole.BROKER_JSON_NAME, + NodeRole.HISTORICAL_JSON_NAME, + NodeRole.INDEXER_JSON_NAME, + NodeRole.PEON_JSON_NAME +}) +public class StorageMonitor extends AbstractMonitor Review Comment: Please add a short javadoc for this monitor. Would be nice to include it in docs too. ########## embedded-tests/src/test/java/org/apache/druid/testing/embedded/query/QueryVirtualStorageTest.java: ########## @@ -152,88 +158,141 @@ void testQueryPartials() "select count(*) from \"%s\" WHERE __time >= TIMESTAMP '2015-09-12 14:00:00' and __time < TIMESTAMP '2015-09-12 19:00:00'", "select count(*) from \"%s\" WHERE __time >= TIMESTAMP '2015-09-12 19:00:00' and __time < TIMESTAMP '2015-09-13 00:00:00'" }; - final long[] expectedResults = new long[] { - 9770, - 10524, - 10267, - 8683 - }; + final long[] expectedResults = new long[]{9770, 10524, 10267, 8683}; + final long[] expectedLoads = new long[]{8L, 6L, 5L, 5L}; + + + LatchableEmitter emitter = historical.latchableEmitter(); + // sleep to clear out the pipe to get zerod out storage monitor metrics and then flush (which clears out the + // internal events stores in test emitter) + sleepForStorageMonitor(); Review Comment: Can we avoid the sleep somehow? Maybe by waiting for some specific metric to be emitted? ########## embedded-tests/src/test/java/org/apache/druid/testing/embedded/query/QueryVirtualStorageTest.java: ########## @@ -152,88 +158,141 @@ void testQueryPartials() "select count(*) from \"%s\" WHERE __time >= TIMESTAMP '2015-09-12 14:00:00' and __time < TIMESTAMP '2015-09-12 19:00:00'", "select count(*) from \"%s\" WHERE __time >= TIMESTAMP '2015-09-12 19:00:00' and __time < TIMESTAMP '2015-09-13 00:00:00'" }; - final long[] expectedResults = new long[] { - 9770, - 10524, - 10267, - 8683 - }; + final long[] expectedResults = new long[]{9770, 10524, 10267, 8683}; + final long[] expectedLoads = new long[]{8L, 6L, 5L, 5L}; + + + LatchableEmitter emitter = historical.latchableEmitter(); + // sleep to clear out the pipe to get zerod out storage monitor metrics and then flush (which clears out the + // internal events stores in test emitter) + sleepForStorageMonitor(); + emitter.flush(); + + emitter.waitForAnyEventWithMetricName(StorageMonitor.VSF_LOAD_COUNT); + long beforeLoads = getMetricTotal(emitter, StorageMonitor.VSF_LOAD_COUNT); + // confirm flushed + Assertions.assertEquals(0, beforeLoads); + // run the queries in order Assertions.assertEquals(expectedResults[0], Long.parseLong(cluster.runSql(queries[0], dataSource))); - assertMetrics(1, 8L); + assertQueryMetrics(1, expectedLoads[0]); Assertions.assertEquals(expectedResults[1], Long.parseLong(cluster.runSql(queries[1], dataSource))); - assertMetrics(2, 6L); + assertQueryMetrics(2, expectedLoads[1]); Assertions.assertEquals(expectedResults[2], Long.parseLong(cluster.runSql(queries[2], dataSource))); - assertMetrics(3, 5L); + assertQueryMetrics(3, expectedLoads[2]); Assertions.assertEquals(expectedResults[3], Long.parseLong(cluster.runSql(queries[3], dataSource))); - assertMetrics(4, 5L); + assertQueryMetrics(4, expectedLoads[3]); + sleepForStorageMonitor(); + emitter.waitForAnyEventWithMetricName(StorageMonitor.VSF_LOAD_COUNT); + long firstLoads = getMetricTotal(emitter, StorageMonitor.VSF_LOAD_COUNT); + Assertions.assertTrue(firstLoads >= 24, "expected " + 24 + " but only got " + firstLoads); + + long expectedTotalHits = 0; + long expectedTotalLoad = 0; for (int i = 0; i < 1000; i++) { int nextQuery = ThreadLocalRandom.current().nextInt(queries.length); Assertions.assertEquals(expectedResults[nextQuery], Long.parseLong(cluster.runSql(queries[nextQuery], dataSource))); - assertMetrics(i + 5, null); + assertQueryMetrics(i + 5, null); + long actualLoads = getMetricLatestEvent(emitter, DefaultQueryMetrics.QUERY_ON_DEMAND_LOAD_COUNT, i + 5); + expectedTotalLoad += actualLoads; + expectedTotalHits += (expectedLoads[nextQuery] - actualLoads); } + + sleepForStorageMonitor(); + + emitter.waitForAnyEventWithMetricName(StorageMonitor.VSF_HIT_COUNT); + long hits = getMetricTotal(emitter, StorageMonitor.VSF_HIT_COUNT); + Assertions.assertTrue(hits >= expectedTotalHits, "expected " + expectedTotalHits + " but only got " + hits); + emitter.waitForAnyEventWithMetricName(StorageMonitor.VSF_LOAD_COUNT); + long loads = getMetricTotal(emitter, StorageMonitor.VSF_LOAD_COUNT); + Assertions.assertTrue(loads >= expectedTotalLoad, "expected " + expectedTotalLoad + " but only got " + loads); + Assertions.assertTrue(getMetricTotal(emitter, StorageMonitor.VSF_LOAD_BYTES) > 0); + emitter.waitForAnyEventWithMetricName(StorageMonitor.VSF_EVICT_COUNT); + Assertions.assertTrue(getMetricTotal(emitter, StorageMonitor.VSF_EVICT_COUNT) >= 0); + Assertions.assertTrue(getMetricTotal(emitter, StorageMonitor.VSF_EVICT_BYTES) > 0); + Assertions.assertEquals(0, getMetricTotal(emitter, StorageMonitor.VSF_REJECT_COUNT)); } - private void assertMetrics(int expectedEventCount, @Nullable Long expectedLoadCount) + private void sleepForStorageMonitor() + { + try { + Thread.sleep(1100); + } + catch (InterruptedException e) { + throw new RuntimeException(e); + } + } + + private void assertQueryMetrics(int expectedEventCount, @Nullable Long expectedLoadCount) { LatchableEmitter emitter = historical.latchableEmitter(); - final int lastIndex = expectedEventCount - 1; - List<ServiceMetricEvent> countEvents = emitter.getMetricEvents(DefaultQueryMetrics.QUERY_ON_DEMAND_LOAD_COUNT); - Assertions.assertEquals(expectedEventCount, countEvents.size()); + long loadCount = getMetricLatestEvent(emitter, DefaultQueryMetrics.QUERY_ON_DEMAND_LOAD_COUNT, expectedEventCount); if (expectedLoadCount != null) { - Assertions.assertEquals(expectedLoadCount, countEvents.get(lastIndex).getValue()); + Assertions.assertEquals(expectedLoadCount, loadCount); } - boolean hasLoads = countEvents.get(lastIndex).getValue().longValue() > 0; + boolean hasLoads = loadCount > 0; - List<ServiceMetricEvent> timeEvents = emitter.getMetricEvents(DefaultQueryMetrics.QUERY_ON_DEMAND_LOAD_BATCH_TIME); - Assertions.assertEquals(expectedEventCount, timeEvents.size()); + long time = getMetricLatestEvent(emitter, DefaultQueryMetrics.QUERY_ON_DEMAND_LOAD_BATCH_TIME, expectedEventCount); if (hasLoads) { - Assertions.assertTrue(timeEvents.get(lastIndex).getValue().longValue() > 0); + Assertions.assertTrue(time > 0); } else { - Assertions.assertEquals(0, timeEvents.get(lastIndex).getValue().longValue()); + Assertions.assertEquals(0, time); } - List<ServiceMetricEvent> timeMaxEvents = emitter.getMetricEvents(DefaultQueryMetrics.QUERY_ON_DEMAND_LOAD_TIME_MAX); - Assertions.assertEquals(expectedEventCount, timeMaxEvents.size()); + long maxTime = getMetricLatestEvent(emitter, DefaultQueryMetrics.QUERY_ON_DEMAND_LOAD_TIME_MAX, expectedEventCount); if (hasLoads) { - Assertions.assertTrue(timeMaxEvents.get(lastIndex).getValue().longValue() > 0); + Assertions.assertTrue(maxTime > 0); } else { - Assertions.assertEquals(0, timeMaxEvents.get(lastIndex).getValue().longValue()); + Assertions.assertEquals(0, maxTime); } - List<ServiceMetricEvent> timeAvgEvents = emitter.getMetricEvents(DefaultQueryMetrics.QUERY_ON_DEMAND_LOAD_TIME_AVG); - Assertions.assertEquals(expectedEventCount, timeAvgEvents.size()); + long avgTime = getMetricLatestEvent(emitter, DefaultQueryMetrics.QUERY_ON_DEMAND_LOAD_TIME_AVG, expectedEventCount); if (hasLoads) { - Assertions.assertTrue(timeAvgEvents.get(lastIndex).getValue().longValue() > 0); + Assertions.assertTrue(avgTime > 0); } else { - Assertions.assertEquals(0, timeAvgEvents.get(lastIndex).getValue().longValue()); + Assertions.assertEquals(0, avgTime); } - List<ServiceMetricEvent> waitMaxEvents = emitter.getMetricEvents(DefaultQueryMetrics.QUERY_ON_DEMAND_WAIT_TIME_MAX); - Assertions.assertEquals(expectedEventCount, waitMaxEvents.size()); + long maxWait = getMetricLatestEvent(emitter, DefaultQueryMetrics.QUERY_ON_DEMAND_WAIT_TIME_MAX, expectedEventCount); if (hasLoads) { - Assertions.assertTrue(waitMaxEvents.get(lastIndex).getValue().longValue() >= 0); + Assertions.assertTrue(maxWait >= 0); } else { - Assertions.assertEquals(0, waitMaxEvents.get(lastIndex).getValue().longValue()); + Assertions.assertEquals(0, maxWait); } - List<ServiceMetricEvent> waitAvgEvents = emitter.getMetricEvents(DefaultQueryMetrics.QUERY_ON_DEMAND_WAIT_TIME_AVG); - Assertions.assertEquals(expectedEventCount, waitAvgEvents.size()); + long avgWait = getMetricLatestEvent(emitter, DefaultQueryMetrics.QUERY_ON_DEMAND_WAIT_TIME_AVG, expectedEventCount); if (hasLoads) { - Assertions.assertTrue(waitAvgEvents.get(lastIndex).getValue().longValue() >= 0); + Assertions.assertTrue(avgWait >= 0); } else { - Assertions.assertEquals(0, waitAvgEvents.get(lastIndex).getValue().longValue()); + Assertions.assertEquals(0, avgWait); } - List<ServiceMetricEvent> loadSizeEvents = emitter.getMetricEvents(DefaultQueryMetrics.QUERY_ON_DEMAND_LOAD_BYTES); - Assertions.assertEquals(expectedEventCount, loadSizeEvents.size()); + long bytes = getMetricLatestEvent(emitter, DefaultQueryMetrics.QUERY_ON_DEMAND_LOAD_BYTES, expectedEventCount); if (hasLoads) { - Assertions.assertTrue(loadSizeEvents.get(lastIndex).getValue().longValue() > 0); + Assertions.assertTrue(bytes > 0); } else { - Assertions.assertEquals(0, loadSizeEvents.get(lastIndex).getValue().longValue()); + Assertions.assertEquals(0, bytes); + } + } + + private long getMetricLatestEvent(LatchableEmitter emitter, String metricName, int expectedCount) + { + final int lastIndex = expectedCount - 1; + List<ServiceMetricEvent> events = emitter.getMetricEvents(metricName); + Assertions.assertEquals(expectedCount, events.size()); + return events.get(lastIndex).getValue().longValue(); + } + + private long getMetricTotal(LatchableEmitter emitter, String metricName) Review Comment: I suppose this should be a utility method on `StubServiceEmitter` or `MetricsVerifier` itself. But we can do that later too. ########## server/src/main/java/org/apache/druid/segment/loading/SegmentCacheManager.java: ########## @@ -138,4 +138,11 @@ public interface SegmentCacheManager void shutdownBootstrap(); void shutdown(); + + /** + * Collect {@link StorageStats}, if available. Calling this method 'resets' the stats values, so that the measurements + * returned by this method only includes the events which have happened since the previous call to this method. + */ + @Nullable + StorageStats getStorageStats(); Review Comment: Maybe rename to `getAndResetStorageStats()` to align with behaviour? ########## server/src/main/java/org/apache/druid/segment/loading/StorageStats.java: ########## @@ -0,0 +1,61 @@ +/* + * 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.druid.segment.loading; + +import java.util.Map; + +/** + * Collection of {@link StorageLocationStats} and {@link VirtualStorageLocationStats} for all storage locations within + * a {@link SegmentCacheManager} so that {@link SegmentCacheManager#getStorageStats()} can be used by + * {@link org.apache.druid.server.metrics.StorageMonitor} to track segment cache activity. + * <p> + * Note that the stats are not tied explicitly to the {@link StorageLocation} implementation used by + * {@link SegmentLocalCacheManager}, but it does implement this stuff. + */ +public class StorageStats +{ + private final Map<String, StorageLocationStats> stats; + private final Map<String, VirtualStorageLocationStats> virtualStats; + + public StorageStats( + final Map<String, StorageLocationStats> stats, + final Map<String, VirtualStorageLocationStats> virtualStats + ) + { + this.stats = stats; + this.virtualStats = virtualStats; Review Comment: Can we ever have a case where both maps are non-empty i.e. some static storage locations and some virtual on the same service? ########## server/src/main/java/org/apache/druid/segment/loading/VirtualStorageLocationStats.java: ########## @@ -0,0 +1,35 @@ +/* + * 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.druid.segment.loading; + +public interface VirtualStorageLocationStats Review Comment: Should this extend `StorageLocationStats`? -- This is an automated message from the Apache Git Service. To respond to the message, please log on to GitHub and use the URL above to go to the specific comment. To unsubscribe, e-mail: [email protected] For queries about this service, please contact Infrastructure at: [email protected] --------------------------------------------------------------------- To unsubscribe, e-mail: [email protected] For additional commands, e-mail: [email protected]
