From 929635cb290b7ba6225e4da5ad343de1848e41c3 Mon Sep 17 00:00:00 2001 From: peteralfonsi Date: Mon, 29 Apr 2024 14:47:52 -0700 Subject: [PATCH 1/5] [Tiered Caching] Expose new cache stats API (#13237) Step 3 out of 4 --------- Signed-off-by: Peter Alfonsi Co-authored-by: Peter Alfonsi (cherry picked from commit 2a37bddefe876a91ace2faf1a3cc91ebe0b7b6e5) --- CHANGELOG.md | 1 + .../core/common/io/stream/StreamInput.java | 40 ++- .../common/tier/TieredSpilloverCache.java | 2 +- .../cache/common/tier/MockDiskCache.java | 5 + .../cache/store/disk/EhcacheDiskCache.java | 19 +- .../store/disk/EhCacheDiskCacheTests.java | 6 +- .../CacheStatsAPIIndicesRequestCacheIT.java | 291 +++++++++++++++++ .../admin/cluster/node/stats/NodeStats.java | 25 +- .../cluster/node/stats/NodesStatsRequest.java | 3 +- .../node/stats/TransportNodesStatsAction.java | 3 +- .../stats/TransportClusterStatsAction.java | 1 + .../admin/indices/stats/CommonStatsFlags.java | 42 +++ .../opensearch/common/cache/CacheType.java | 36 ++- .../org/opensearch/common/cache/ICache.java | 8 +- .../common/cache/service/CacheService.java | 12 + .../common/cache/service/NodeCacheStats.java | 80 +++++ .../common/cache/stats/CacheStats.java | 34 +- .../common/cache/stats/CacheStatsHolder.java | 6 +- .../cache/stats/DefaultCacheStatsHolder.java | 42 +-- .../cache/stats/ImmutableCacheStats.java | 46 ++- .../stats/ImmutableCacheStatsHolder.java | 293 +++++++++++++++++- .../cache/stats/NoopCacheStatsHolder.java | 11 +- .../cache/store/OpenSearchOnHeapCache.java | 12 +- .../cache/request/ShardRequestCache.java | 12 +- .../AbstractIndexShardCacheEntity.java | 21 +- .../indices/IndicesRequestCache.java | 20 +- .../main/java/org/opensearch/node/Node.java | 3 +- .../java/org/opensearch/node/NodeService.java | 12 +- .../admin/cluster/RestNodesStatsAction.java | 24 ++ .../cluster/node/stats/NodeStatsTests.java | 50 ++- .../opensearch/cluster/DiskUsageTests.java | 6 + .../stats/DefaultCacheStatsHolderTests.java | 69 +++-- .../stats/ImmutableCacheStatsHolderTests.java | 283 ++++++++++++++++- .../store/OpenSearchOnHeapCacheTests.java | 6 +- .../indices/IndicesRequestCacheTests.java | 4 +- .../MockInternalClusterInfoService.java | 3 +- .../opensearch/test/InternalTestCluster.java | 1 + 37 files changed, 1365 insertions(+), 167 deletions(-) create mode 100644 server/src/internalClusterTest/java/org/opensearch/indices/CacheStatsAPIIndicesRequestCacheIT.java create mode 100644 server/src/main/java/org/opensearch/common/cache/service/NodeCacheStats.java diff --git a/CHANGELOG.md b/CHANGELOG.md index 2568cc52087d2..13f16e962263b 100644 --- a/CHANGELOG.md +++ b/CHANGELOG.md @@ -23,6 +23,7 @@ The format is based on [Keep a Changelog](https://keepachangelog.com/en/1.0.0/), - Add changes for overriding remote store and replication settings during snapshot restore. ([#11868](https://github.com/opensearch-project/OpenSearch/pull/11868)) - Reject Resize index requests (i.e, split, shrink and clone), While DocRep to SegRep migration is in progress.([#12686](https://github.com/opensearch-project/OpenSearch/pull/12686)) - Add an individual setting of rate limiter for segment replication ([#12959](https://github.com/opensearch-project/OpenSearch/pull/12959)) +- [Tiered Caching] Expose new cache stats API ([#13237](https://github.com/opensearch-project/OpenSearch/pull/13237)) - [Streaming Indexing] Ensure support of the new transport by security plugin ([#13174](https://github.com/opensearch-project/OpenSearch/pull/13174)) - Add cluster setting to dynamically configure the buckets for filter rewrite optimization. ([#13179](https://github.com/opensearch-project/OpenSearch/pull/13179)) - [Tiered caching] Make Indices Request Cache Stale Key Mgmt Threshold setting dynamic ([#12941](https://github.com/opensearch-project/OpenSearch/pull/12941)) diff --git a/libs/core/src/main/java/org/opensearch/core/common/io/stream/StreamInput.java b/libs/core/src/main/java/org/opensearch/core/common/io/stream/StreamInput.java index 75402c6618e6e..a9beb565d5fee 100644 --- a/libs/core/src/main/java/org/opensearch/core/common/io/stream/StreamInput.java +++ b/libs/core/src/main/java/org/opensearch/core/common/io/stream/StreamInput.java @@ -80,6 +80,7 @@ import java.util.Arrays; import java.util.Collection; import java.util.Collections; +import java.util.Comparator; import java.util.Date; import java.util.EnumSet; import java.util.HashMap; @@ -90,6 +91,8 @@ import java.util.Locale; import java.util.Map; import java.util.Set; +import java.util.SortedMap; +import java.util.TreeMap; import java.util.concurrent.TimeUnit; import java.util.function.IntFunction; @@ -642,12 +645,47 @@ public Map readMap(Writeable.Reader keyReader, Writeable.Reader< return Collections.emptyMap(); } Map map = new HashMap<>(size); + readIntoMap(keyReader, valueReader, map, size); + return map; + } + + /** + * Read a serialized map into a SortedMap using the default ordering for the keys. If the result is empty it might be immutable. + */ + public , V> SortedMap readOrderedMap(Writeable.Reader keyReader, Writeable.Reader valueReader) + throws IOException { + return readOrderedMap(keyReader, valueReader, null); + } + + /** + * Read a serialized map into a SortedMap, specifying a Comparator for the keys. If the result is empty it might be immutable. + */ + public , V> SortedMap readOrderedMap( + Writeable.Reader keyReader, + Writeable.Reader valueReader, + @Nullable Comparator keyComparator + ) throws IOException { + int size = readArraySize(); + if (size == 0) { + return Collections.emptySortedMap(); + } + SortedMap sortedMap; + if (keyComparator == null) { + sortedMap = new TreeMap<>(); + } else { + sortedMap = new TreeMap<>(keyComparator); + } + readIntoMap(keyReader, valueReader, sortedMap, size); + return sortedMap; + } + + private void readIntoMap(Writeable.Reader keyReader, Writeable.Reader valueReader, Map map, int size) + throws IOException { for (int i = 0; i < size; i++) { K key = keyReader.read(this); V value = valueReader.read(this); map.put(key, value); } - return map; } /** diff --git a/modules/cache-common/src/main/java/org/opensearch/cache/common/tier/TieredSpilloverCache.java b/modules/cache-common/src/main/java/org/opensearch/cache/common/tier/TieredSpilloverCache.java index 34f17df751d7a..bca81ebd958ce 100644 --- a/modules/cache-common/src/main/java/org/opensearch/cache/common/tier/TieredSpilloverCache.java +++ b/modules/cache-common/src/main/java/org/opensearch/cache/common/tier/TieredSpilloverCache.java @@ -230,7 +230,7 @@ public void close() throws IOException { } @Override - public ImmutableCacheStatsHolder stats() { + public ImmutableCacheStatsHolder stats(String[] levels) { return null; // TODO: in TSC stats PR } diff --git a/modules/cache-common/src/test/java/org/opensearch/cache/common/tier/MockDiskCache.java b/modules/cache-common/src/test/java/org/opensearch/cache/common/tier/MockDiskCache.java index 0d98503af635f..8aed3f004e7b2 100644 --- a/modules/cache-common/src/test/java/org/opensearch/cache/common/tier/MockDiskCache.java +++ b/modules/cache-common/src/test/java/org/opensearch/cache/common/tier/MockDiskCache.java @@ -99,6 +99,11 @@ public ImmutableCacheStatsHolder stats() { return null; } + @Override + public ImmutableCacheStatsHolder stats(String[] levels) { + return null; + } + @Override public void close() { diff --git a/plugins/cache-ehcache/src/main/java/org/opensearch/cache/store/disk/EhcacheDiskCache.java b/plugins/cache-ehcache/src/main/java/org/opensearch/cache/store/disk/EhcacheDiskCache.java index eea13ce70ccb5..93c54a48d59da 100644 --- a/plugins/cache-ehcache/src/main/java/org/opensearch/cache/store/disk/EhcacheDiskCache.java +++ b/plugins/cache-ehcache/src/main/java/org/opensearch/cache/store/disk/EhcacheDiskCache.java @@ -164,7 +164,7 @@ private EhcacheDiskCache(Builder builder) { this.cache = buildCache(Duration.ofMillis(expireAfterAccess.getMillis()), builder); List dimensionNames = Objects.requireNonNull(builder.dimensionNames, "Dimension names can't be null"); // If this cache is being used, FeatureFlags.PLUGGABLE_CACHE is already on, so we can always use the DefaultCacheStatsHolder. - this.cacheStatsHolder = new DefaultCacheStatsHolder(dimensionNames); + this.cacheStatsHolder = new DefaultCacheStatsHolder(dimensionNames, EhcacheDiskCacheFactory.EHCACHE_DISK_CACHE_NAME); } @SuppressWarnings({ "rawtypes" }) @@ -446,12 +446,13 @@ public void close() { } /** - * Relevant stats for this cache. - * @return CacheStats + * Relevant stats for this cache, aggregated by levels. + * @param levels The levels to aggregate by. + * @return ImmutableCacheStatsHolder */ @Override - public ImmutableCacheStatsHolder stats() { - return cacheStatsHolder.getImmutableCacheStatsHolder(); + public ImmutableCacheStatsHolder stats(String[] levels) { + return cacheStatsHolder.getImmutableCacheStatsHolder(levels); } /** @@ -510,7 +511,7 @@ private long getNewValuePairSize(CacheEvent, ? extends By public void onEvent(CacheEvent, ? extends ByteArrayWrapper> event) { switch (event.getType()) { case CREATED: - cacheStatsHolder.incrementEntries(event.getKey().dimensions); + cacheStatsHolder.incrementItems(event.getKey().dimensions); cacheStatsHolder.incrementSizeInBytes(event.getKey().dimensions, getNewValuePairSize(event)); assert event.getOldValue() == null; break; @@ -518,7 +519,7 @@ public void onEvent(CacheEvent, ? extends ByteArrayWrappe this.removalListener.onRemoval( new RemovalNotification<>(event.getKey(), deserializeValue(event.getOldValue()), RemovalReason.EVICTED) ); - cacheStatsHolder.decrementEntries(event.getKey().dimensions); + cacheStatsHolder.decrementItems(event.getKey().dimensions); cacheStatsHolder.decrementSizeInBytes(event.getKey().dimensions, getOldValuePairSize(event)); cacheStatsHolder.incrementEvictions(event.getKey().dimensions); assert event.getNewValue() == null; @@ -527,7 +528,7 @@ public void onEvent(CacheEvent, ? extends ByteArrayWrappe this.removalListener.onRemoval( new RemovalNotification<>(event.getKey(), deserializeValue(event.getOldValue()), RemovalReason.EXPLICIT) ); - cacheStatsHolder.decrementEntries(event.getKey().dimensions); + cacheStatsHolder.decrementItems(event.getKey().dimensions); cacheStatsHolder.decrementSizeInBytes(event.getKey().dimensions, getOldValuePairSize(event)); assert event.getNewValue() == null; break; @@ -535,7 +536,7 @@ public void onEvent(CacheEvent, ? extends ByteArrayWrappe this.removalListener.onRemoval( new RemovalNotification<>(event.getKey(), deserializeValue(event.getOldValue()), RemovalReason.INVALIDATED) ); - cacheStatsHolder.decrementEntries(event.getKey().dimensions); + cacheStatsHolder.decrementItems(event.getKey().dimensions); cacheStatsHolder.decrementSizeInBytes(event.getKey().dimensions, getOldValuePairSize(event)); assert event.getNewValue() == null; break; diff --git a/plugins/cache-ehcache/src/test/java/org/opensearch/cache/store/disk/EhCacheDiskCacheTests.java b/plugins/cache-ehcache/src/test/java/org/opensearch/cache/store/disk/EhCacheDiskCacheTests.java index 06ebed08d7525..f93b09cf2d4f4 100644 --- a/plugins/cache-ehcache/src/test/java/org/opensearch/cache/store/disk/EhCacheDiskCacheTests.java +++ b/plugins/cache-ehcache/src/test/java/org/opensearch/cache/store/disk/EhCacheDiskCacheTests.java @@ -93,7 +93,7 @@ public void testBasicGetAndPut() throws IOException { String value = ehcacheTest.get(getICacheKey(entry.getKey())); assertEquals(entry.getValue(), value); } - assertEquals(randomKeys, ehcacheTest.stats().getTotalEntries()); + assertEquals(randomKeys, ehcacheTest.stats().getTotalItems()); assertEquals(randomKeys, ehcacheTest.stats().getTotalHits()); assertEquals(expectedSize, ehcacheTest.stats().getTotalSizeInBytes()); assertEquals(randomKeys, ehcacheTest.count()); @@ -217,7 +217,7 @@ public void testConcurrentPut() throws Exception { assertEquals(entry.getValue(), value); } assertEquals(randomKeys, ehcacheTest.count()); - assertEquals(randomKeys, ehcacheTest.stats().getTotalEntries()); + assertEquals(randomKeys, ehcacheTest.stats().getTotalItems()); ehcacheTest.close(); } } @@ -416,7 +416,7 @@ public String load(ICacheKey key) { assertEquals(1, numberOfTimesValueLoaded); assertEquals(0, ((EhcacheDiskCache) ehcacheTest).getCompletableFutureMap().size()); assertEquals(1, ehcacheTest.stats().getTotalMisses()); - assertEquals(1, ehcacheTest.stats().getTotalEntries()); + assertEquals(1, ehcacheTest.stats().getTotalItems()); assertEquals(numberOfRequest - 1, ehcacheTest.stats().getTotalHits()); assertEquals(1, ehcacheTest.count()); ehcacheTest.close(); diff --git a/server/src/internalClusterTest/java/org/opensearch/indices/CacheStatsAPIIndicesRequestCacheIT.java b/server/src/internalClusterTest/java/org/opensearch/indices/CacheStatsAPIIndicesRequestCacheIT.java new file mode 100644 index 0000000000000..de7a52761c77c --- /dev/null +++ b/server/src/internalClusterTest/java/org/opensearch/indices/CacheStatsAPIIndicesRequestCacheIT.java @@ -0,0 +1,291 @@ +/* + * SPDX-License-Identifier: Apache-2.0 + * + * The OpenSearch Contributors require contributions made to + * this file be licensed under the Apache-2.0 license or a + * compatible open source license. + */ + +package org.opensearch.indices; + +import com.carrotsearch.randomizedtesting.annotations.ParametersFactory; + +import org.opensearch.action.admin.cluster.node.stats.NodesStatsRequest; +import org.opensearch.action.admin.cluster.node.stats.NodesStatsResponse; +import org.opensearch.action.admin.indices.stats.CommonStatsFlags; +import org.opensearch.action.search.SearchResponse; +import org.opensearch.client.Client; +import org.opensearch.cluster.metadata.IndexMetadata; +import org.opensearch.common.Randomness; +import org.opensearch.common.cache.CacheType; +import org.opensearch.common.cache.service.NodeCacheStats; +import org.opensearch.common.cache.stats.ImmutableCacheStats; +import org.opensearch.common.cache.stats.ImmutableCacheStatsHolderTests; +import org.opensearch.common.settings.Settings; +import org.opensearch.common.util.FeatureFlags; +import org.opensearch.common.xcontent.XContentFactory; +import org.opensearch.common.xcontent.XContentHelper; +import org.opensearch.core.xcontent.MediaTypeRegistry; +import org.opensearch.core.xcontent.ToXContent; +import org.opensearch.core.xcontent.XContentBuilder; +import org.opensearch.index.cache.request.RequestCacheStats; +import org.opensearch.index.query.QueryBuilders; +import org.opensearch.test.OpenSearchIntegTestCase; +import org.opensearch.test.ParameterizedStaticSettingsOpenSearchIntegTestCase; +import org.opensearch.test.hamcrest.OpenSearchAssertions; + +import java.io.IOException; +import java.util.Arrays; +import java.util.Collection; +import java.util.HashMap; +import java.util.List; +import java.util.Map; + +import static org.opensearch.test.hamcrest.OpenSearchAssertions.assertAcked; +import static org.opensearch.test.hamcrest.OpenSearchAssertions.assertSearchResponse; + +// Use a single data node to simplify logic about cache stats across different shards. +@OpenSearchIntegTestCase.ClusterScope(scope = OpenSearchIntegTestCase.Scope.TEST, numDataNodes = 1) +public class CacheStatsAPIIndicesRequestCacheIT extends ParameterizedStaticSettingsOpenSearchIntegTestCase { + public CacheStatsAPIIndicesRequestCacheIT(Settings settings) { + super(settings); + } + + @ParametersFactory + public static Collection parameters() { + return Arrays.asList(new Object[] { Settings.builder().put(FeatureFlags.PLUGGABLE_CACHE, "true").build() }); + } + + public void testCacheStatsAPIWIthOnHeapCache() throws Exception { + String index1Name = "index1"; + String index2Name = "index2"; + Client client = client(); + + startIndex(client, index1Name); + startIndex(client, index2Name); + + // Search twice for the same doc in index 1 + for (int i = 0; i < 2; i++) { + searchIndex(client, index1Name, ""); + } + + // Search once for a doc in index 2 + searchIndex(client, index2Name, ""); + + // First, aggregate by indices only + Map xContentMap = getNodeCacheStatsXContentMap(client, List.of(IndicesRequestCache.INDEX_DIMENSION_NAME)); + + List index1Keys = List.of(CacheType.INDICES_REQUEST_CACHE.getValue(), IndicesRequestCache.INDEX_DIMENSION_NAME, index1Name); + // Since we searched twice, we expect to see 1 hit, 1 miss and 1 entry for index 1 + ImmutableCacheStats expectedStats = new ImmutableCacheStats(1, 1, 0, 0, 1); + checkCacheStatsAPIResponse(xContentMap, index1Keys, expectedStats, false, true); + // Get the request size for one request, so we can reuse it for next index + int requestSize = (int) ((Map) ImmutableCacheStatsHolderTests.getValueFromNestedXContentMap( + xContentMap, + index1Keys + )).get(ImmutableCacheStats.Fields.SIZE_IN_BYTES); + assertTrue(requestSize > 0); + + List index2Keys = List.of(CacheType.INDICES_REQUEST_CACHE.getValue(), IndicesRequestCache.INDEX_DIMENSION_NAME, index2Name); + // We searched once in index 2, we expect 1 miss + 1 entry + expectedStats = new ImmutableCacheStats(0, 1, 0, requestSize, 1); + checkCacheStatsAPIResponse(xContentMap, index2Keys, expectedStats, true, true); + + // The total stats for the node should be 1 hit, 2 misses, and 2 entries + expectedStats = new ImmutableCacheStats(1, 2, 0, 2 * requestSize, 2); + List totalStatsKeys = List.of(CacheType.INDICES_REQUEST_CACHE.getValue()); + checkCacheStatsAPIResponse(xContentMap, totalStatsKeys, expectedStats, true, true); + + // Aggregate by shards only + xContentMap = getNodeCacheStatsXContentMap(client, List.of(IndicesRequestCache.SHARD_ID_DIMENSION_NAME)); + + List index1Shard0Keys = List.of( + CacheType.INDICES_REQUEST_CACHE.getValue(), + IndicesRequestCache.SHARD_ID_DIMENSION_NAME, + "[" + index1Name + "][0]" + ); + + expectedStats = new ImmutableCacheStats(1, 1, 0, requestSize, 1); + checkCacheStatsAPIResponse(xContentMap, index1Shard0Keys, expectedStats, true, true); + + List index2Shard0Keys = List.of( + CacheType.INDICES_REQUEST_CACHE.getValue(), + IndicesRequestCache.SHARD_ID_DIMENSION_NAME, + "[" + index2Name + "][0]" + ); + expectedStats = new ImmutableCacheStats(0, 1, 0, requestSize, 1); + checkCacheStatsAPIResponse(xContentMap, index2Shard0Keys, expectedStats, true, true); + + // Aggregate by indices and shards + xContentMap = getNodeCacheStatsXContentMap( + client, + List.of(IndicesRequestCache.INDEX_DIMENSION_NAME, IndicesRequestCache.SHARD_ID_DIMENSION_NAME) + ); + + index1Keys = List.of( + CacheType.INDICES_REQUEST_CACHE.getValue(), + IndicesRequestCache.INDEX_DIMENSION_NAME, + index1Name, + IndicesRequestCache.SHARD_ID_DIMENSION_NAME, + "[" + index1Name + "][0]" + ); + + expectedStats = new ImmutableCacheStats(1, 1, 0, requestSize, 1); + checkCacheStatsAPIResponse(xContentMap, index1Keys, expectedStats, true, true); + + index2Keys = List.of( + CacheType.INDICES_REQUEST_CACHE.getValue(), + IndicesRequestCache.INDEX_DIMENSION_NAME, + index2Name, + IndicesRequestCache.SHARD_ID_DIMENSION_NAME, + "[" + index2Name + "][0]" + ); + + expectedStats = new ImmutableCacheStats(0, 1, 0, requestSize, 1); + checkCacheStatsAPIResponse(xContentMap, index2Keys, expectedStats, true, true); + + } + + // TODO: Add testCacheStatsAPIWithTieredCache when TSC stats implementation PR is merged + + public void testStatsMatchOldApi() throws Exception { + // The main purpose of this test is to check that the new and old APIs are both correctly estimating memory size, + // using the logic that includes the overhead memory in ICacheKey. + String index = "index"; + Client client = client(); + startIndex(client, index); + + int numKeys = Randomness.get().nextInt(100) + 1; + for (int i = 0; i < numKeys; i++) { + searchIndex(client, index, String.valueOf(i)); + } + // Get some hits as well + for (int i = 0; i < numKeys / 2; i++) { + searchIndex(client, index, String.valueOf(i)); + } + + RequestCacheStats oldApiStats = client.admin() + .indices() + .prepareStats(index) + .setRequestCache(true) + .get() + .getTotal() + .getRequestCache(); + assertNotEquals(0, oldApiStats.getMemorySizeInBytes()); + + List xContentMapKeys = List.of(CacheType.INDICES_REQUEST_CACHE.getValue()); + Map xContentMap = getNodeCacheStatsXContentMap(client, List.of()); + ImmutableCacheStats expected = new ImmutableCacheStats( + oldApiStats.getHitCount(), + oldApiStats.getMissCount(), + oldApiStats.getEvictions(), + oldApiStats.getMemorySizeInBytes(), + 0 + ); + // Don't check entries, as the old API doesn't track this + checkCacheStatsAPIResponse(xContentMap, xContentMapKeys, expected, true, false); + } + + public void testNullLevels() throws Exception { + String index = "index"; + Client client = client(); + startIndex(client, index); + int numKeys = Randomness.get().nextInt(100) + 1; + for (int i = 0; i < numKeys; i++) { + searchIndex(client, index, String.valueOf(i)); + } + Map xContentMap = getNodeCacheStatsXContentMap(client, null); + // Null levels should result in only the total cache stats being returned -> 6 fields inside the response. + assertEquals(6, ((Map) xContentMap.get("request_cache")).size()); + } + + private void startIndex(Client client, String indexName) throws InterruptedException { + assertAcked( + client.admin() + .indices() + .prepareCreate(indexName) + .setMapping("k", "type=keyword") + .setSettings( + Settings.builder() + .put(IndicesRequestCache.INDEX_CACHE_REQUEST_ENABLED_SETTING.getKey(), true) + .put(IndexMetadata.SETTING_NUMBER_OF_SHARDS, 1) + .put(IndexMetadata.SETTING_NUMBER_OF_REPLICAS, 0) + ) + .get() + ); + indexRandom(true, client.prepareIndex(indexName).setSource("k", "hello")); + ensureSearchable(indexName); + } + + private SearchResponse searchIndex(Client client, String index, String searchSuffix) { + SearchResponse resp = client.prepareSearch(index) + .setRequestCache(true) + .setQuery(QueryBuilders.termQuery("k", "hello" + searchSuffix)) + .get(); + assertSearchResponse(resp); + OpenSearchAssertions.assertAllSuccessful(resp); + return resp; + } + + private static Map getNodeCacheStatsXContentMap(Client client, List aggregationLevels) throws IOException { + + CommonStatsFlags statsFlags = new CommonStatsFlags(); + statsFlags.includeAllCacheTypes(); + String[] flagsLevels; + if (aggregationLevels == null) { + flagsLevels = null; + } else { + flagsLevels = aggregationLevels.toArray(new String[0]); + } + statsFlags.setLevels(flagsLevels); + + NodesStatsResponse nodeStatsResponse = client.admin() + .cluster() + .prepareNodesStats("data:true") + .addMetric(NodesStatsRequest.Metric.CACHE_STATS.metricName()) + .setIndices(statsFlags) + .get(); + // Can always get the first data node as there's only one in this test suite + assertEquals(1, nodeStatsResponse.getNodes().size()); + NodeCacheStats ncs = nodeStatsResponse.getNodes().get(0).getNodeCacheStats(); + + XContentBuilder builder = XContentFactory.jsonBuilder(); + Map paramMap = new HashMap<>(); + if (aggregationLevels != null && !aggregationLevels.isEmpty()) { + paramMap.put("level", String.join(",", aggregationLevels)); + } + ToXContent.Params params = new ToXContent.MapParams(paramMap); + + builder.startObject(); + ncs.toXContent(builder, params); + builder.endObject(); + + String resultString = builder.toString(); + return XContentHelper.convertToMap(MediaTypeRegistry.JSON.xContent(), resultString, true); + } + + private static void checkCacheStatsAPIResponse( + Map xContentMap, + List xContentMapKeys, + ImmutableCacheStats expectedStats, + boolean checkMemorySize, + boolean checkEntries + ) { + // Assumes the keys point to a level whose keys are the field values ("size_in_bytes", "evictions", etc) and whose values store + // those stats + Map aggregatedStatsResponse = (Map) ImmutableCacheStatsHolderTests.getValueFromNestedXContentMap( + xContentMap, + xContentMapKeys + ); + assertNotNull(aggregatedStatsResponse); + assertEquals(expectedStats.getHits(), (int) aggregatedStatsResponse.get(ImmutableCacheStats.Fields.HIT_COUNT)); + assertEquals(expectedStats.getMisses(), (int) aggregatedStatsResponse.get(ImmutableCacheStats.Fields.MISS_COUNT)); + assertEquals(expectedStats.getEvictions(), (int) aggregatedStatsResponse.get(ImmutableCacheStats.Fields.EVICTIONS)); + if (checkMemorySize) { + assertEquals(expectedStats.getSizeInBytes(), (int) aggregatedStatsResponse.get(ImmutableCacheStats.Fields.SIZE_IN_BYTES)); + } + if (checkEntries) { + assertEquals(expectedStats.getItems(), (int) aggregatedStatsResponse.get(ImmutableCacheStats.Fields.ITEM_COUNT)); + } + } +} diff --git a/server/src/main/java/org/opensearch/action/admin/cluster/node/stats/NodeStats.java b/server/src/main/java/org/opensearch/action/admin/cluster/node/stats/NodeStats.java index 0e8e12d65b892..76759a230a6d6 100644 --- a/server/src/main/java/org/opensearch/action/admin/cluster/node/stats/NodeStats.java +++ b/server/src/main/java/org/opensearch/action/admin/cluster/node/stats/NodeStats.java @@ -40,6 +40,7 @@ import org.opensearch.cluster.routing.WeightedRoutingStats; import org.opensearch.cluster.service.ClusterManagerThrottlingStats; import org.opensearch.common.Nullable; +import org.opensearch.common.cache.service.NodeCacheStats; import org.opensearch.core.common.io.stream.StreamInput; import org.opensearch.core.common.io.stream.StreamOutput; import org.opensearch.core.indices.breaker.AllCircuitBreakerStats; @@ -159,6 +160,9 @@ public class NodeStats extends BaseNodeResponse implements ToXContentFragment { @Nullable private AdmissionControlStats admissionControlStats; + @Nullable + private NodeCacheStats nodeCacheStats; + public NodeStats(StreamInput in) throws IOException { super(in); timestamp = in.readVLong(); @@ -248,6 +252,12 @@ public NodeStats(StreamInput in) throws IOException { } else { admissionControlStats = null; } + // TODO: change from V_3_0_0 to V_2_14_0 on main after backport to 2.x + if (in.getVersion().onOrAfter(Version.V_2_14_0)) { + nodeCacheStats = in.readOptionalWriteable(NodeCacheStats::new); + } else { + nodeCacheStats = null; + } } public NodeStats( @@ -278,7 +288,8 @@ public NodeStats( @Nullable SearchPipelineStats searchPipelineStats, @Nullable SegmentReplicationRejectionStats segmentReplicationRejectionStats, @Nullable RepositoriesStats repositoriesStats, - @Nullable AdmissionControlStats admissionControlStats + @Nullable AdmissionControlStats admissionControlStats, + @Nullable NodeCacheStats nodeCacheStats ) { super(node); this.timestamp = timestamp; @@ -308,6 +319,7 @@ public NodeStats( this.segmentReplicationRejectionStats = segmentReplicationRejectionStats; this.repositoriesStats = repositoriesStats; this.admissionControlStats = admissionControlStats; + this.nodeCacheStats = nodeCacheStats; } public long getTimestamp() { @@ -465,6 +477,11 @@ public AdmissionControlStats getAdmissionControlStats() { return admissionControlStats; } + @Nullable + public NodeCacheStats getNodeCacheStats() { + return nodeCacheStats; + } + @Override public void writeTo(StreamOutput out) throws IOException { super.writeTo(out); @@ -527,6 +544,9 @@ public void writeTo(StreamOutput out) throws IOException { if (out.getVersion().onOrAfter(Version.V_2_12_0)) { out.writeOptionalWriteable(admissionControlStats); } + if (out.getVersion().onOrAfter(Version.V_2_14_0)) { + out.writeOptionalWriteable(nodeCacheStats); + } } @Override @@ -629,6 +649,9 @@ public XContentBuilder toXContent(XContentBuilder builder, Params params) throws if (getAdmissionControlStats() != null) { getAdmissionControlStats().toXContent(builder, params); } + if (getNodeCacheStats() != null) { + getNodeCacheStats().toXContent(builder, params); + } return builder; } } diff --git a/server/src/main/java/org/opensearch/action/admin/cluster/node/stats/NodesStatsRequest.java b/server/src/main/java/org/opensearch/action/admin/cluster/node/stats/NodesStatsRequest.java index 27d20c9775091..56711b6ec5fb8 100644 --- a/server/src/main/java/org/opensearch/action/admin/cluster/node/stats/NodesStatsRequest.java +++ b/server/src/main/java/org/opensearch/action/admin/cluster/node/stats/NodesStatsRequest.java @@ -250,7 +250,8 @@ public enum Metric { RESOURCE_USAGE_STATS("resource_usage_stats"), SEGMENT_REPLICATION_BACKPRESSURE("segment_replication_backpressure"), REPOSITORIES("repositories"), - ADMISSION_CONTROL("admission_control"); + ADMISSION_CONTROL("admission_control"), + CACHE_STATS("caches"); private String metricName; diff --git a/server/src/main/java/org/opensearch/action/admin/cluster/node/stats/TransportNodesStatsAction.java b/server/src/main/java/org/opensearch/action/admin/cluster/node/stats/TransportNodesStatsAction.java index 2363b3583fc25..1c1ebc240ee67 100644 --- a/server/src/main/java/org/opensearch/action/admin/cluster/node/stats/TransportNodesStatsAction.java +++ b/server/src/main/java/org/opensearch/action/admin/cluster/node/stats/TransportNodesStatsAction.java @@ -128,7 +128,8 @@ protected NodeStats nodeOperation(NodeStatsRequest nodeStatsRequest) { NodesStatsRequest.Metric.RESOURCE_USAGE_STATS.containedIn(metrics), NodesStatsRequest.Metric.SEGMENT_REPLICATION_BACKPRESSURE.containedIn(metrics), NodesStatsRequest.Metric.REPOSITORIES.containedIn(metrics), - NodesStatsRequest.Metric.ADMISSION_CONTROL.containedIn(metrics) + NodesStatsRequest.Metric.ADMISSION_CONTROL.containedIn(metrics), + NodesStatsRequest.Metric.CACHE_STATS.containedIn(metrics) ); } diff --git a/server/src/main/java/org/opensearch/action/admin/cluster/stats/TransportClusterStatsAction.java b/server/src/main/java/org/opensearch/action/admin/cluster/stats/TransportClusterStatsAction.java index 144080bb581a5..61adca8d89747 100644 --- a/server/src/main/java/org/opensearch/action/admin/cluster/stats/TransportClusterStatsAction.java +++ b/server/src/main/java/org/opensearch/action/admin/cluster/stats/TransportClusterStatsAction.java @@ -172,6 +172,7 @@ protected ClusterStatsNodeResponse nodeOperation(ClusterStatsNodeRequest nodeReq false, false, false, + false, false ); List shardsStats = new ArrayList<>(); diff --git a/server/src/main/java/org/opensearch/action/admin/indices/stats/CommonStatsFlags.java b/server/src/main/java/org/opensearch/action/admin/indices/stats/CommonStatsFlags.java index 5710ca3bf7938..6583a6427ced8 100644 --- a/server/src/main/java/org/opensearch/action/admin/indices/stats/CommonStatsFlags.java +++ b/server/src/main/java/org/opensearch/action/admin/indices/stats/CommonStatsFlags.java @@ -35,6 +35,7 @@ import org.opensearch.LegacyESVersion; import org.opensearch.Version; import org.opensearch.common.annotation.PublicApi; +import org.opensearch.common.cache.CacheType; import org.opensearch.core.common.Strings; import org.opensearch.core.common.io.stream.StreamInput; import org.opensearch.core.common.io.stream.StreamOutput; @@ -43,6 +44,7 @@ import java.io.IOException; import java.util.Collections; import java.util.EnumSet; +import java.util.Set; /** * Common Stats Flags for OpenSearch @@ -63,6 +65,9 @@ public class CommonStatsFlags implements Writeable, Cloneable { private boolean includeUnloadedSegments = false; private boolean includeAllShardIndexingPressureTrackers = false; private boolean includeOnlyTopIndexingPressureMetrics = false; + // Used for metric CACHE_STATS, to determine which caches to report stats for + private EnumSet includeCaches = EnumSet.noneOf(CacheType.class); + private String[] levels; /** * @param flags flags to set. If no flags are supplied, default flags will be set. @@ -96,6 +101,11 @@ public CommonStatsFlags(StreamInput in) throws IOException { includeAllShardIndexingPressureTrackers = in.readBoolean(); includeOnlyTopIndexingPressureMetrics = in.readBoolean(); } + // TODO: change from V_3_0_0 to V_2_14_0 on main after backport to 2.x + if (in.getVersion().onOrAfter(Version.V_2_14_0)) { + includeCaches = in.readEnumSet(CacheType.class); + levels = in.readStringArray(); + } } @Override @@ -120,6 +130,11 @@ public void writeTo(StreamOutput out) throws IOException { out.writeBoolean(includeAllShardIndexingPressureTrackers); out.writeBoolean(includeOnlyTopIndexingPressureMetrics); } + // TODO: change from V_3_0_0 to V_2_14_0 on main after backport to 2.x + if (out.getVersion().onOrAfter(Version.V_2_14_0)) { + out.writeEnumSet(includeCaches); + out.writeStringArrayNullable(levels); + } } /** @@ -134,6 +149,8 @@ public CommonStatsFlags all() { includeUnloadedSegments = false; includeAllShardIndexingPressureTrackers = false; includeOnlyTopIndexingPressureMetrics = false; + includeCaches = EnumSet.noneOf(CacheType.class); + levels = null; return this; } @@ -149,6 +166,8 @@ public CommonStatsFlags clear() { includeUnloadedSegments = false; includeAllShardIndexingPressureTrackers = false; includeOnlyTopIndexingPressureMetrics = false; + includeCaches = EnumSet.noneOf(CacheType.class); + levels = null; return this; } @@ -160,6 +179,14 @@ public Flag[] getFlags() { return flags.toArray(new Flag[flags.size()]); } + public Set getIncludeCaches() { + return includeCaches; + } + + public String[] getLevels() { + return levels; + } + /** * Sets specific search group stats to retrieve the stats for. Mainly affects search * when enabled. @@ -215,6 +242,21 @@ public CommonStatsFlags includeOnlyTopIndexingPressureMetrics(boolean includeOnl return this; } + public CommonStatsFlags includeCacheType(CacheType cacheType) { + includeCaches.add(cacheType); + return this; + } + + public CommonStatsFlags includeAllCacheTypes() { + includeCaches = EnumSet.allOf(CacheType.class); + return this; + } + + public CommonStatsFlags setLevels(String[] inputLevels) { + levels = inputLevels; + return this; + } + public boolean includeUnloadedSegments() { return this.includeUnloadedSegments; } diff --git a/server/src/main/java/org/opensearch/common/cache/CacheType.java b/server/src/main/java/org/opensearch/common/cache/CacheType.java index c5aeb7cd1fa40..eee6204ac5412 100644 --- a/server/src/main/java/org/opensearch/common/cache/CacheType.java +++ b/server/src/main/java/org/opensearch/common/cache/CacheType.java @@ -10,20 +10,52 @@ import org.opensearch.common.annotation.ExperimentalApi; +import java.util.Collections; +import java.util.HashMap; +import java.util.Map; +import java.util.Set; + /** * Cache types available within OpenSearch. */ @ExperimentalApi public enum CacheType { - INDICES_REQUEST_CACHE("indices.requests.cache"); + INDICES_REQUEST_CACHE("indices.requests.cache", "request_cache"); private final String settingPrefix; + private final String value; // The value displayed for this cache type in stats API responses + + private static final Map valuesMap; + static { + Map values = new HashMap<>(); + for (CacheType cacheType : values()) { + values.put(cacheType.value, cacheType); + } + valuesMap = Collections.unmodifiableMap(values); + } - CacheType(String settingPrefix) { + CacheType(String settingPrefix, String representation) { this.settingPrefix = settingPrefix; + this.value = representation; } public String getSettingPrefix() { return settingPrefix; } + + public String getValue() { + return value; + } + + public static CacheType getByValue(String value) { + CacheType result = valuesMap.get(value); + if (result == null) { + throw new IllegalArgumentException("No CacheType with value = " + value); + } + return result; + } + + public static Set allValues() { + return valuesMap.keySet(); + } } diff --git a/server/src/main/java/org/opensearch/common/cache/ICache.java b/server/src/main/java/org/opensearch/common/cache/ICache.java index 8d8964abf0829..bc69ccee0c2fb 100644 --- a/server/src/main/java/org/opensearch/common/cache/ICache.java +++ b/server/src/main/java/org/opensearch/common/cache/ICache.java @@ -45,7 +45,13 @@ public interface ICache extends Closeable { void refresh(); - ImmutableCacheStatsHolder stats(); + // Return all stats without aggregation. + default ImmutableCacheStatsHolder stats() { + return stats(null); + } + + // Return stats aggregated by the provided levels. If levels is null, do not aggregate and return all stats. + ImmutableCacheStatsHolder stats(String[] levels); /** * Factory to create objects. diff --git a/server/src/main/java/org/opensearch/common/cache/service/CacheService.java b/server/src/main/java/org/opensearch/common/cache/service/CacheService.java index b6710e5e4b424..01da78ecec52e 100644 --- a/server/src/main/java/org/opensearch/common/cache/service/CacheService.java +++ b/server/src/main/java/org/opensearch/common/cache/service/CacheService.java @@ -8,10 +8,12 @@ package org.opensearch.common.cache.service; +import org.opensearch.action.admin.indices.stats.CommonStatsFlags; import org.opensearch.common.annotation.ExperimentalApi; import org.opensearch.common.cache.CacheType; import org.opensearch.common.cache.ICache; import org.opensearch.common.cache.settings.CacheSettings; +import org.opensearch.common.cache.stats.ImmutableCacheStatsHolder; import org.opensearch.common.cache.store.OpenSearchOnHeapCache; import org.opensearch.common.cache.store.config.CacheConfig; import org.opensearch.common.settings.Setting; @@ -20,6 +22,8 @@ import java.util.HashMap; import java.util.Map; +import java.util.SortedMap; +import java.util.TreeMap; /** * Service responsible to create caches. @@ -62,4 +66,12 @@ public ICache createCache(CacheConfig config, CacheType cache cacheTypeMap.put(cacheType, iCache); return iCache; } + + public NodeCacheStats stats(CommonStatsFlags flags) { + final SortedMap statsMap = new TreeMap<>(); + for (CacheType type : cacheTypeMap.keySet()) { + statsMap.put(type, cacheTypeMap.get(type).stats(flags.getLevels())); + } + return new NodeCacheStats(statsMap, flags); + } } diff --git a/server/src/main/java/org/opensearch/common/cache/service/NodeCacheStats.java b/server/src/main/java/org/opensearch/common/cache/service/NodeCacheStats.java new file mode 100644 index 0000000000000..07c75eab34194 --- /dev/null +++ b/server/src/main/java/org/opensearch/common/cache/service/NodeCacheStats.java @@ -0,0 +1,80 @@ +/* + * SPDX-License-Identifier: Apache-2.0 + * + * The OpenSearch Contributors require contributions made to + * this file be licensed under the Apache-2.0 license or a + * compatible open source license. + */ + +package org.opensearch.common.cache.service; + +import org.opensearch.action.admin.indices.stats.CommonStatsFlags; +import org.opensearch.common.annotation.ExperimentalApi; +import org.opensearch.common.cache.CacheType; +import org.opensearch.common.cache.stats.ImmutableCacheStatsHolder; +import org.opensearch.core.common.io.stream.StreamInput; +import org.opensearch.core.common.io.stream.StreamOutput; +import org.opensearch.core.common.io.stream.Writeable; +import org.opensearch.core.xcontent.ToXContentFragment; +import org.opensearch.core.xcontent.XContentBuilder; + +import java.io.IOException; +import java.util.Objects; +import java.util.SortedMap; + +/** + * A class creating XContent responses to cache stats API requests. + * + * @opensearch.experimental + */ +@ExperimentalApi +public class NodeCacheStats implements ToXContentFragment, Writeable { + // Use SortedMap to force consistent ordering of caches in API responses + private final SortedMap statsByCache; + private final CommonStatsFlags flags; + + public NodeCacheStats(SortedMap statsByCache, CommonStatsFlags flags) { + this.statsByCache = statsByCache; + this.flags = flags; + } + + public NodeCacheStats(StreamInput in) throws IOException { + this.flags = new CommonStatsFlags(in); + this.statsByCache = in.readOrderedMap(i -> i.readEnum(CacheType.class), ImmutableCacheStatsHolder::new); + } + + @Override + public void writeTo(StreamOutput out) throws IOException { + flags.writeTo(out); + out.writeMap(statsByCache, StreamOutput::writeEnum, (o, immutableCacheStatsHolder) -> immutableCacheStatsHolder.writeTo(o)); + } + + @Override + public XContentBuilder toXContent(XContentBuilder builder, Params params) throws IOException { + for (CacheType type : statsByCache.keySet()) { + if (flags.getIncludeCaches().contains(type)) { + builder.startObject(type.getValue()); + statsByCache.get(type).toXContent(builder, params); + builder.endObject(); + } + } + return builder; + } + + @Override + public boolean equals(Object o) { + if (o == null) { + return false; + } + if (o.getClass() != NodeCacheStats.class) { + return false; + } + NodeCacheStats other = (NodeCacheStats) o; + return statsByCache.equals(other.statsByCache) && flags.getIncludeCaches().equals(other.flags.getIncludeCaches()); + } + + @Override + public int hashCode() { + return Objects.hash(statsByCache, flags); + } +} diff --git a/server/src/main/java/org/opensearch/common/cache/stats/CacheStats.java b/server/src/main/java/org/opensearch/common/cache/stats/CacheStats.java index b0cb66b56b70d..93fa1ff7fcddf 100644 --- a/server/src/main/java/org/opensearch/common/cache/stats/CacheStats.java +++ b/server/src/main/java/org/opensearch/common/cache/stats/CacheStats.java @@ -20,9 +20,9 @@ public class CacheStats { CounterMetric misses; CounterMetric evictions; CounterMetric sizeInBytes; - CounterMetric entries; + CounterMetric items; - public CacheStats(long hits, long misses, long evictions, long sizeInBytes, long entries) { + public CacheStats(long hits, long misses, long evictions, long sizeInBytes, long items) { this.hits = new CounterMetric(); this.hits.inc(hits); this.misses = new CounterMetric(); @@ -31,8 +31,8 @@ public CacheStats(long hits, long misses, long evictions, long sizeInBytes, long this.evictions.inc(evictions); this.sizeInBytes = new CounterMetric(); this.sizeInBytes.inc(sizeInBytes); - this.entries = new CounterMetric(); - this.entries.inc(entries); + this.items = new CounterMetric(); + this.items.inc(items); } public CacheStats() { @@ -44,33 +44,33 @@ private void internalAdd(long otherHits, long otherMisses, long otherEvictions, this.misses.inc(otherMisses); this.evictions.inc(otherEvictions); this.sizeInBytes.inc(otherSizeInBytes); - this.entries.inc(otherEntries); + this.items.inc(otherEntries); } public void add(CacheStats other) { if (other == null) { return; } - internalAdd(other.getHits(), other.getMisses(), other.getEvictions(), other.getSizeInBytes(), other.getEntries()); + internalAdd(other.getHits(), other.getMisses(), other.getEvictions(), other.getSizeInBytes(), other.getItems()); } public void add(ImmutableCacheStats snapshot) { if (snapshot == null) { return; } - internalAdd(snapshot.getHits(), snapshot.getMisses(), snapshot.getEvictions(), snapshot.getSizeInBytes(), snapshot.getEntries()); + internalAdd(snapshot.getHits(), snapshot.getMisses(), snapshot.getEvictions(), snapshot.getSizeInBytes(), snapshot.getItems()); } public void subtract(ImmutableCacheStats other) { if (other == null) { return; } - internalAdd(-other.getHits(), -other.getMisses(), -other.getEvictions(), -other.getSizeInBytes(), -other.getEntries()); + internalAdd(-other.getHits(), -other.getMisses(), -other.getEvictions(), -other.getSizeInBytes(), -other.getItems()); } @Override public int hashCode() { - return Objects.hash(hits.count(), misses.count(), evictions.count(), sizeInBytes.count(), entries.count()); + return Objects.hash(hits.count(), misses.count(), evictions.count(), sizeInBytes.count(), items.count()); } public void incrementHits() { @@ -93,12 +93,12 @@ public void decrementSizeInBytes(long amount) { sizeInBytes.dec(amount); } - public void incrementEntries() { - entries.inc(); + public void incrementItems() { + items.inc(); } - public void decrementEntries() { - entries.dec(); + public void decrementItems() { + items.dec(); } public long getHits() { @@ -117,16 +117,16 @@ public long getSizeInBytes() { return sizeInBytes.count(); } - public long getEntries() { - return entries.count(); + public long getItems() { + return items.count(); } public void resetSizeAndEntries() { sizeInBytes = new CounterMetric(); - entries = new CounterMetric(); + items = new CounterMetric(); } public ImmutableCacheStats immutableSnapshot() { - return new ImmutableCacheStats(hits.count(), misses.count(), evictions.count(), sizeInBytes.count(), entries.count()); + return new ImmutableCacheStats(hits.count(), misses.count(), evictions.count(), sizeInBytes.count(), items.count()); } } diff --git a/server/src/main/java/org/opensearch/common/cache/stats/CacheStatsHolder.java b/server/src/main/java/org/opensearch/common/cache/stats/CacheStatsHolder.java index a1cfb8d806af3..27cb7679efb0c 100644 --- a/server/src/main/java/org/opensearch/common/cache/stats/CacheStatsHolder.java +++ b/server/src/main/java/org/opensearch/common/cache/stats/CacheStatsHolder.java @@ -24,9 +24,9 @@ public interface CacheStatsHolder { void decrementSizeInBytes(List dimensionValues, long amountBytes); - void incrementEntries(List dimensionValues); + void incrementItems(List dimensionValues); - void decrementEntries(List dimensionValues); + void decrementItems(List dimensionValues); void reset(); @@ -34,5 +34,5 @@ public interface CacheStatsHolder { void removeDimensions(List dimensionValues); - ImmutableCacheStatsHolder getImmutableCacheStatsHolder(); + ImmutableCacheStatsHolder getImmutableCacheStatsHolder(String[] levels); } diff --git a/server/src/main/java/org/opensearch/common/cache/stats/DefaultCacheStatsHolder.java b/server/src/main/java/org/opensearch/common/cache/stats/DefaultCacheStatsHolder.java index ad943e0b2ed1a..0162a10487eba 100644 --- a/server/src/main/java/org/opensearch/common/cache/stats/DefaultCacheStatsHolder.java +++ b/server/src/main/java/org/opensearch/common/cache/stats/DefaultCacheStatsHolder.java @@ -12,7 +12,6 @@ import java.util.HashMap; import java.util.List; import java.util.Map; -import java.util.TreeMap; import java.util.concurrent.ConcurrentHashMap; import java.util.concurrent.locks.Lock; import java.util.concurrent.locks.ReentrantLock; @@ -41,9 +40,12 @@ public class DefaultCacheStatsHolder implements CacheStatsHolder { // To avoid sync problems, obtain a lock before creating or removing nodes in the stats tree. // No lock is needed to edit stats on existing nodes. private final Lock lock = new ReentrantLock(); + // The name of the cache type using these stats + private final String storeName; - public DefaultCacheStatsHolder(List dimensionNames) { + public DefaultCacheStatsHolder(List dimensionNames, String storeName) { this.dimensionNames = Collections.unmodifiableList(dimensionNames); + this.storeName = storeName; this.statsRoot = new Node("", true); // The root node has the empty string as its dimension value } @@ -81,13 +83,13 @@ public void decrementSizeInBytes(List dimensionValues, long amountBytes) } @Override - public void incrementEntries(List dimensionValues) { - internalIncrement(dimensionValues, Node::incrementEntries, true); + public void incrementItems(List dimensionValues) { + internalIncrement(dimensionValues, Node::incrementItems, true); } @Override - public void decrementEntries(List dimensionValues) { - internalIncrement(dimensionValues, Node::decrementEntries, false); + public void decrementItems(List dimensionValues) { + internalIncrement(dimensionValues, Node::decrementItems, false); } /** @@ -163,11 +165,12 @@ private boolean internalIncrementHelper( } /** - * Produce an immutable version of these stats. + * Produce an immutable version of these stats, aggregated according to levels. + * If levels is null, do not aggregate and return an immutable version of the original tree. */ @Override - public ImmutableCacheStatsHolder getImmutableCacheStatsHolder() { - return new ImmutableCacheStatsHolder(statsRoot.snapshot(), dimensionNames); + public ImmutableCacheStatsHolder getImmutableCacheStatsHolder(String[] levels) { + return new ImmutableCacheStatsHolder(this.statsRoot, levels, dimensionNames, storeName); } @Override @@ -260,16 +263,16 @@ void decrementSizeInBytes(long amountBytes) { this.stats.decrementSizeInBytes(amountBytes); } - void incrementEntries() { - this.stats.incrementEntries(); + void incrementItems() { + this.stats.incrementItems(); } - void decrementEntries() { - this.stats.decrementEntries(); + void decrementItems() { + this.stats.decrementItems(); } long getEntries() { - return this.stats.getEntries(); + return this.stats.getItems(); } ImmutableCacheStats getImmutableStats() { @@ -291,16 +294,5 @@ Node getChild(String dimensionValue) { Node createChild(String dimensionValue, boolean createMapInChild) { return children.computeIfAbsent(dimensionValue, (key) -> new Node(dimensionValue, createMapInChild)); } - - ImmutableCacheStatsHolder.Node snapshot() { - TreeMap snapshotChildren = null; - if (!children.isEmpty()) { - snapshotChildren = new TreeMap<>(); - for (Node child : children.values()) { - snapshotChildren.put(child.getDimensionValue(), child.snapshot()); - } - } - return new ImmutableCacheStatsHolder.Node(dimensionValue, snapshotChildren, getImmutableStats()); - } } } diff --git a/server/src/main/java/org/opensearch/common/cache/stats/ImmutableCacheStats.java b/server/src/main/java/org/opensearch/common/cache/stats/ImmutableCacheStats.java index 7549490fd6b74..dbd78a2584f9c 100644 --- a/server/src/main/java/org/opensearch/common/cache/stats/ImmutableCacheStats.java +++ b/server/src/main/java/org/opensearch/common/cache/stats/ImmutableCacheStats.java @@ -12,6 +12,9 @@ import org.opensearch.core.common.io.stream.StreamInput; import org.opensearch.core.common.io.stream.StreamOutput; import org.opensearch.core.common.io.stream.Writeable; +import org.opensearch.core.common.unit.ByteSizeValue; +import org.opensearch.core.xcontent.ToXContent; +import org.opensearch.core.xcontent.XContentBuilder; import java.io.IOException; import java.util.Objects; @@ -22,19 +25,19 @@ * @opensearch.experimental */ @ExperimentalApi -public class ImmutableCacheStats implements Writeable { // TODO: Make this extend ToXContent (in API PR) +public class ImmutableCacheStats implements Writeable, ToXContent { private final long hits; private final long misses; private final long evictions; private final long sizeInBytes; - private final long entries; + private final long items; - public ImmutableCacheStats(long hits, long misses, long evictions, long sizeInBytes, long entries) { + public ImmutableCacheStats(long hits, long misses, long evictions, long sizeInBytes, long items) { this.hits = hits; this.misses = misses; this.evictions = evictions; this.sizeInBytes = sizeInBytes; - this.entries = entries; + this.items = items; } public ImmutableCacheStats(StreamInput in) throws IOException { @@ -47,7 +50,7 @@ public static ImmutableCacheStats addSnapshots(ImmutableCacheStats s1, Immutable s1.misses + s2.misses, s1.evictions + s2.evictions, s1.sizeInBytes + s2.sizeInBytes, - s1.entries + s2.entries + s1.items + s2.items ); } @@ -67,8 +70,8 @@ public long getSizeInBytes() { return sizeInBytes; } - public long getEntries() { - return entries; + public long getItems() { + return items; } @Override @@ -77,7 +80,7 @@ public void writeTo(StreamOutput out) throws IOException { out.writeVLong(misses); out.writeVLong(evictions); out.writeVLong(sizeInBytes); - out.writeVLong(entries); + out.writeVLong(items); } @Override @@ -93,11 +96,34 @@ public boolean equals(Object o) { && (misses == other.misses) && (evictions == other.evictions) && (sizeInBytes == other.sizeInBytes) - && (entries == other.entries); + && (items == other.items); } @Override public int hashCode() { - return Objects.hash(hits, misses, evictions, sizeInBytes, entries); + return Objects.hash(hits, misses, evictions, sizeInBytes, items); + } + + @Override + public XContentBuilder toXContent(XContentBuilder builder, Params params) throws IOException { + // We don't write the header in CacheStatsResponse's toXContent, because it doesn't know the name of aggregation it's part of + builder.humanReadableField(Fields.SIZE_IN_BYTES, Fields.SIZE, new ByteSizeValue(sizeInBytes)); + builder.field(Fields.EVICTIONS, evictions); + builder.field(Fields.HIT_COUNT, hits); + builder.field(Fields.MISS_COUNT, misses); + builder.field(Fields.ITEM_COUNT, items); + return builder; + } + + /** + * Field names used to write the values in this object to XContent. + */ + public static final class Fields { + public static final String SIZE = "size"; + public static final String SIZE_IN_BYTES = "size_in_bytes"; + public static final String EVICTIONS = "evictions"; + public static final String HIT_COUNT = "hit_count"; + public static final String MISS_COUNT = "miss_count"; + public static final String ITEM_COUNT = "item_count"; } } diff --git a/server/src/main/java/org/opensearch/common/cache/stats/ImmutableCacheStatsHolder.java b/server/src/main/java/org/opensearch/common/cache/stats/ImmutableCacheStatsHolder.java index 12e325046d83b..92383626236b8 100644 --- a/server/src/main/java/org/opensearch/common/cache/stats/ImmutableCacheStatsHolder.java +++ b/server/src/main/java/org/opensearch/common/cache/stats/ImmutableCacheStatsHolder.java @@ -9,11 +9,21 @@ package org.opensearch.common.cache.stats; import org.opensearch.common.annotation.ExperimentalApi; +import org.opensearch.core.common.io.stream.StreamInput; +import org.opensearch.core.common.io.stream.StreamOutput; +import org.opensearch.core.common.io.stream.Writeable; +import org.opensearch.core.xcontent.ToXContent; +import org.opensearch.core.xcontent.XContentBuilder; +import java.io.IOException; +import java.util.ArrayList; +import java.util.Arrays; import java.util.Collections; -import java.util.HashMap; import java.util.List; import java.util.Map; +import java.util.Objects; +import java.util.SortedMap; +import java.util.Stack; import java.util.TreeMap; /** @@ -23,15 +33,102 @@ */ @ExperimentalApi -public class ImmutableCacheStatsHolder { // TODO: extends Writeable, ToXContent - // An immutable snapshot of a stats within a CacheStatsHolder, containing all the stats maintained by the cache. +public class ImmutableCacheStatsHolder implements Writeable, ToXContent { + // Root node of immutable snapshot of stats within a CacheStatsHolder, containing all the stats maintained by the cache. // Pkg-private for testing. final Node statsRoot; + // The dimension names for each level in this tree. final List dimensionNames; + // The name of the cache type producing these stats. Returned in API response. + final String storeName; + public static String STORE_NAME_FIELD = "store_name"; - public ImmutableCacheStatsHolder(Node statsRoot, List dimensionNames) { - this.statsRoot = statsRoot; - this.dimensionNames = dimensionNames; + // Values used for serializing/deserializing the tree. + private static final String SERIALIZATION_CHILDREN_OPEN_BRACKET = "<"; + private static final String SERIALIZATION_CHILDREN_CLOSE_BRACKET = ">"; + private static final String SERIALIZATION_BEGIN_NODE = "_"; + private static final String SERIALIZATION_DONE = "end"; + + ImmutableCacheStatsHolder( + DefaultCacheStatsHolder.Node originalStatsRoot, + String[] levels, + List originalDimensionNames, + String storeName + ) { + // Aggregate from the original CacheStatsHolder according to the levels passed in. + // The dimension names for this immutable snapshot should reflect the levels we aggregate in the snapshot + this.dimensionNames = filterLevels(levels, originalDimensionNames); + this.storeName = storeName; + this.statsRoot = aggregateByLevels(originalStatsRoot, originalDimensionNames); + makeNodeUnmodifiable(statsRoot); + } + + public ImmutableCacheStatsHolder(StreamInput in) throws IOException { + this.dimensionNames = List.of(in.readStringArray()); + this.storeName = in.readString(); + this.statsRoot = deserializeTree(in); + makeNodeUnmodifiable(statsRoot); + } + + public void writeTo(StreamOutput out) throws IOException { + out.writeStringArray(dimensionNames.toArray(new String[0])); + out.writeString(storeName); + writeNode(statsRoot, out); + out.writeString(SERIALIZATION_DONE); + } + + private void writeNode(Node node, StreamOutput out) throws IOException { + out.writeString(SERIALIZATION_BEGIN_NODE); + out.writeString(node.dimensionValue); + out.writeBoolean(node.children.isEmpty()); // Write whether this is a leaf node + node.stats.writeTo(out); + + out.writeString(SERIALIZATION_CHILDREN_OPEN_BRACKET); + for (Map.Entry entry : node.children.entrySet()) { + out.writeString(entry.getKey()); + writeNode(entry.getValue(), out); + } + out.writeString(SERIALIZATION_CHILDREN_CLOSE_BRACKET); + } + + private Node deserializeTree(StreamInput in) throws IOException { + final Stack stack = new Stack<>(); + in.readString(); // Read and discard SERIALIZATION_BEGIN_NODE for the root node + Node statsRoot = readSingleNode(in); + Node current = statsRoot; + stack.push(statsRoot); + String nextSymbol = in.readString(); + while (!nextSymbol.equals(SERIALIZATION_DONE)) { + switch (nextSymbol) { + case SERIALIZATION_CHILDREN_OPEN_BRACKET: + stack.push(current); + break; + case SERIALIZATION_CHILDREN_CLOSE_BRACKET: + stack.pop(); + break; + case SERIALIZATION_BEGIN_NODE: + current = readSingleNode(in); + stack.peek().children.put(current.dimensionValue, current); + } + nextSymbol = in.readString(); + } + return statsRoot; + } + + private Node readSingleNode(StreamInput in) throws IOException { + String dimensionValue = in.readString(); + boolean isLeafNode = in.readBoolean(); + ImmutableCacheStats stats = new ImmutableCacheStats(in); + return new Node(dimensionValue, isLeafNode, stats); + } + + private void makeNodeUnmodifiable(Node node) { + if (!node.children.isEmpty()) { + node.children = Collections.unmodifiableSortedMap(node.children); + } + for (Node child : node.children.values()) { + makeNodeUnmodifiable(child); + } } public ImmutableCacheStats getTotalStats() { @@ -54,8 +151,8 @@ public long getTotalSizeInBytes() { return getTotalStats().getSizeInBytes(); } - public long getTotalEntries() { - return getTotalStats().getEntries(); + public long getTotalItems() { + return getTotalStats().getItems(); } public ImmutableCacheStats getStatsForDimensionValues(List dimensionValues) { @@ -69,23 +166,179 @@ public ImmutableCacheStats getStatsForDimensionValues(List dimensionValu return current.stats; } - // A similar class to CacheStatsHolder.Node, which uses an ordered TreeMap and holds immutable CacheStatsSnapshot as its stats. + /** + * Returns a new tree containing the stats aggregated by the levels passed in. + * The new tree only has dimensions matching the levels passed in. + * The levels passed in must be in the proper order, as they would be in the output of filterLevels(). + */ + Node aggregateByLevels(DefaultCacheStatsHolder.Node originalStatsRoot, List originalDimensionNames) { + Node newRoot = new Node("", false, originalStatsRoot.getImmutableStats()); + for (DefaultCacheStatsHolder.Node child : originalStatsRoot.children.values()) { + aggregateByLevelsHelper(newRoot, child, originalDimensionNames, 0); + } + return newRoot; + } + + /** + * Because we may have to combine nodes that have the same dimension name, I don't think there's a clean way to aggregate + * fully recursively while also passing in a completed map of children nodes before constructing the parent node. + * For this reason, in this function we have to build the new tree top down rather than bottom up. + * We use private methods allowing us to add children to/increment the stats for an existing node. + * This should be ok because the resulting tree is unmodifiable after creation in the constructor. + * + * @param allDimensions the list of all dimensions present in the original CacheStatsHolder which produced + * the CacheStatsHolder.Node object we are traversing. + */ + private void aggregateByLevelsHelper( + Node parentInNewTree, + DefaultCacheStatsHolder.Node currentInOriginalTree, + List allDimensions, + int depth + ) { + if (dimensionNames.contains(allDimensions.get(depth))) { + // If this node is in a level we want to aggregate, create a new dimension node with the same value and stats, and connect it to + // the last parent node in the new tree. If it already exists, increment it instead. + String dimensionValue = currentInOriginalTree.getDimensionValue(); + Node nodeInNewTree = parentInNewTree.children.get(dimensionValue); + if (nodeInNewTree == null) { + // Create new node with stats matching the node from the original tree + int indexOfLastLevel = allDimensions.indexOf(dimensionNames.get(dimensionNames.size() - 1)); + boolean isLeafNode = depth == indexOfLastLevel; // If this is the last level we aggregate, the new node should be a leaf + // node + nodeInNewTree = new Node(dimensionValue, isLeafNode, currentInOriginalTree.getImmutableStats()); + parentInNewTree.addChild(dimensionValue, nodeInNewTree); + } else { + // Otherwise increment existing stats + nodeInNewTree.incrementStats(currentInOriginalTree.getImmutableStats()); + } + // Finally set the parent node to be this node for the next callers of this function + parentInNewTree = nodeInNewTree; + } + + for (Map.Entry childEntry : currentInOriginalTree.children.entrySet()) { + DefaultCacheStatsHolder.Node child = childEntry.getValue(); + aggregateByLevelsHelper(parentInNewTree, child, allDimensions, depth + 1); + } + } + + /** + * Filters out levels that aren't in dimensionNames, and orders the resulting list to match the order in dimensionNames. + * Unrecognized levels are ignored. + */ + private List filterLevels(String[] levels, List originalDimensionNames) { + if (levels == null) { + return originalDimensionNames; + } + List levelsList = Arrays.asList(levels); + List filtered = new ArrayList<>(); + for (String dimensionName : originalDimensionNames) { + if (levelsList.contains(dimensionName)) { + filtered.add(dimensionName); + } + } + return filtered; + } + + @Override + public XContentBuilder toXContent(XContentBuilder builder, Params params) throws IOException { + // Always show total stats, regardless of levels + getTotalStats().toXContent(builder, params); + + List filteredLevels = filterLevels(getLevels(params), dimensionNames); + assert filteredLevels.equals(dimensionNames); + if (!filteredLevels.isEmpty()) { + // Depth -1 corresponds to the dummy root node + toXContentForLevels(-1, statsRoot, builder, params); + } + + // Also add the store name for the cache that produced the stats + builder.field(STORE_NAME_FIELD, storeName); + return builder; + } + + private void toXContentForLevels(int depth, Node current, XContentBuilder builder, Params params) throws IOException { + if (depth >= 0) { + builder.startObject(current.dimensionValue); + } + + if (depth == dimensionNames.size() - 1) { + // This is a leaf node + current.getStats().toXContent(builder, params); + } else { + builder.startObject(dimensionNames.get(depth + 1)); + for (Node nextNode : current.children.values()) { + toXContentForLevels(depth + 1, nextNode, builder, params); + } + builder.endObject(); + } + + if (depth >= 0) { + builder.endObject(); + } + } + + private String[] getLevels(Params params) { + String levels = params.param("level"); + if (levels == null) { + return null; + } + return levels.split(","); + } + + @Override + public boolean equals(Object o) { + if (o == null || o.getClass() != ImmutableCacheStatsHolder.class) { + return false; + } + ImmutableCacheStatsHolder other = (ImmutableCacheStatsHolder) o; + if (!dimensionNames.equals(other.dimensionNames) || !storeName.equals(other.storeName)) { + return false; + } + return equalsHelper(statsRoot, other.getStatsRoot()); + } + + private boolean equalsHelper(Node thisNode, Node otherNode) { + if (otherNode == null) { + return false; + } + if (!thisNode.getStats().equals(otherNode.getStats())) { + return false; + } + boolean allChildrenMatch = true; + for (String childValue : thisNode.getChildren().keySet()) { + allChildrenMatch = equalsHelper(thisNode.children.get(childValue), otherNode.children.get(childValue)); + if (!allChildrenMatch) { + return false; + } + } + return allChildrenMatch; + } + + @Override + public int hashCode() { + // Should be sufficient to hash based on the total stats value (found in the root node) + return Objects.hash(statsRoot.stats, dimensionNames); + } + + // A similar class to CacheStatsHolder.Node, which uses a SortedMap and holds immutable CacheStatsSnapshot as its stats. static class Node { private final String dimensionValue; - final Map children; // Map from dimensionValue to the Node for that dimension value + // Map from dimensionValue to the Node for that dimension value. Not final so we can set it to be unmodifiable before we are done in + // the constructor. + SortedMap children; // The stats for this node. If a leaf node, corresponds to the stats for this combination of dimensions; if not, // contains the sum of its children's stats. - private final ImmutableCacheStats stats; - private static final Map EMPTY_CHILDREN_MAP = new HashMap<>(); + private ImmutableCacheStats stats; + private static final SortedMap EMPTY_CHILDREN_MAP = Collections.unmodifiableSortedMap(new TreeMap<>()); - Node(String dimensionValue, TreeMap snapshotChildren, ImmutableCacheStats stats) { + private Node(String dimensionValue, boolean isLeafNode, ImmutableCacheStats stats) { this.dimensionValue = dimensionValue; this.stats = stats; - if (snapshotChildren == null) { + if (isLeafNode) { this.children = EMPTY_CHILDREN_MAP; } else { - this.children = Collections.unmodifiableMap(snapshotChildren); + this.children = new TreeMap<>(); } } @@ -100,12 +353,18 @@ public ImmutableCacheStats getStats() { public String getDimensionValue() { return dimensionValue; } + + private void addChild(String dimensionValue, Node child) { + this.children.putIfAbsent(dimensionValue, child); + } + + private void incrementStats(ImmutableCacheStats toIncrement) { + stats = ImmutableCacheStats.addSnapshots(stats, toIncrement); + } } // pkg-private for testing Node getStatsRoot() { return statsRoot; } - - // TODO (in API PR): Produce XContent based on aggregateByLevels() } diff --git a/server/src/main/java/org/opensearch/common/cache/stats/NoopCacheStatsHolder.java b/server/src/main/java/org/opensearch/common/cache/stats/NoopCacheStatsHolder.java index b7debbd8a8eab..9cb69a3a0a365 100644 --- a/server/src/main/java/org/opensearch/common/cache/stats/NoopCacheStatsHolder.java +++ b/server/src/main/java/org/opensearch/common/cache/stats/NoopCacheStatsHolder.java @@ -16,11 +16,12 @@ * A singleton instance is used for memory purposes. */ public class NoopCacheStatsHolder implements CacheStatsHolder { + private static final String dummyStoreName = "noop_store"; private static final NoopCacheStatsHolder singletonInstance = new NoopCacheStatsHolder(); private static final ImmutableCacheStatsHolder immutableCacheStatsHolder; static { - ImmutableCacheStatsHolder.Node dummyNode = new ImmutableCacheStatsHolder.Node("", null, new ImmutableCacheStats(0, 0, 0, 0, 0)); - immutableCacheStatsHolder = new ImmutableCacheStatsHolder(dummyNode, List.of()); + DefaultCacheStatsHolder.Node dummyNode = new DefaultCacheStatsHolder.Node("", false); + immutableCacheStatsHolder = new ImmutableCacheStatsHolder(dummyNode, new String[0], List.of(), dummyStoreName); } private NoopCacheStatsHolder() {} @@ -45,10 +46,10 @@ public void incrementSizeInBytes(List dimensionValues, long amountBytes) public void decrementSizeInBytes(List dimensionValues, long amountBytes) {} @Override - public void incrementEntries(List dimensionValues) {} + public void incrementItems(List dimensionValues) {} @Override - public void decrementEntries(List dimensionValues) {} + public void decrementItems(List dimensionValues) {} @Override public void reset() {} @@ -62,7 +63,7 @@ public long count() { public void removeDimensions(List dimensionValues) {} @Override - public ImmutableCacheStatsHolder getImmutableCacheStatsHolder() { + public ImmutableCacheStatsHolder getImmutableCacheStatsHolder(String[] levels) { return immutableCacheStatsHolder; } } diff --git a/server/src/main/java/org/opensearch/common/cache/store/OpenSearchOnHeapCache.java b/server/src/main/java/org/opensearch/common/cache/store/OpenSearchOnHeapCache.java index 35c951e240a3a..f4cf9f3a8fa61 100644 --- a/server/src/main/java/org/opensearch/common/cache/store/OpenSearchOnHeapCache.java +++ b/server/src/main/java/org/opensearch/common/cache/store/OpenSearchOnHeapCache.java @@ -69,7 +69,7 @@ public OpenSearchOnHeapCache(Builder builder) { if (useNoopStats) { this.cacheStatsHolder = NoopCacheStatsHolder.getInstance(); } else { - this.cacheStatsHolder = new DefaultCacheStatsHolder(dimensionNames); + this.cacheStatsHolder = new DefaultCacheStatsHolder(dimensionNames, OpenSearchOnHeapCacheFactory.NAME); } this.removalListener = builder.getRemovalListener(); this.weigher = builder.getWeigher(); @@ -89,7 +89,7 @@ public V get(ICacheKey key) { @Override public void put(ICacheKey key, V value) { cache.put(key, value); - cacheStatsHolder.incrementEntries(key.dimensions); + cacheStatsHolder.incrementItems(key.dimensions); cacheStatsHolder.incrementSizeInBytes(key.dimensions, weigher.applyAsLong(key, value)); } @@ -100,7 +100,7 @@ public V computeIfAbsent(ICacheKey key, LoadAwareCacheLoader, V> cacheStatsHolder.incrementHits(key.dimensions); } else { cacheStatsHolder.incrementMisses(key.dimensions); - cacheStatsHolder.incrementEntries(key.dimensions); + cacheStatsHolder.incrementItems(key.dimensions); cacheStatsHolder.incrementSizeInBytes(key.dimensions, cache.getWeigher().applyAsLong(key, value)); } return value; @@ -141,14 +141,14 @@ public void refresh() { public void close() {} @Override - public ImmutableCacheStatsHolder stats() { - return cacheStatsHolder.getImmutableCacheStatsHolder(); + public ImmutableCacheStatsHolder stats(String[] levels) { + return cacheStatsHolder.getImmutableCacheStatsHolder(levels); } @Override public void onRemoval(RemovalNotification, V> notification) { removalListener.onRemoval(notification); - cacheStatsHolder.decrementEntries(notification.getKey().dimensions); + cacheStatsHolder.decrementItems(notification.getKey().dimensions); cacheStatsHolder.decrementSizeInBytes( notification.getKey().dimensions, cache.getWeigher().applyAsLong(notification.getKey(), notification.getValue()) diff --git a/server/src/main/java/org/opensearch/index/cache/request/ShardRequestCache.java b/server/src/main/java/org/opensearch/index/cache/request/ShardRequestCache.java index bb35a09ccab46..c08ff73e3d6b2 100644 --- a/server/src/main/java/org/opensearch/index/cache/request/ShardRequestCache.java +++ b/server/src/main/java/org/opensearch/index/cache/request/ShardRequestCache.java @@ -32,7 +32,6 @@ package org.opensearch.index.cache.request; -import org.apache.lucene.util.Accountable; import org.opensearch.common.annotation.PublicApi; import org.opensearch.common.metrics.CounterMetric; import org.opensearch.core.common.bytes.BytesReference; @@ -62,18 +61,15 @@ public void onMiss() { missCount.inc(); } - public void onCached(Accountable key, BytesReference value) { - totalMetric.inc(key.ramBytesUsed() + value.ramBytesUsed()); + public void onCached(long keyRamBytesUsed, BytesReference value) { + totalMetric.inc(keyRamBytesUsed + value.ramBytesUsed()); } - public void onRemoval(Accountable key, BytesReference value, boolean evicted) { + public void onRemoval(long keyRamBytesUsed, BytesReference value, boolean evicted) { if (evicted) { evictionsMetric.inc(); } - long dec = 0; - if (key != null) { - dec += key.ramBytesUsed(); - } + long dec = keyRamBytesUsed; if (value != null) { dec += value.ramBytesUsed(); } diff --git a/server/src/main/java/org/opensearch/indices/AbstractIndexShardCacheEntity.java b/server/src/main/java/org/opensearch/indices/AbstractIndexShardCacheEntity.java index bb1201cb910a9..6b4c53654d871 100644 --- a/server/src/main/java/org/opensearch/indices/AbstractIndexShardCacheEntity.java +++ b/server/src/main/java/org/opensearch/indices/AbstractIndexShardCacheEntity.java @@ -32,6 +32,7 @@ package org.opensearch.indices; +import org.opensearch.common.cache.ICacheKey; import org.opensearch.common.cache.RemovalNotification; import org.opensearch.common.cache.RemovalReason; import org.opensearch.core.common.bytes.BytesReference; @@ -51,8 +52,8 @@ abstract class AbstractIndexShardCacheEntity implements IndicesRequestCache.Cach protected abstract ShardRequestCache stats(); @Override - public final void onCached(IndicesRequestCache.Key key, BytesReference value) { - stats().onCached(key, value); + public final void onCached(ICacheKey key, BytesReference value) { + stats().onCached(getRamBytesUsedInKey(key), value); } @Override @@ -66,7 +67,19 @@ public final void onMiss() { } @Override - public final void onRemoval(RemovalNotification notification) { - stats().onRemoval(notification.getKey(), notification.getValue(), notification.getRemovalReason() == RemovalReason.EVICTED); + public final void onRemoval(RemovalNotification, BytesReference> notification) { + stats().onRemoval( + getRamBytesUsedInKey(notification.getKey()), + notification.getValue(), + notification.getRemovalReason() == RemovalReason.EVICTED + ); + } + + private long getRamBytesUsedInKey(ICacheKey key) { + long innerKeyRamBytesUsed = 0; + if (key.key != null) { + innerKeyRamBytesUsed = key.key.ramBytesUsed(); + } + return key.ramBytesUsed(innerKeyRamBytesUsed); } } diff --git a/server/src/main/java/org/opensearch/indices/IndicesRequestCache.java b/server/src/main/java/org/opensearch/indices/IndicesRequestCache.java index d5651cc01e654..8aa83820ffffd 100644 --- a/server/src/main/java/org/opensearch/indices/IndicesRequestCache.java +++ b/server/src/main/java/org/opensearch/indices/IndicesRequestCache.java @@ -233,14 +233,9 @@ public void onRemoval(RemovalNotification, BytesReference> notifi // shards as part of request cache. // Pass a new removal notification containing Key rather than ICacheKey to the CacheEntity for backwards compatibility. Key key = notification.getKey().key; - RemovalNotification newNotification = new RemovalNotification<>( - key, - notification.getValue(), - notification.getRemovalReason() - ); - cacheEntityLookup.apply(key.shardId).ifPresent(entity -> entity.onRemoval(newNotification)); + cacheEntityLookup.apply(key.shardId).ifPresent(entity -> entity.onRemoval(notification)); CleanupKey cleanupKey = new CleanupKey(cacheEntityLookup.apply(key.shardId).orElse(null), key.readerCacheKeyId); - cacheCleanupManager.updateStaleCountOnEntryRemoval(cleanupKey, newNotification); + cacheCleanupManager.updateStaleCountOnEntryRemoval(cleanupKey, notification); } private ICacheKey getICacheKey(Key key) { @@ -331,7 +326,7 @@ public boolean isLoaded() { @Override public BytesReference load(ICacheKey key) throws Exception { BytesReference value = loader.get(); - entity.onCached(key.key, value); + entity.onCached(key, value); loaded = true; return value; } @@ -345,7 +340,7 @@ interface CacheEntity extends Accountable { /** * Called after the value was loaded. */ - void onCached(Key key, BytesReference value); + void onCached(ICacheKey key, BytesReference value); /** * Returns true iff the resource behind this entity is still open ie. @@ -372,7 +367,7 @@ interface CacheEntity extends Accountable { /** * Called when this entity instance is removed */ - void onRemoval(RemovalNotification notification); + void onRemoval(RemovalNotification, BytesReference> notification); } @@ -561,7 +556,10 @@ private void updateStaleCountOnCacheInsert(CleanupKey cleanupKey) { * @param cleanupKey the CleanupKey that has been evicted from the cache * @param notification RemovalNotification of the cache entry evicted */ - private void updateStaleCountOnEntryRemoval(CleanupKey cleanupKey, RemovalNotification notification) { + private void updateStaleCountOnEntryRemoval( + CleanupKey cleanupKey, + RemovalNotification, BytesReference> notification + ) { if (notification.getRemovalReason() == RemovalReason.REPLACED) { // The reason of the notification is REPLACED when a cache entry's value is updated, since replacing an entry // does not affect the staleness count, we skip such notifications. diff --git a/server/src/main/java/org/opensearch/node/Node.java b/server/src/main/java/org/opensearch/node/Node.java index 696496abf255e..f536f8c73b2df 100644 --- a/server/src/main/java/org/opensearch/node/Node.java +++ b/server/src/main/java/org/opensearch/node/Node.java @@ -1188,7 +1188,8 @@ protected Node( resourceUsageCollectorService, segmentReplicationStatsTracker, repositoryService, - admissionControlService + admissionControlService, + cacheService ); final SearchService searchService = newSearchService( diff --git a/server/src/main/java/org/opensearch/node/NodeService.java b/server/src/main/java/org/opensearch/node/NodeService.java index 15cc8f3d20bb3..1eb38ea63ad5a 100644 --- a/server/src/main/java/org/opensearch/node/NodeService.java +++ b/server/src/main/java/org/opensearch/node/NodeService.java @@ -41,6 +41,7 @@ import org.opensearch.cluster.routing.WeightedRoutingStats; import org.opensearch.cluster.service.ClusterService; import org.opensearch.common.Nullable; +import org.opensearch.common.cache.service.CacheService; import org.opensearch.common.settings.Settings; import org.opensearch.common.settings.SettingsFilter; import org.opensearch.common.util.io.IOUtils; @@ -99,6 +100,7 @@ public class NodeService implements Closeable { private final RepositoriesService repositoriesService; private final AdmissionControlService admissionControlService; private final SegmentReplicationStatsTracker segmentReplicationStatsTracker; + private final CacheService cacheService; NodeService( Settings settings, @@ -125,7 +127,8 @@ public class NodeService implements Closeable { ResourceUsageCollectorService resourceUsageCollectorService, SegmentReplicationStatsTracker segmentReplicationStatsTracker, RepositoriesService repositoriesService, - AdmissionControlService admissionControlService + AdmissionControlService admissionControlService, + CacheService cacheService ) { this.settings = settings; this.threadPool = threadPool; @@ -154,6 +157,7 @@ public class NodeService implements Closeable { clusterService.addStateApplier(ingestService); clusterService.addStateApplier(searchPipelineService); this.segmentReplicationStatsTracker = segmentReplicationStatsTracker; + this.cacheService = cacheService; } public NodeInfo info( @@ -236,7 +240,8 @@ public NodeStats stats( boolean resourceUsageStats, boolean segmentReplicationTrackerStats, boolean repositoriesStats, - boolean admissionControl + boolean admissionControl, + boolean cacheService ) { // for indices stats we want to include previous allocated shards stats as well (it will // only be applied to the sensible ones to use, like refresh/merge/flush/indexing stats) @@ -268,7 +273,8 @@ public NodeStats stats( searchPipelineStats ? this.searchPipelineService.stats() : null, segmentReplicationTrackerStats ? this.segmentReplicationStatsTracker.getTotalRejectionStats() : null, repositoriesStats ? this.repositoriesService.getRepositoriesStats() : null, - admissionControl ? this.admissionControlService.stats() : null + admissionControl ? this.admissionControlService.stats() : null, + cacheService ? this.cacheService.stats(indices) : null ); } diff --git a/server/src/main/java/org/opensearch/rest/action/admin/cluster/RestNodesStatsAction.java b/server/src/main/java/org/opensearch/rest/action/admin/cluster/RestNodesStatsAction.java index 66b9afda06eb6..267bfde576dec 100644 --- a/server/src/main/java/org/opensearch/rest/action/admin/cluster/RestNodesStatsAction.java +++ b/server/src/main/java/org/opensearch/rest/action/admin/cluster/RestNodesStatsAction.java @@ -36,6 +36,7 @@ import org.opensearch.action.admin.indices.stats.CommonStatsFlags; import org.opensearch.action.admin.indices.stats.CommonStatsFlags.Flag; import org.opensearch.client.node.NodeClient; +import org.opensearch.common.cache.CacheType; import org.opensearch.core.common.Strings; import org.opensearch.rest.BaseRestHandler; import org.opensearch.rest.RestRequest; @@ -175,6 +176,25 @@ public RestChannelConsumer prepareRequest(final RestRequest request, final NodeC nodesStatsRequest.indices(flags); } + } else if (metrics.contains("caches")) { + // Extract the list of caches we want to get stats for from the submetrics (which we get from index_metric) + Set cacheMetrics = Strings.tokenizeByCommaToSet(request.param("index_metric", "_all")); + CommonStatsFlags cacheFlags = new CommonStatsFlags(); + cacheFlags.clear(); + if (cacheMetrics.contains("_all")) { + cacheFlags.includeAllCacheTypes(); + } else { + for (String cacheName : cacheMetrics) { + try { + cacheFlags.includeCacheType(CacheType.getByValue(cacheName)); + } catch (IllegalArgumentException e) { + throw new IllegalArgumentException( + unrecognized(request, Set.of(cacheName), CacheType.allValues(), "cache type") + ); + } + } + } + nodesStatsRequest.indices(cacheFlags); } else if (request.hasParam("index_metric")) { throw new IllegalArgumentException( String.format( @@ -209,6 +229,10 @@ public RestChannelConsumer prepareRequest(final RestRequest request, final NodeC nodesStatsRequest.indices().includeOnlyTopIndexingPressureMetrics(request.paramAsBoolean("top", false)); } + // If no levels are passed in this results in an empty array. + String[] levels = Strings.splitStringByCommaToArray(request.param("level")); + nodesStatsRequest.indices().setLevels(levels); + return channel -> client.admin().cluster().nodesStats(nodesStatsRequest, new NodesResponseRestListener<>(channel)); } diff --git a/server/src/test/java/org/opensearch/action/admin/cluster/node/stats/NodeStatsTests.java b/server/src/test/java/org/opensearch/action/admin/cluster/node/stats/NodeStatsTests.java index af76292e83d61..e6dd2c7671c33 100644 --- a/server/src/test/java/org/opensearch/action/admin/cluster/node/stats/NodeStatsTests.java +++ b/server/src/test/java/org/opensearch/action/admin/cluster/node/stats/NodeStatsTests.java @@ -42,6 +42,12 @@ import org.opensearch.cluster.routing.WeightedRoutingStats; import org.opensearch.cluster.service.ClusterManagerThrottlingStats; import org.opensearch.cluster.service.ClusterStateStats; +import org.opensearch.common.cache.CacheType; +import org.opensearch.common.cache.service.NodeCacheStats; +import org.opensearch.common.cache.stats.CacheStats; +import org.opensearch.common.cache.stats.DefaultCacheStatsHolder; +import org.opensearch.common.cache.stats.DefaultCacheStatsHolderTests; +import org.opensearch.common.cache.stats.ImmutableCacheStatsHolder; import org.opensearch.common.io.stream.BytesStreamOutput; import org.opensearch.common.metrics.OperationStats; import org.opensearch.common.settings.ClusterSettings; @@ -89,6 +95,7 @@ import java.util.Iterator; import java.util.List; import java.util.Map; +import java.util.TreeMap; import java.util.function.Function; import java.util.stream.Collectors; import java.util.stream.StreamSupport; @@ -577,6 +584,13 @@ public void testSerialization() throws IOException { deserializedAdmissionControllerStats.getRejectionCount().get(AdmissionControlActionType.INDEXING.getType()) ); } + NodeCacheStats nodeCacheStats = nodeStats.getNodeCacheStats(); + NodeCacheStats deserializedNodeCacheStats = deserializedNodeStats.getNodeCacheStats(); + if (nodeCacheStats == null) { + assertNull(deserializedNodeCacheStats); + } else { + assertEquals(nodeCacheStats, deserializedNodeCacheStats); + } } } } @@ -927,6 +941,39 @@ public void apply(String action, AdmissionControlActionType admissionControlActi NodeIndicesStats indicesStats = getNodeIndicesStats(remoteStoreStats); + NodeCacheStats nodeCacheStats = null; + if (frequently()) { + int numIndices = randomIntBetween(1, 10); + int numShardsPerIndex = randomIntBetween(1, 50); + + List dimensionNames = List.of("index", "shard", "tier"); + DefaultCacheStatsHolder statsHolder = new DefaultCacheStatsHolder(dimensionNames, "dummyStoreName"); + for (int indexNum = 0; indexNum < numIndices; indexNum++) { + String indexName = "index" + indexNum; + for (int shardNum = 0; shardNum < numShardsPerIndex; shardNum++) { + String shardName = "[" + indexName + "][" + shardNum + "]"; + for (String tierName : new String[] { "dummy_tier_1", "dummy_tier_2" }) { + List dimensionValues = List.of(indexName, shardName, tierName); + CacheStats toIncrement = new CacheStats(randomInt(20), randomInt(20), randomInt(20), randomInt(20), randomInt(20)); + DefaultCacheStatsHolderTests.populateStatsHolderFromStatsValueMap( + statsHolder, + Map.of(dimensionValues, toIncrement) + ); + } + } + } + CommonStatsFlags flags = new CommonStatsFlags(); + for (CacheType cacheType : CacheType.values()) { + if (frequently()) { + flags.includeCacheType(cacheType); + } + } + ImmutableCacheStatsHolder cacheStats = statsHolder.getImmutableCacheStatsHolder(dimensionNames.toArray(new String[0])); + TreeMap cacheStatsMap = new TreeMap<>(); + cacheStatsMap.put(CacheType.INDICES_REQUEST_CACHE, cacheStats); + nodeCacheStats = new NodeCacheStats(cacheStatsMap, flags); + } + // TODO: Only remote_store based aspects of NodeIndicesStats are being tested here. // It is possible to test other metrics in NodeIndicesStats as well since it extends Writeable now return new NodeStats( @@ -957,7 +1004,8 @@ public void apply(String action, AdmissionControlActionType admissionControlActi null, segmentReplicationRejectionStats, null, - admissionControlStats + admissionControlStats, + nodeCacheStats ); } diff --git a/server/src/test/java/org/opensearch/cluster/DiskUsageTests.java b/server/src/test/java/org/opensearch/cluster/DiskUsageTests.java index ff47ec3015697..5539dd26dd52d 100644 --- a/server/src/test/java/org/opensearch/cluster/DiskUsageTests.java +++ b/server/src/test/java/org/opensearch/cluster/DiskUsageTests.java @@ -194,6 +194,7 @@ public void testFillDiskUsage() { null, null, null, + null, null ), new NodeStats( @@ -224,6 +225,7 @@ public void testFillDiskUsage() { null, null, null, + null, null ), new NodeStats( @@ -254,6 +256,7 @@ public void testFillDiskUsage() { null, null, null, + null, null ) ); @@ -315,6 +318,7 @@ public void testFillDiskUsageSomeInvalidValues() { null, null, null, + null, null ), new NodeStats( @@ -345,6 +349,7 @@ public void testFillDiskUsageSomeInvalidValues() { null, null, null, + null, null ), new NodeStats( @@ -375,6 +380,7 @@ public void testFillDiskUsageSomeInvalidValues() { null, null, null, + null, null ) ); diff --git a/server/src/test/java/org/opensearch/common/cache/stats/DefaultCacheStatsHolderTests.java b/server/src/test/java/org/opensearch/common/cache/stats/DefaultCacheStatsHolderTests.java index fe12673bb9f6a..c6e8252ddf806 100644 --- a/server/src/test/java/org/opensearch/common/cache/stats/DefaultCacheStatsHolderTests.java +++ b/server/src/test/java/org/opensearch/common/cache/stats/DefaultCacheStatsHolderTests.java @@ -22,9 +22,11 @@ import java.util.concurrent.CountDownLatch; public class DefaultCacheStatsHolderTests extends OpenSearchTestCase { + private final String storeName = "dummy_store"; + public void testAddAndGet() throws Exception { List dimensionNames = List.of("dim1", "dim2", "dim3", "dim4"); - DefaultCacheStatsHolder cacheStatsHolder = new DefaultCacheStatsHolder(dimensionNames); + DefaultCacheStatsHolder cacheStatsHolder = new DefaultCacheStatsHolder(dimensionNames, storeName); Map> usedDimensionValues = DefaultCacheStatsHolderTests.getUsedDimensionValues(cacheStatsHolder, 10); Map, CacheStats> expected = DefaultCacheStatsHolderTests.populateStats( cacheStatsHolder, @@ -58,7 +60,7 @@ public void testAddAndGet() throws Exception { public void testReset() throws Exception { List dimensionNames = List.of("dim1", "dim2"); - DefaultCacheStatsHolder cacheStatsHolder = new DefaultCacheStatsHolder(dimensionNames); + DefaultCacheStatsHolder cacheStatsHolder = new DefaultCacheStatsHolder(dimensionNames, storeName); Map> usedDimensionValues = getUsedDimensionValues(cacheStatsHolder, 10); Map, CacheStats> expected = populateStats(cacheStatsHolder, usedDimensionValues, 100, 10); @@ -67,7 +69,7 @@ public void testReset() throws Exception { for (List dimensionValues : expected.keySet()) { CacheStats originalCounter = expected.get(dimensionValues); originalCounter.sizeInBytes = new CounterMetric(); - originalCounter.entries = new CounterMetric(); + originalCounter.items = new CounterMetric(); DefaultCacheStatsHolder.Node node = getNode(dimensionValues, cacheStatsHolder.getStatsRoot()); ImmutableCacheStats actual = node.getImmutableStats(); @@ -77,7 +79,7 @@ public void testReset() throws Exception { public void testDropStatsForDimensions() throws Exception { List dimensionNames = List.of("dim1", "dim2"); - DefaultCacheStatsHolder cacheStatsHolder = new DefaultCacheStatsHolder(dimensionNames); + DefaultCacheStatsHolder cacheStatsHolder = new DefaultCacheStatsHolder(dimensionNames, storeName); // Create stats for the following dimension sets List> populatedStats = List.of(List.of("A1", "B1"), List.of("A2", "B2"), List.of("A2", "B3")); @@ -113,20 +115,20 @@ public void testDropStatsForDimensions() throws Exception { public void testCount() throws Exception { List dimensionNames = List.of("dim1", "dim2"); - DefaultCacheStatsHolder cacheStatsHolder = new DefaultCacheStatsHolder(dimensionNames); + DefaultCacheStatsHolder cacheStatsHolder = new DefaultCacheStatsHolder(dimensionNames, storeName); Map> usedDimensionValues = getUsedDimensionValues(cacheStatsHolder, 10); Map, CacheStats> expected = populateStats(cacheStatsHolder, usedDimensionValues, 100, 10); long expectedCount = 0L; for (CacheStats counter : expected.values()) { - expectedCount += counter.getEntries(); + expectedCount += counter.getItems(); } assertEquals(expectedCount, cacheStatsHolder.count()); } public void testConcurrentRemoval() throws Exception { List dimensionNames = List.of("dim1", "dim2"); - DefaultCacheStatsHolder cacheStatsHolder = new DefaultCacheStatsHolder(dimensionNames); + DefaultCacheStatsHolder cacheStatsHolder = new DefaultCacheStatsHolder(dimensionNames, storeName); // Create stats for the following dimension sets List> populatedStats = List.of(List.of("A1", "B1"), List.of("A2", "B2"), List.of("A2", "B3")); @@ -191,32 +193,46 @@ static Map, CacheStats> populateStats( int numDistinctValuePairs, int numRepetitionsPerValue ) throws InterruptedException { + return populateStats(List.of(cacheStatsHolder), usedDimensionValues, numDistinctValuePairs, numRepetitionsPerValue); + } + + static Map, CacheStats> populateStats( + List cacheStatsHolders, + Map> usedDimensionValues, + int numDistinctValuePairs, + int numRepetitionsPerValue + ) throws InterruptedException { + for (DefaultCacheStatsHolder statsHolder : cacheStatsHolders) { + assertEquals(cacheStatsHolders.get(0).getDimensionNames(), statsHolder.getDimensionNames()); + } Map, CacheStats> expected = new ConcurrentHashMap<>(); Thread[] threads = new Thread[numDistinctValuePairs]; CountDownLatch countDownLatch = new CountDownLatch(numDistinctValuePairs); Random rand = Randomness.get(); List> dimensionsForThreads = new ArrayList<>(); for (int i = 0; i < numDistinctValuePairs; i++) { - dimensionsForThreads.add(getRandomDimList(cacheStatsHolder.getDimensionNames(), usedDimensionValues, true, rand)); + dimensionsForThreads.add(getRandomDimList(cacheStatsHolders.get(0).getDimensionNames(), usedDimensionValues, true, rand)); int finalI = i; threads[i] = new Thread(() -> { Random threadRand = Randomness.get(); List dimensions = dimensionsForThreads.get(finalI); expected.computeIfAbsent(dimensions, (key) -> new CacheStats()); - for (int j = 0; j < numRepetitionsPerValue; j++) { - CacheStats statsToInc = new CacheStats( - threadRand.nextInt(10), - threadRand.nextInt(10), - threadRand.nextInt(10), - threadRand.nextInt(5000), - threadRand.nextInt(10) - ); - expected.get(dimensions).hits.inc(statsToInc.getHits()); - expected.get(dimensions).misses.inc(statsToInc.getMisses()); - expected.get(dimensions).evictions.inc(statsToInc.getEvictions()); - expected.get(dimensions).sizeInBytes.inc(statsToInc.getSizeInBytes()); - expected.get(dimensions).entries.inc(statsToInc.getEntries()); - DefaultCacheStatsHolderTests.populateStatsHolderFromStatsValueMap(cacheStatsHolder, Map.of(dimensions, statsToInc)); + for (DefaultCacheStatsHolder cacheStatsHolder : cacheStatsHolders) { + for (int j = 0; j < numRepetitionsPerValue; j++) { + CacheStats statsToInc = new CacheStats( + threadRand.nextInt(10), + threadRand.nextInt(10), + threadRand.nextInt(10), + threadRand.nextInt(5000), + threadRand.nextInt(10) + ); + expected.get(dimensions).hits.inc(statsToInc.getHits()); + expected.get(dimensions).misses.inc(statsToInc.getMisses()); + expected.get(dimensions).evictions.inc(statsToInc.getEvictions()); + expected.get(dimensions).sizeInBytes.inc(statsToInc.getSizeInBytes()); + expected.get(dimensions).items.inc(statsToInc.getItems()); + DefaultCacheStatsHolderTests.populateStatsHolderFromStatsValueMap(cacheStatsHolder, Map.of(dimensions, statsToInc)); + } } countDownLatch.countDown(); }); @@ -270,7 +286,10 @@ private void assertSumOfChildrenStats(DefaultCacheStatsHolder.Node current) { } } - static void populateStatsHolderFromStatsValueMap(DefaultCacheStatsHolder cacheStatsHolder, Map, CacheStats> statsMap) { + public static void populateStatsHolderFromStatsValueMap( + DefaultCacheStatsHolder cacheStatsHolder, + Map, CacheStats> statsMap + ) { for (Map.Entry, CacheStats> entry : statsMap.entrySet()) { CacheStats stats = entry.getValue(); List dims = entry.getKey(); @@ -284,8 +303,8 @@ static void populateStatsHolderFromStatsValueMap(DefaultCacheStatsHolder cacheSt cacheStatsHolder.incrementEvictions(dims); } cacheStatsHolder.incrementSizeInBytes(dims, stats.getSizeInBytes()); - for (int i = 0; i < stats.getEntries(); i++) { - cacheStatsHolder.incrementEntries(dims); + for (int i = 0; i < stats.getItems(); i++) { + cacheStatsHolder.incrementItems(dims); } } } diff --git a/server/src/test/java/org/opensearch/common/cache/stats/ImmutableCacheStatsHolderTests.java b/server/src/test/java/org/opensearch/common/cache/stats/ImmutableCacheStatsHolderTests.java index 5a4511fa654dd..46483e92b76bf 100644 --- a/server/src/test/java/org/opensearch/common/cache/stats/ImmutableCacheStatsHolderTests.java +++ b/server/src/test/java/org/opensearch/common/cache/stats/ImmutableCacheStatsHolderTests.java @@ -8,16 +8,74 @@ package org.opensearch.common.cache.stats; +import org.opensearch.common.io.stream.BytesStreamOutput; +import org.opensearch.common.xcontent.XContentFactory; +import org.opensearch.common.xcontent.XContentHelper; +import org.opensearch.core.common.bytes.BytesReference; +import org.opensearch.core.common.io.stream.BytesStreamInput; +import org.opensearch.core.xcontent.MediaTypeRegistry; +import org.opensearch.core.xcontent.ToXContent; +import org.opensearch.core.xcontent.XContentBuilder; import org.opensearch.test.OpenSearchTestCase; +import java.util.ArrayList; +import java.util.HashMap; import java.util.List; import java.util.Map; +import java.util.function.BiConsumer; public class ImmutableCacheStatsHolderTests extends OpenSearchTestCase { + private final String storeName = "dummy_store"; + + public void testSerialization() throws Exception { + List dimensionNames = List.of("dim1", "dim2", "dim3"); + DefaultCacheStatsHolder statsHolder = new DefaultCacheStatsHolder(dimensionNames, storeName); + Map> usedDimensionValues = DefaultCacheStatsHolderTests.getUsedDimensionValues(statsHolder, 10); + DefaultCacheStatsHolderTests.populateStats(statsHolder, usedDimensionValues, 100, 10); + ImmutableCacheStatsHolder stats = statsHolder.getImmutableCacheStatsHolder(null); + assertNotEquals(0, stats.getStatsRoot().children.size()); + + BytesStreamOutput os = new BytesStreamOutput(); + stats.writeTo(os); + BytesStreamInput is = new BytesStreamInput(BytesReference.toBytes(os.bytes())); + ImmutableCacheStatsHolder deserialized = new ImmutableCacheStatsHolder(is); + + assertEquals(stats, deserialized); + + // also test empty dimension stats + ImmutableCacheStatsHolder emptyDims = statsHolder.getImmutableCacheStatsHolder(new String[] {}); + assertEquals(0, emptyDims.getStatsRoot().children.size()); + assertEquals(stats.getTotalStats(), emptyDims.getTotalStats()); + + os = new BytesStreamOutput(); + emptyDims.writeTo(os); + is = new BytesStreamInput(BytesReference.toBytes(os.bytes())); + deserialized = new ImmutableCacheStatsHolder(is); + + assertEquals(emptyDims, deserialized); + } + + public void testEquals() throws Exception { + List dimensionNames = List.of("dim1", "dim2", "dim3"); + DefaultCacheStatsHolder statsHolder = new DefaultCacheStatsHolder(dimensionNames, storeName); + DefaultCacheStatsHolder differentStoreNameStatsHolder = new DefaultCacheStatsHolder(dimensionNames, "nonMatchingStoreName"); + DefaultCacheStatsHolder nonMatchingStatsHolder = new DefaultCacheStatsHolder(dimensionNames, storeName); + Map> usedDimensionValues = DefaultCacheStatsHolderTests.getUsedDimensionValues(statsHolder, 10); + DefaultCacheStatsHolderTests.populateStats(List.of(statsHolder, differentStoreNameStatsHolder), usedDimensionValues, 100, 10); + DefaultCacheStatsHolderTests.populateStats(nonMatchingStatsHolder, usedDimensionValues, 100, 10); + ImmutableCacheStatsHolder stats = statsHolder.getImmutableCacheStatsHolder(null); + + ImmutableCacheStatsHolder secondStats = statsHolder.getImmutableCacheStatsHolder(null); + assertEquals(stats, secondStats); + ImmutableCacheStatsHolder nonMatchingStats = nonMatchingStatsHolder.getImmutableCacheStatsHolder(null); + assertNotEquals(stats, nonMatchingStats); + ImmutableCacheStatsHolder differentStoreNameStats = differentStoreNameStatsHolder.getImmutableCacheStatsHolder(null); + assertNotEquals(stats, differentStoreNameStats); + } public void testGet() throws Exception { List dimensionNames = List.of("dim1", "dim2", "dim3", "dim4"); - DefaultCacheStatsHolder cacheStatsHolder = new DefaultCacheStatsHolder(dimensionNames); + DefaultCacheStatsHolder cacheStatsHolder = new DefaultCacheStatsHolder(dimensionNames, storeName); Map> usedDimensionValues = DefaultCacheStatsHolderTests.getUsedDimensionValues(cacheStatsHolder, 10); Map, CacheStats> expected = DefaultCacheStatsHolderTests.populateStats( cacheStatsHolder, @@ -25,7 +83,7 @@ public void testGet() throws Exception { 1000, 10 ); - ImmutableCacheStatsHolder stats = cacheStatsHolder.getImmutableCacheStatsHolder(); + ImmutableCacheStatsHolder stats = cacheStatsHolder.getImmutableCacheStatsHolder(dimensionNames.toArray(new String[0])); // test the value in the map is as expected for each distinct combination of values for (List dimensionValues : expected.keySet()) { @@ -52,23 +110,238 @@ public void testGet() throws Exception { assertEquals(expectedTotal.getMisses(), stats.getTotalMisses()); assertEquals(expectedTotal.getEvictions(), stats.getTotalEvictions()); assertEquals(expectedTotal.getSizeInBytes(), stats.getTotalSizeInBytes()); - assertEquals(expectedTotal.getEntries(), stats.getTotalEntries()); + assertEquals(expectedTotal.getItems(), stats.getTotalItems()); assertSumOfChildrenStats(stats.getStatsRoot()); } public void testEmptyDimsList() throws Exception { // If the dimension list is empty, the tree should have only the root node containing the total stats. - DefaultCacheStatsHolder cacheStatsHolder = new DefaultCacheStatsHolder(List.of()); + DefaultCacheStatsHolder cacheStatsHolder = new DefaultCacheStatsHolder(List.of(), storeName); Map> usedDimensionValues = DefaultCacheStatsHolderTests.getUsedDimensionValues(cacheStatsHolder, 100); DefaultCacheStatsHolderTests.populateStats(cacheStatsHolder, usedDimensionValues, 10, 100); - ImmutableCacheStatsHolder stats = cacheStatsHolder.getImmutableCacheStatsHolder(); + ImmutableCacheStatsHolder stats = cacheStatsHolder.getImmutableCacheStatsHolder(null); ImmutableCacheStatsHolder.Node statsRoot = stats.getStatsRoot(); assertEquals(0, statsRoot.children.size()); assertEquals(stats.getTotalStats(), statsRoot.getStats()); } + public void testAggregateByAllDimensions() throws Exception { + // Aggregating with all dimensions as levels should just give us the same values that were in the original map + List dimensionNames = List.of("dim1", "dim2", "dim3", "dim4"); + DefaultCacheStatsHolder statsHolder = new DefaultCacheStatsHolder(dimensionNames, storeName); + Map> usedDimensionValues = DefaultCacheStatsHolderTests.getUsedDimensionValues(statsHolder, 10); + Map, CacheStats> expected = DefaultCacheStatsHolderTests.populateStats(statsHolder, usedDimensionValues, 1000, 10); + ImmutableCacheStatsHolder stats = statsHolder.getImmutableCacheStatsHolder(dimensionNames.toArray(new String[0])); + + for (Map.Entry, CacheStats> expectedEntry : expected.entrySet()) { + List dimensionValues = new ArrayList<>(); + for (String dimValue : expectedEntry.getKey()) { + dimensionValues.add(dimValue); + } + assertEquals(expectedEntry.getValue().immutableSnapshot(), getNode(dimensionValues, stats.statsRoot).getStats()); + } + assertSumOfChildrenStats(stats.statsRoot); + } + + public void testAggregateBySomeDimensions() throws Exception { + List dimensionNames = List.of("dim1", "dim2", "dim3", "dim4"); + DefaultCacheStatsHolder statsHolder = new DefaultCacheStatsHolder(dimensionNames, storeName); + Map> usedDimensionValues = DefaultCacheStatsHolderTests.getUsedDimensionValues(statsHolder, 10); + Map, CacheStats> expected = DefaultCacheStatsHolderTests.populateStats(statsHolder, usedDimensionValues, 1000, 10); + + for (int i = 0; i < (1 << dimensionNames.size()); i++) { + // Test each combination of possible levels + List levels = new ArrayList<>(); + for (int nameIndex = 0; nameIndex < dimensionNames.size(); nameIndex++) { + if ((i & (1 << nameIndex)) != 0) { + levels.add(dimensionNames.get(nameIndex)); + } + } + + if (levels.size() == 0) { + // If we pass empty levels to CacheStatsHolder to aggregate by, we should only get a root node with the total stats in it + ImmutableCacheStatsHolder stats = statsHolder.getImmutableCacheStatsHolder(levels.toArray(new String[0])); + assertEquals(statsHolder.getStatsRoot().getImmutableStats(), stats.getStatsRoot().getStats()); + assertEquals(0, stats.getStatsRoot().children.size()); + } else { + ImmutableCacheStatsHolder stats = statsHolder.getImmutableCacheStatsHolder(levels.toArray(new String[0])); + Map, ImmutableCacheStatsHolder.Node> aggregatedLeafNodes = getAllLeafNodes(stats.statsRoot); + + for (Map.Entry, ImmutableCacheStatsHolder.Node> aggEntry : aggregatedLeafNodes.entrySet()) { + CacheStats expectedCounter = new CacheStats(); + for (List expectedDims : expected.keySet()) { + if (expectedDims.containsAll(aggEntry.getKey())) { + expectedCounter.add(expected.get(expectedDims)); + } + } + assertEquals(expectedCounter.immutableSnapshot(), aggEntry.getValue().getStats()); + } + assertSumOfChildrenStats(stats.statsRoot); + } + } + } + + public void testXContentForLevels() throws Exception { + List dimensionNames = List.of("A", "B", "C"); + + DefaultCacheStatsHolder statsHolder = new DefaultCacheStatsHolder(dimensionNames, storeName); + DefaultCacheStatsHolderTests.populateStatsHolderFromStatsValueMap( + statsHolder, + Map.of( + List.of("A1", "B1", "C1"), + new CacheStats(1, 1, 1, 1, 1), + List.of("A1", "B1", "C2"), + new CacheStats(2, 2, 2, 2, 2), + List.of("A1", "B2", "C1"), + new CacheStats(3, 3, 3, 3, 3), + List.of("A2", "B1", "C3"), + new CacheStats(4, 4, 4, 4, 4) + ) + ); + ImmutableCacheStatsHolder stats = statsHolder.getImmutableCacheStatsHolder(dimensionNames.toArray(new String[0])); + + XContentBuilder builder = XContentFactory.jsonBuilder(); + ToXContent.Params params = ToXContent.EMPTY_PARAMS; + + builder.startObject(); + stats.toXContent(builder, params); + builder.endObject(); + String resultString = builder.toString(); + Map result = XContentHelper.convertToMap(MediaTypeRegistry.JSON.xContent(), resultString, true); + + Map> fieldNamesMap = Map.of( + ImmutableCacheStats.Fields.SIZE_IN_BYTES, + (counter, value) -> counter.sizeInBytes.inc(value), + ImmutableCacheStats.Fields.EVICTIONS, + (counter, value) -> counter.evictions.inc(value), + ImmutableCacheStats.Fields.HIT_COUNT, + (counter, value) -> counter.hits.inc(value), + ImmutableCacheStats.Fields.MISS_COUNT, + (counter, value) -> counter.misses.inc(value), + ImmutableCacheStats.Fields.ITEM_COUNT, + (counter, value) -> counter.items.inc(value) + ); + + Map, ImmutableCacheStatsHolder.Node> leafNodes = getAllLeafNodes(stats.getStatsRoot()); + for (Map.Entry, ImmutableCacheStatsHolder.Node> entry : leafNodes.entrySet()) { + List xContentKeys = new ArrayList<>(); + for (int i = 0; i < dimensionNames.size(); i++) { + xContentKeys.add(dimensionNames.get(i)); + xContentKeys.add(entry.getKey().get(i)); + } + CacheStats counterFromXContent = new CacheStats(); + + for (Map.Entry> fieldNamesEntry : fieldNamesMap.entrySet()) { + List fullXContentKeys = new ArrayList<>(xContentKeys); + fullXContentKeys.add(fieldNamesEntry.getKey()); + int valueInXContent = (int) getValueFromNestedXContentMap(result, fullXContentKeys); + BiConsumer incrementer = fieldNamesEntry.getValue(); + incrementer.accept(counterFromXContent, valueInXContent); + } + + ImmutableCacheStats expected = entry.getValue().getStats(); + assertEquals(counterFromXContent.immutableSnapshot(), expected); + } + } + + public void testXContent() throws Exception { + // Tests logic of filtering levels out, logic for aggregating by those levels is already covered + List dimensionNames = List.of("A", "B", "C"); + DefaultCacheStatsHolder statsHolder = new DefaultCacheStatsHolder(dimensionNames, storeName); + Map> usedDimensionValues = DefaultCacheStatsHolderTests.getUsedDimensionValues(statsHolder, 10); + DefaultCacheStatsHolderTests.populateStats(statsHolder, usedDimensionValues, 100, 10); + + // If the levels in the params are empty or contains only unrecognized levels, we should only see the total stats and no level + // aggregation + List> levelsList = List.of(List.of(), List.of("D")); + for (List levels : levelsList) { + ImmutableCacheStatsHolder stats = statsHolder.getImmutableCacheStatsHolder(levels.toArray(new String[0])); + ToXContent.Params params = getLevelParams(levels); + XContentBuilder builder = XContentFactory.jsonBuilder(); + builder.startObject(); + stats.toXContent(builder, params); + builder.endObject(); + + String resultString = builder.toString(); + Map result = XContentHelper.convertToMap(MediaTypeRegistry.JSON.xContent(), resultString, true); + + assertTotalStatsPresentInXContentResponse(result); + // assert there are no other entries in the map besides these 6 + assertEquals(6, result.size()); + } + + // if we pass recognized levels in any order, alongside ignored unrecognized levels, we should see the above plus level aggregation + List levels = List.of("C", "A", "E"); + ImmutableCacheStatsHolder stats = statsHolder.getImmutableCacheStatsHolder(levels.toArray(new String[0])); + ToXContent.Params params = getLevelParams(levels); + XContentBuilder builder = XContentFactory.jsonBuilder(); + builder.startObject(); + stats.toXContent(builder, params); + builder.endObject(); + + String resultString = builder.toString(); + Map result = XContentHelper.convertToMap(MediaTypeRegistry.JSON.xContent(), resultString, true); + assertTotalStatsPresentInXContentResponse(result); + assertNotNull(result.get("A")); + assertEquals(7, result.size()); + } + + private void assertTotalStatsPresentInXContentResponse(Map result) { + // assert the total stats are present + assertNotEquals(0, (int) result.get(ImmutableCacheStats.Fields.SIZE_IN_BYTES)); + assertNotEquals(0, (int) result.get(ImmutableCacheStats.Fields.EVICTIONS)); + assertNotEquals(0, (int) result.get(ImmutableCacheStats.Fields.HIT_COUNT)); + assertNotEquals(0, (int) result.get(ImmutableCacheStats.Fields.MISS_COUNT)); + assertNotEquals(0, (int) result.get(ImmutableCacheStats.Fields.ITEM_COUNT)); + // assert the store name is present + assertEquals(storeName, (String) result.get(ImmutableCacheStatsHolder.STORE_NAME_FIELD)); + } + + private ToXContent.Params getLevelParams(List levels) { + Map paramMap = new HashMap<>(); + if (!levels.isEmpty()) { + paramMap.put("level", String.join(",", levels)); + } + return new ToXContent.MapParams(paramMap); + } + + public static Object getValueFromNestedXContentMap(Map xContentMap, List keys) { + Map current = xContentMap; + for (int i = 0; i < keys.size() - 1; i++) { + Object next = current.get(keys.get(i)); + if (next == null) { + return null; + } + current = (Map) next; + } + return current.get(keys.get(keys.size() - 1)); + } + + // Get a map from the list of dimension values to the corresponding leaf node. + private Map, ImmutableCacheStatsHolder.Node> getAllLeafNodes(ImmutableCacheStatsHolder.Node root) { + Map, ImmutableCacheStatsHolder.Node> result = new HashMap<>(); + getAllLeafNodesHelper(result, root, new ArrayList<>()); + return result; + } + + private void getAllLeafNodesHelper( + Map, ImmutableCacheStatsHolder.Node> result, + ImmutableCacheStatsHolder.Node current, + List pathToCurrent + ) { + if (current.children.isEmpty()) { + result.put(pathToCurrent, current); + } else { + for (Map.Entry entry : current.children.entrySet()) { + List newPath = new ArrayList<>(pathToCurrent); + newPath.add(entry.getKey()); + getAllLeafNodesHelper(result, entry.getValue(), newPath); + } + } + } + private ImmutableCacheStatsHolder.Node getNode(List dimensionValues, ImmutableCacheStatsHolder.Node root) { ImmutableCacheStatsHolder.Node current = root; for (String dimensionValue : dimensionValues) { diff --git a/server/src/test/java/org/opensearch/common/cache/store/OpenSearchOnHeapCacheTests.java b/server/src/test/java/org/opensearch/common/cache/store/OpenSearchOnHeapCacheTests.java index 00dbf43bc37be..3208fde306e5a 100644 --- a/server/src/test/java/org/opensearch/common/cache/store/OpenSearchOnHeapCacheTests.java +++ b/server/src/test/java/org/opensearch/common/cache/store/OpenSearchOnHeapCacheTests.java @@ -52,7 +52,7 @@ public void testStats() throws Exception { assertEquals(i + 1, cache.stats().getTotalMisses()); assertEquals(0, cache.stats().getTotalHits()); - assertEquals(Math.min(maxKeys, i + 1), cache.stats().getTotalEntries()); + assertEquals(Math.min(maxKeys, i + 1), cache.stats().getTotalItems()); assertEquals(Math.min(maxKeys, i + 1) * keyValueSize, cache.stats().getTotalSizeInBytes()); assertEquals(Math.max(0, i + 1 - maxKeys), cache.stats().getTotalEvictions()); } @@ -63,7 +63,7 @@ public void testStats() throws Exception { assertEquals(numAdded, cache.stats().getTotalMisses()); assertEquals(numHits, cache.stats().getTotalHits()); - assertEquals(maxKeys, cache.stats().getTotalEntries()); + assertEquals(maxKeys, cache.stats().getTotalItems()); assertEquals(maxKeys * keyValueSize, cache.stats().getTotalSizeInBytes()); assertEquals(numEvicted, cache.stats().getTotalEvictions()); } @@ -75,7 +75,7 @@ public void testStats() throws Exception { assertEquals(numAdded, cache.stats().getTotalMisses()); assertEquals(maxKeys, cache.stats().getTotalHits()); - assertEquals(maxKeys - numInvalidated, cache.stats().getTotalEntries()); + assertEquals(maxKeys - numInvalidated, cache.stats().getTotalItems()); assertEquals((maxKeys - numInvalidated) * keyValueSize, cache.stats().getTotalSizeInBytes()); assertEquals(numEvicted, cache.stats().getTotalEvictions()); } diff --git a/server/src/test/java/org/opensearch/indices/IndicesRequestCacheTests.java b/server/src/test/java/org/opensearch/indices/IndicesRequestCacheTests.java index fc306f7c595d6..9e2c33998abd6 100644 --- a/server/src/test/java/org/opensearch/indices/IndicesRequestCacheTests.java +++ b/server/src/test/java/org/opensearch/indices/IndicesRequestCacheTests.java @@ -878,7 +878,7 @@ public void testClosingIndexWipesStats() throws Exception { assertNotNull(snapshot); // check the values are not empty by confirming entries != 0, this should always be true since the missed value is loaded // into the cache - assertNotEquals(0, snapshot.getEntries()); + assertNotEquals(0, snapshot.getItems()); } } @@ -902,7 +902,7 @@ public void testClosingIndexWipesStats() throws Exception { assertNotNull(snapshot); // check the values are not empty by confirming entries != 0, this should always be true since the missed value is loaded // into the cache - assertNotEquals(0, snapshot.getEntries()); + assertNotEquals(0, snapshot.getItems()); } } diff --git a/test/framework/src/main/java/org/opensearch/cluster/MockInternalClusterInfoService.java b/test/framework/src/main/java/org/opensearch/cluster/MockInternalClusterInfoService.java index 1ad6083074025..35ca5d80aeb4e 100644 --- a/test/framework/src/main/java/org/opensearch/cluster/MockInternalClusterInfoService.java +++ b/test/framework/src/main/java/org/opensearch/cluster/MockInternalClusterInfoService.java @@ -124,7 +124,8 @@ List adjustNodesStats(List nodesStats) { nodeStats.getSearchPipelineStats(), nodeStats.getSegmentReplicationRejectionStats(), nodeStats.getRepositoriesStats(), - nodeStats.getAdmissionControlStats() + nodeStats.getAdmissionControlStats(), + nodeStats.getNodeCacheStats() ); }).collect(Collectors.toList()); } diff --git a/test/framework/src/main/java/org/opensearch/test/InternalTestCluster.java b/test/framework/src/main/java/org/opensearch/test/InternalTestCluster.java index abdcc3f918aa7..637d54983c8b9 100644 --- a/test/framework/src/main/java/org/opensearch/test/InternalTestCluster.java +++ b/test/framework/src/main/java/org/opensearch/test/InternalTestCluster.java @@ -2736,6 +2736,7 @@ public void ensureEstimatedStats() { false, false, false, + false, false ); assertThat( From 3030b8164769f97d94fca28718ccaa5691f09181 Mon Sep 17 00:00:00 2001 From: Peter Alfonsi Date: Mon, 29 Apr 2024 15:23:22 -0700 Subject: [PATCH 2/5] rerun Signed-off-by: Peter Alfonsi From 474c19f43691005fa907bb2853f51e5293f3001c Mon Sep 17 00:00:00 2001 From: Peter Alfonsi Date: Mon, 29 Apr 2024 15:57:34 -0700 Subject: [PATCH 3/5] Re-added old versions of onCached and onRemoval to ShardRequestCache Signed-off-by: Peter Alfonsi --- .../cache/request/ShardRequestCache.java | 20 +++++++++++++++++++ 1 file changed, 20 insertions(+) diff --git a/server/src/main/java/org/opensearch/index/cache/request/ShardRequestCache.java b/server/src/main/java/org/opensearch/index/cache/request/ShardRequestCache.java index c08ff73e3d6b2..502eae55df83e 100644 --- a/server/src/main/java/org/opensearch/index/cache/request/ShardRequestCache.java +++ b/server/src/main/java/org/opensearch/index/cache/request/ShardRequestCache.java @@ -32,6 +32,7 @@ package org.opensearch.index.cache.request; +import org.apache.lucene.util.Accountable; import org.opensearch.common.annotation.PublicApi; import org.opensearch.common.metrics.CounterMetric; import org.opensearch.core.common.bytes.BytesReference; @@ -61,6 +62,7 @@ public void onMiss() { missCount.inc(); } + // Functions used to increment size by passing in the size directly, Used now, as we use ICacheKey in the IndicesRequestCache.. public void onCached(long keyRamBytesUsed, BytesReference value) { totalMetric.inc(keyRamBytesUsed + value.ramBytesUsed()); } @@ -75,4 +77,22 @@ public void onRemoval(long keyRamBytesUsed, BytesReference value, boolean evicte } totalMetric.dec(dec); } + + // Old functions which increment size by passing in an Accountable. Functional but no longer used. + public void onCached(Accountable key, BytesReference value) { + totalMetric.inc(key.ramBytesUsed() + value.ramBytesUsed()); + } + + public void onRemoval(Accountable key, BytesReference value, boolean evicted) { + if (evicted) { + evictionsMetric.inc(); + } + long dec = 0; + if (key != null) { + dec += key.ramBytesUsed(); + } + if (value != null) { + dec += value.ramBytesUsed(); + } + } } From 1e04ce52bd3586a0fe5fce35a96f8a4794139e2a Mon Sep 17 00:00:00 2001 From: Peter Alfonsi Date: Mon, 29 Apr 2024 17:02:05 -0700 Subject: [PATCH 4/5] rerun Signed-off-by: Peter Alfonsi From 1f24c3085ae08dcd2b75ad351132648f68063ebd Mon Sep 17 00:00:00 2001 From: Peter Alfonsi Date: Mon, 29 Apr 2024 17:38:18 -0700 Subject: [PATCH 5/5] rerun Signed-off-by: Peter Alfonsi