Browse Source

Fetch health info action (#89820)

This PR adds FetchHealthInfoCacheAction, which fetches the data sent to
the health node by the UpdateHealthInfoCacheAction (#89275).
Keith Massey 3 years ago
parent
commit
691c08abb0

+ 48 - 37
server/src/internalClusterTest/java/org/elasticsearch/health/UpdateHealthInfoCacheIT.java

@@ -9,13 +9,14 @@
 package org.elasticsearch.health;
 
 import org.elasticsearch.action.admin.cluster.settings.ClusterUpdateSettingsRequest;
+import org.elasticsearch.client.internal.Client;
 import org.elasticsearch.cluster.ClusterState;
 import org.elasticsearch.cluster.node.DiscoveryNode;
 import org.elasticsearch.common.settings.Settings;
 import org.elasticsearch.core.Nullable;
 import org.elasticsearch.core.TimeValue;
 import org.elasticsearch.health.node.DiskHealthInfo;
-import org.elasticsearch.health.node.HealthInfoCache;
+import org.elasticsearch.health.node.FetchHealthInfoCacheAction;
 import org.elasticsearch.health.node.LocalHealthMonitor;
 import org.elasticsearch.health.node.selection.HealthNode;
 import org.elasticsearch.test.ESIntegTestCase;
@@ -23,6 +24,7 @@ import org.elasticsearch.test.InternalTestCluster;
 
 import java.io.IOException;
 import java.util.Collection;
+import java.util.List;
 import java.util.Map;
 import java.util.concurrent.TimeUnit;
 
@@ -40,14 +42,7 @@ public class UpdateHealthInfoCacheIT extends ESIntegTestCase {
             String[] nodeIds = getNodes(internalCluster).keySet().toArray(new String[0]);
             DiscoveryNode healthNode = waitAndGetHealthNode(internalCluster);
             assertThat(healthNode, notNullValue());
-            assertBusy(() -> {
-                Map<String, DiskHealthInfo> healthInfoCache = internalCluster.getInstance(HealthInfoCache.class, healthNode.getName())
-                    .getDiskHealthInfo();
-                assertThat(healthInfoCache.size(), equalTo(nodeIds.length));
-                for (String nodeId : nodeIds) {
-                    assertThat(healthInfoCache.get(nodeId), equalTo(GREEN));
-                }
-            });
+            assertBusy(() -> assertResultsCanBeFetched(internalCluster, healthNode, List.of(nodeIds), null));
         } catch (IOException e) {
             throw new RuntimeException("Failed to close internal cluster: " + e.getMessage(), e);
         }
@@ -66,18 +61,14 @@ public class UpdateHealthInfoCacheIT extends ESIntegTestCase {
                 return isMaster == false && isHealthNode == false;
             }).findAny().orElseThrow();
             internalCluster.stopNode(nodeToLeave.getName());
-            assertBusy(() -> {
-                Map<String, DiskHealthInfo> healthInfoCache = internalCluster.getInstance(HealthInfoCache.class, healthNode.getName())
-                    .getDiskHealthInfo();
-                assertThat(healthInfoCache.size(), equalTo(nodes.size() - 1));
-                for (DiscoveryNode node : nodes) {
-                    if (node.getId().equals(nodeToLeave.getId())) {
-                        assertThat(healthInfoCache.containsKey(node.getId()), equalTo(false));
-                    } else {
-                        assertThat(healthInfoCache.get(node.getId()), equalTo(GREEN));
-                    }
-                }
-            });
+            assertBusy(
+                () -> assertResultsCanBeFetched(
+                    internalCluster,
+                    healthNode,
+                    nodes.stream().filter(node -> node.equals(nodeToLeave) == false).map(DiscoveryNode::getId).toList(),
+                    nodeToLeave.getId()
+                )
+            );
         } catch (IOException e) {
             throw new RuntimeException("Failed to close internal cluster: " + e.getMessage(), e);
         }
@@ -94,14 +85,7 @@ public class UpdateHealthInfoCacheIT extends ESIntegTestCase {
             DiscoveryNode newHealthNode = waitAndGetHealthNode(internalCluster);
             assertThat(newHealthNode, notNullValue());
             logger.info("Previous health node {}, new health node {}.", healthNodeToBeShutDown, newHealthNode);
-            assertBusy(() -> {
-                Map<String, DiskHealthInfo> healthInfoCache = internalCluster.getInstance(HealthInfoCache.class, newHealthNode.getName())
-                    .getDiskHealthInfo();
-                assertThat(healthInfoCache.size(), equalTo(nodeIds.length));
-                for (String nodeId : nodeIds) {
-                    assertThat(healthInfoCache.get(nodeId), equalTo(GREEN));
-                }
-            });
+            assertBusy(() -> assertResultsCanBeFetched(internalCluster, newHealthNode, List.of(nodeIds), null));
         } catch (IOException e) {
             throw new RuntimeException("Failed to close internal cluster: " + e.getMessage(), e);
         }
@@ -119,14 +103,7 @@ public class UpdateHealthInfoCacheIT extends ESIntegTestCase {
             ensureStableCluster(nodeIds.length);
             DiscoveryNode newHealthNode = waitAndGetHealthNode(internalCluster);
             assertThat(newHealthNode, notNullValue());
-            assertBusy(() -> {
-                Map<String, DiskHealthInfo> healthInfoCache = internalCluster.getInstance(HealthInfoCache.class, newHealthNode.getName())
-                    .getDiskHealthInfo();
-                assertThat(healthInfoCache.size(), equalTo(nodeIds.length));
-                for (String nodeId : nodeIds) {
-                    assertThat(healthInfoCache.get(nodeId), equalTo(GREEN));
-                }
-            });
+            assertBusy(() -> assertResultsCanBeFetched(internalCluster, newHealthNode, List.of(nodeIds), null));
         } catch (IOException e) {
             throw new RuntimeException("Failed to close internal cluster: " + e.getMessage(), e);
         }
@@ -151,6 +128,40 @@ public class UpdateHealthInfoCacheIT extends ESIntegTestCase {
         return healthNode[0];
     }
 
+    /**
+     * This method fetches the health data using FetchHealthInfoCacheAction. It does this using both a random non-health-node client and
+     * a health node client. It asserts that all expected nodeIds are there with GREEN status, and that the notExpectedNodeId (if not
+     * null) is not in the results.
+     * @param internalCluster The cluster to use to get clients
+     * @param healthNode The health node
+     * @param expectedNodeIds The list of nodeIds we expect to see in the results (with a GREEN status)
+     * @param notExpectedNodeId A nullable nodeId that we do _not_ expect to see in the results
+     * @throws Exception
+     */
+    private void assertResultsCanBeFetched(
+        InternalTestCluster internalCluster,
+        DiscoveryNode healthNode,
+        Iterable<String> expectedNodeIds,
+        @Nullable String notExpectedNodeId
+    ) throws Exception {
+        Client nonHealthNodeClient = internalCluster.client(randomValueOtherThan(healthNode.getName(), internalCluster::getRandomNodeName));
+        FetchHealthInfoCacheAction.Response healthResponse = nonHealthNodeClient.execute(
+            FetchHealthInfoCacheAction.INSTANCE,
+            new FetchHealthInfoCacheAction.Request()
+        ).get();
+        for (String nodeId : expectedNodeIds) {
+            assertThat(healthResponse.getHealthInfo().diskInfoByNode().get(nodeId), equalTo(GREEN));
+        }
+        if (notExpectedNodeId != null) {
+            assertThat(healthResponse.getHealthInfo().diskInfoByNode().containsKey(notExpectedNodeId), equalTo(false));
+        }
+        Client healthNodeClient = internalCluster.client(healthNode.getName());
+        healthResponse = healthNodeClient.execute(FetchHealthInfoCacheAction.INSTANCE, new FetchHealthInfoCacheAction.Request()).get();
+        for (String nodeId : expectedNodeIds) {
+            assertThat(healthResponse.getHealthInfo().diskInfoByNode().get(nodeId), equalTo(GREEN));
+        }
+    }
+
     private void decreasePollingInterval(InternalTestCluster internalCluster) {
         internalCluster.client()
             .admin()

+ 2 - 0
server/src/main/java/org/elasticsearch/action/ActionModule.java

@@ -263,6 +263,7 @@ import org.elasticsearch.common.settings.SettingsFilter;
 import org.elasticsearch.gateway.TransportNodesListGatewayStartedShards;
 import org.elasticsearch.health.GetHealthAction;
 import org.elasticsearch.health.RestGetHealthAction;
+import org.elasticsearch.health.node.FetchHealthInfoCacheAction;
 import org.elasticsearch.health.node.UpdateHealthInfoCacheAction;
 import org.elasticsearch.health.node.selection.HealthNode;
 import org.elasticsearch.index.seqno.GlobalCheckpointSyncAction;
@@ -704,6 +705,7 @@ public class ActionModule extends AbstractModule {
 
         if (HealthNode.isEnabled()) {
             actions.register(UpdateHealthInfoCacheAction.INSTANCE, UpdateHealthInfoCacheAction.TransportAction.class);
+            actions.register(FetchHealthInfoCacheAction.INSTANCE, FetchHealthInfoCacheAction.TransportAction.class);
         }
 
         return unmodifiableMap(actions.getRegistry());

+ 126 - 0
server/src/main/java/org/elasticsearch/health/node/FetchHealthInfoCacheAction.java

@@ -0,0 +1,126 @@
+/*
+ * Copyright Elasticsearch B.V. and/or licensed to Elasticsearch B.V. under one
+ * or more contributor license agreements. Licensed under the Elastic License
+ * 2.0 and the Server Side Public License, v 1; you may not use this file except
+ * in compliance with, at your election, the Elastic License 2.0 or the Server
+ * Side Public License, v 1.
+ */
+
+package org.elasticsearch.health.node;
+
+import org.elasticsearch.action.ActionListener;
+import org.elasticsearch.action.ActionRequest;
+import org.elasticsearch.action.ActionRequestValidationException;
+import org.elasticsearch.action.ActionResponse;
+import org.elasticsearch.action.ActionType;
+import org.elasticsearch.action.support.ActionFilters;
+import org.elasticsearch.cluster.ClusterState;
+import org.elasticsearch.cluster.service.ClusterService;
+import org.elasticsearch.common.inject.Inject;
+import org.elasticsearch.common.io.stream.StreamInput;
+import org.elasticsearch.common.io.stream.StreamOutput;
+import org.elasticsearch.health.node.action.TransportHealthNodeAction;
+import org.elasticsearch.tasks.Task;
+import org.elasticsearch.threadpool.ThreadPool;
+import org.elasticsearch.transport.TransportService;
+
+import java.io.IOException;
+import java.util.Objects;
+
+/**
+ * This action retrieves all the HealthInfo data from the health node. It is meant to be used when a user makes a health API request. The
+ * data that this action retrieves is populated by UpdateHealthInfoCacheAction.
+ */
+public class FetchHealthInfoCacheAction extends ActionType<FetchHealthInfoCacheAction.Response> {
+
+    public static class Request extends ActionRequest {
+        public Request() {}
+
+        public Request(StreamInput in) throws IOException {
+            super(in);
+        }
+
+        @Override
+        public ActionRequestValidationException validate() {
+            return null;
+        }
+    }
+
+    public static class Response extends ActionResponse {
+        private final HealthInfo healthInfo;
+
+        public Response(final HealthInfo healthInfo) {
+            this.healthInfo = healthInfo;
+        }
+
+        public Response(StreamInput input) throws IOException {
+            this.healthInfo = new HealthInfo(input);
+        }
+
+        @Override
+        public void writeTo(StreamOutput output) throws IOException {
+            this.healthInfo.writeTo(output);
+        }
+
+        public HealthInfo getHealthInfo() {
+            return healthInfo;
+        }
+
+        @Override
+        public boolean equals(Object o) {
+            if (this == o) return true;
+            if (o == null || getClass() != o.getClass()) return false;
+            FetchHealthInfoCacheAction.Response response = (FetchHealthInfoCacheAction.Response) o;
+            return healthInfo.equals(response.healthInfo);
+        }
+
+        @Override
+        public int hashCode() {
+            return Objects.hash(healthInfo);
+        }
+    }
+
+    public static final FetchHealthInfoCacheAction INSTANCE = new FetchHealthInfoCacheAction();
+    public static final String NAME = "cluster:monitor/fetch/health/info";
+
+    private FetchHealthInfoCacheAction() {
+        super(NAME, FetchHealthInfoCacheAction.Response::new);
+    }
+
+    public static class TransportAction extends TransportHealthNodeAction<
+        FetchHealthInfoCacheAction.Request,
+        FetchHealthInfoCacheAction.Response> {
+        private final HealthInfoCache nodeHealthOverview;
+
+        @Inject
+        public TransportAction(
+            TransportService transportService,
+            ClusterService clusterService,
+            ThreadPool threadPool,
+            ActionFilters actionFilters,
+            HealthInfoCache nodeHealthOverview
+        ) {
+            super(
+                FetchHealthInfoCacheAction.NAME,
+                transportService,
+                clusterService,
+                threadPool,
+                actionFilters,
+                FetchHealthInfoCacheAction.Request::new,
+                FetchHealthInfoCacheAction.Response::new,
+                ThreadPool.Names.MANAGEMENT
+            );
+            this.nodeHealthOverview = nodeHealthOverview;
+        }
+
+        @Override
+        protected void healthOperation(
+            Task task,
+            FetchHealthInfoCacheAction.Request request,
+            ClusterState clusterState,
+            ActionListener<FetchHealthInfoCacheAction.Response> listener
+        ) {
+            listener.onResponse(new Response(nodeHealthOverview.getHealthInfo()));
+        }
+    }
+}

+ 31 - 0
server/src/main/java/org/elasticsearch/health/node/HealthInfo.java

@@ -0,0 +1,31 @@
+/*
+ * Copyright Elasticsearch B.V. and/or licensed to Elasticsearch B.V. under one
+ * or more contributor license agreements. Licensed under the Elastic License
+ * 2.0 and the Server Side Public License, v 1; you may not use this file except
+ * in compliance with, at your election, the Elastic License 2.0 or the Server
+ * Side Public License, v 1.
+ */
+
+package org.elasticsearch.health.node;
+
+import org.elasticsearch.common.io.stream.StreamInput;
+import org.elasticsearch.common.io.stream.StreamOutput;
+import org.elasticsearch.common.io.stream.Writeable;
+
+import java.io.IOException;
+import java.util.Map;
+
+/**
+ * This class wraps all the data returned by the health node.
+ * @param diskInfoByNode A Map of node id to DiskHealthInfo for that node
+ */
+public record HealthInfo(Map<String, DiskHealthInfo> diskInfoByNode) implements Writeable {
+    public HealthInfo(StreamInput input) throws IOException {
+        this(input.readMap(StreamInput::readString, DiskHealthInfo::new));
+    }
+
+    @Override
+    public void writeTo(StreamOutput output) throws IOException {
+        output.writeMap(diskInfoByNode, StreamOutput::writeString, (out, diskHealthInfo) -> diskHealthInfo.writeTo(out));
+    }
+}

+ 7 - 3
server/src/main/java/org/elasticsearch/health/node/HealthInfoCache.java

@@ -59,8 +59,12 @@ public class HealthInfoCache implements ClusterStateListener {
         }
     }
 
-    // A shallow copy is enough because the inner data is immutable.
-    public Map<String, DiskHealthInfo> getDiskHealthInfo() {
-        return Map.copyOf(diskInfoByNode);
+    /**
+     * This returns all the health info stored in this cache
+     * @return A HealthInfo object wrapping all health data in the cache
+     */
+    public HealthInfo getHealthInfo() {
+        // A shallow copy is enough because the inner data is immutable.
+        return new HealthInfo(Map.copyOf(diskInfoByNode));
     }
 }

+ 163 - 0
server/src/test/java/org/elasticsearch/health/node/FetchHealthInfoCacheActionTests.java

@@ -0,0 +1,163 @@
+/*
+ * Copyright Elasticsearch B.V. and/or licensed to Elasticsearch B.V. under one
+ * or more contributor license agreements. Licensed under the Elastic License
+ * 2.0 and the Server Side Public License, v 1; you may not use this file except
+ * in compliance with, at your election, the Elastic License 2.0 or the Server
+ * Side Public License, v 1.
+ */
+
+package org.elasticsearch.health.node;
+
+import org.elasticsearch.Version;
+import org.elasticsearch.action.support.ActionFilters;
+import org.elasticsearch.action.support.ActionTestUtils;
+import org.elasticsearch.action.support.PlainActionFuture;
+import org.elasticsearch.action.support.replication.ClusterStateCreationUtils;
+import org.elasticsearch.cluster.node.DiscoveryNode;
+import org.elasticsearch.cluster.node.DiscoveryNodeRole;
+import org.elasticsearch.cluster.service.ClusterService;
+import org.elasticsearch.health.HealthStatus;
+import org.elasticsearch.test.ESTestCase;
+import org.elasticsearch.test.EqualsHashCodeTestUtils;
+import org.elasticsearch.test.transport.CapturingTransport;
+import org.elasticsearch.threadpool.TestThreadPool;
+import org.elasticsearch.threadpool.ThreadPool;
+import org.elasticsearch.transport.TransportService;
+import org.junit.After;
+import org.junit.AfterClass;
+import org.junit.Before;
+import org.junit.BeforeClass;
+
+import java.util.Collections;
+import java.util.HashMap;
+import java.util.Map;
+import java.util.Set;
+import java.util.concurrent.ExecutionException;
+import java.util.concurrent.TimeUnit;
+
+import static org.elasticsearch.test.ClusterServiceUtils.createClusterService;
+import static org.elasticsearch.test.ClusterServiceUtils.setState;
+import static org.hamcrest.Matchers.equalTo;
+
+public class FetchHealthInfoCacheActionTests extends ESTestCase {
+    private static ThreadPool threadPool;
+
+    private ClusterService clusterService;
+    private TransportService transportService;
+    private DiscoveryNode localNode;
+    private DiscoveryNode[] allNodes;
+
+    @BeforeClass
+    public static void beforeClass() {
+        threadPool = new TestThreadPool("FetchHealthInfoCacheAction");
+    }
+
+    @Before
+    @Override
+    public void setUp() throws Exception {
+        super.setUp();
+        CapturingTransport transport = new CapturingTransport();
+        clusterService = createClusterService(threadPool);
+        transportService = transport.createTransportService(
+            clusterService.getSettings(),
+            threadPool,
+            TransportService.NOOP_TRANSPORT_INTERCEPTOR,
+            x -> clusterService.localNode(),
+            null,
+            Collections.emptySet()
+        );
+        transportService.start();
+        transportService.acceptIncomingRequests();
+        int totalNodes = randomIntBetween(1, 200);
+        allNodes = new DiscoveryNode[totalNodes];
+        localNode = new DiscoveryNode(
+            "local_node",
+            buildNewFakeTransportAddress(),
+            Collections.emptyMap(),
+            Set.of(DiscoveryNodeRole.MASTER_ROLE, DiscoveryNodeRole.DATA_ROLE),
+            Version.CURRENT
+        );
+        allNodes[0] = localNode;
+        for (int i = 0; i < totalNodes - 1; i++) {
+            DiscoveryNode remoteNode = new DiscoveryNode(
+                "remote_node" + i,
+                buildNewFakeTransportAddress(),
+                Collections.emptyMap(),
+                Set.of(DiscoveryNodeRole.MASTER_ROLE, DiscoveryNodeRole.DATA_ROLE),
+                Version.CURRENT
+            );
+            allNodes[i + 1] = remoteNode;
+        }
+    }
+
+    @After
+    public void tearDown() throws Exception {
+        super.tearDown();
+        clusterService.close();
+        transportService.close();
+    }
+
+    @AfterClass
+    public static void afterClass() {
+        ThreadPool.terminate(threadPool, 30, TimeUnit.SECONDS);
+        threadPool = null;
+    }
+
+    public void testAction() throws ExecutionException, InterruptedException {
+        FetchHealthInfoCacheAction.Request request = new FetchHealthInfoCacheAction.Request();
+        PlainActionFuture<FetchHealthInfoCacheAction.Response> listener = new PlainActionFuture<>();
+        setState(clusterService, ClusterStateCreationUtils.state(localNode, localNode, localNode, allNodes));
+        HealthInfoCache healthInfoCache = getTestHealthInfoCache();
+        final FetchHealthInfoCacheAction.Response expectedResponse = new FetchHealthInfoCacheAction.Response(
+            new HealthInfo(healthInfoCache.getHealthInfo().diskInfoByNode())
+        );
+        ActionTestUtils.execute(
+            new FetchHealthInfoCacheAction.TransportAction(
+                transportService,
+                clusterService,
+                threadPool,
+                new ActionFilters(Set.of()),
+                healthInfoCache
+            ),
+            null,
+            request,
+            listener
+        );
+        FetchHealthInfoCacheAction.Response actualResponse = listener.get();
+        assertThat(actualResponse, equalTo(expectedResponse));
+        assertThat(actualResponse.getHealthInfo(), equalTo(expectedResponse.getHealthInfo()));
+    }
+
+    private HealthInfoCache getTestHealthInfoCache() {
+        HealthInfoCache healthInfoCache = HealthInfoCache.create(clusterService);
+        for (DiscoveryNode allNode : allNodes) {
+            String nodeId = allNode.getId();
+            healthInfoCache.updateNodeHealth(
+                nodeId,
+                new DiskHealthInfo(randomFrom(HealthStatus.values()), randomFrom(DiskHealthInfo.Cause.values()))
+            );
+        }
+        return healthInfoCache;
+    }
+
+    public void testResponseSerialization() {
+        FetchHealthInfoCacheAction.Response response = new FetchHealthInfoCacheAction.Response(
+            new HealthInfo(getTestHealthInfoCache().getHealthInfo().diskInfoByNode())
+        );
+        EqualsHashCodeTestUtils.checkEqualsAndHashCode(
+            response,
+            resopnseWritable -> copyWriteable(resopnseWritable, writableRegistry(), FetchHealthInfoCacheAction.Response::new),
+            this::mutateResponse
+        );
+    }
+
+    private FetchHealthInfoCacheAction.Response mutateResponse(FetchHealthInfoCacheAction.Response originalResponse) {
+        Map<String, DiskHealthInfo> diskHealthInfoMap = originalResponse.getHealthInfo().diskInfoByNode();
+        Map<String, DiskHealthInfo> diskHealthInfoMapCopy = new HashMap<>(diskHealthInfoMap);
+        diskHealthInfoMapCopy.put(
+            randomAlphaOfLength(10),
+            new DiskHealthInfo(randomFrom(HealthStatus.values()), randomFrom(DiskHealthInfo.Cause.values()))
+        );
+        return new FetchHealthInfoCacheAction.Response(new HealthInfo(diskHealthInfoMapCopy));
+    }
+}

+ 3 - 3
server/src/test/java/org/elasticsearch/health/node/HealthInfoCacheTests.java

@@ -55,7 +55,7 @@ public class HealthInfoCacheTests extends ESTestCase {
         healthInfoCache.updateNodeHealth(node1.getId(), GREEN);
         healthInfoCache.updateNodeHealth(node2.getId(), RED);
 
-        Map<String, DiskHealthInfo> diskHealthInfo = healthInfoCache.getDiskHealthInfo();
+        Map<String, DiskHealthInfo> diskHealthInfo = healthInfoCache.getHealthInfo().diskInfoByNode();
         healthInfoCache.updateNodeHealth(node1.getId(), RED);
 
         assertThat(diskHealthInfo.get(node1.getId()), equalTo(GREEN));
@@ -71,7 +71,7 @@ public class HealthInfoCacheTests extends ESTestCase {
         ClusterState current = ClusterStateCreationUtils.state(node1, node1, node1, new DiscoveryNode[] { node1 });
         healthInfoCache.clusterChanged(new ClusterChangedEvent("test", current, previous));
 
-        Map<String, DiskHealthInfo> diskHealthInfo = healthInfoCache.getDiskHealthInfo();
+        Map<String, DiskHealthInfo> diskHealthInfo = healthInfoCache.getHealthInfo().diskInfoByNode();
         assertThat(diskHealthInfo.get(node1.getId()), equalTo(GREEN));
         assertThat(diskHealthInfo.get(node2.getId()), nullValue());
     }
@@ -85,7 +85,7 @@ public class HealthInfoCacheTests extends ESTestCase {
         ClusterState current = ClusterStateCreationUtils.state(node1, node1, node2, allNodes);
         healthInfoCache.clusterChanged(new ClusterChangedEvent("test", current, previous));
 
-        Map<String, DiskHealthInfo> diskHealthInfo = healthInfoCache.getDiskHealthInfo();
+        Map<String, DiskHealthInfo> diskHealthInfo = healthInfoCache.getHealthInfo().diskInfoByNode();
         assertThat(diskHealthInfo.isEmpty(), equalTo(true));
     }
 }

+ 72 - 0
server/src/test/java/org/elasticsearch/health/node/HealthInfoTests.java

@@ -0,0 +1,72 @@
+/*
+ * Copyright Elasticsearch B.V. and/or licensed to Elasticsearch B.V. under one
+ * or more contributor license agreements. Licensed under the Elastic License
+ * 2.0 and the Server Side Public License, v 1; you may not use this file except
+ * in compliance with, at your election, the Elastic License 2.0 or the Server
+ * Side Public License, v 1.
+ */
+
+package org.elasticsearch.health.node;
+
+import org.elasticsearch.common.io.stream.Writeable;
+import org.elasticsearch.health.HealthStatus;
+import org.elasticsearch.test.AbstractWireSerializingTestCase;
+
+import java.util.HashMap;
+import java.util.Map;
+
+public class HealthInfoTests extends AbstractWireSerializingTestCase<HealthInfo> {
+    @Override
+    protected Writeable.Reader<HealthInfo> instanceReader() {
+        return HealthInfo::new;
+    }
+
+    @Override
+    protected HealthInfo createTestInstance() {
+        int numberOfNodes = randomIntBetween(0, 200);
+        Map<String, DiskHealthInfo> diskInfoByNode = new HashMap<>(numberOfNodes);
+        for (int i = 0; i < numberOfNodes; i++) {
+            DiskHealthInfo diskHealthInfo = randomBoolean()
+                ? new DiskHealthInfo(randomFrom(HealthStatus.values()))
+                : new DiskHealthInfo(randomFrom(HealthStatus.values()), randomFrom(DiskHealthInfo.Cause.values()));
+            diskInfoByNode.put(randomAlphaOfLengthBetween(10, 100), diskHealthInfo);
+        }
+        return new HealthInfo(diskInfoByNode);
+    }
+
+    @Override
+    public HealthInfo mutateInstance(HealthInfo originalHealthInfo) {
+        Map<String, DiskHealthInfo> diskHealthInfoMap = originalHealthInfo.diskInfoByNode();
+        Map<String, DiskHealthInfo> diskHealthInfoMapCopy = new HashMap<>(diskHealthInfoMap);
+        if (diskHealthInfoMap.isEmpty()) {
+            diskHealthInfoMapCopy.put(
+                randomAlphaOfLength(10),
+                new DiskHealthInfo(randomFrom(HealthStatus.values()), randomFrom(DiskHealthInfo.Cause.values()))
+            );
+        } else {
+            switch (randomIntBetween(1, 3)) {
+                case 1 -> {
+                    diskHealthInfoMapCopy.put(
+                        randomAlphaOfLength(10),
+                        new DiskHealthInfo(randomFrom(HealthStatus.values()), randomFrom(DiskHealthInfo.Cause.values()))
+                    );
+                }
+                case 2 -> {
+                    String someNode = randomFrom(diskHealthInfoMap.keySet());
+                    diskHealthInfoMapCopy.put(
+                        someNode,
+                        new DiskHealthInfo(
+                            randomValueOtherThan(diskHealthInfoMap.get(someNode).healthStatus(), () -> randomFrom(HealthStatus.values())),
+                            randomFrom(DiskHealthInfo.Cause.values())
+                        )
+                    );
+                }
+                case 3 -> {
+                    diskHealthInfoMapCopy.remove(randomFrom(diskHealthInfoMapCopy.keySet()));
+                }
+                default -> throw new IllegalStateException();
+            }
+        }
+        return new HealthInfo(diskHealthInfoMapCopy);
+    }
+}

+ 1 - 1
server/src/test/java/org/elasticsearch/health/node/UpdateHealthInfoCacheActionTests.java

@@ -111,7 +111,7 @@ public class UpdateHealthInfoCacheActionTests extends ESTestCase {
         );
         AcknowledgedResponse actualResponse = listener.get();
         assertThat(actualResponse, equalTo(expectedResponse));
-        assertThat(healthInfoCache.getDiskHealthInfo().get(localNode.getId()), equalTo(diskHealthInfo));
+        assertThat(healthInfoCache.getHealthInfo().diskInfoByNode().get(localNode.getId()), equalTo(diskHealthInfo));
     }
 
     public void testRequestSerialization() {