You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@accumulo.apache.org by dd...@apache.org on 2024/02/02 04:24:59 UTC

(accumulo) branch 2.1 updated: Better error when compaction executors are not set (#4212)

This is an automated email from the ASF dual-hosted git repository.

ddanielr pushed a commit to branch 2.1
in repository https://gitbox.apache.org/repos/asf/accumulo.git


The following commit(s) were added to refs/heads/2.1 by this push:
     new cdd86f18f2 Better error when compaction executors are not set (#4212)
cdd86f18f2 is described below

commit cdd86f18f2cce1d00a21297bb927587cbd7747bf
Author: Daniel Roberts <dd...@gmail.com>
AuthorDate: Thu Feb 1 23:24:53 2024 -0500

    Better error when compaction executors are not set (#4212)
    
    * Better error when compaction executors are not set
    
    Replaces NPE with proper exception type and message for when the
    Default Compaction Planner's "executors" property is set and empty
    or not set at all.
---
 .../spi/compaction/DefaultCompactionPlanner.java   | 94 ++++++++++++----------
 .../compaction/DefaultCompactionPlannerTest.java   | 34 ++++++++
 2 files changed, 84 insertions(+), 44 deletions(-)

diff --git a/core/src/main/java/org/apache/accumulo/core/spi/compaction/DefaultCompactionPlanner.java b/core/src/main/java/org/apache/accumulo/core/spi/compaction/DefaultCompactionPlanner.java
index aae0591567..5a30556f63 100644
--- a/core/src/main/java/org/apache/accumulo/core/spi/compaction/DefaultCompactionPlanner.java
+++ b/core/src/main/java/org/apache/accumulo/core/spi/compaction/DefaultCompactionPlanner.java
@@ -167,62 +167,68 @@ public class DefaultCompactionPlanner implements CompactionPlanner {
       justification = "Field is written by Gson")
   @Override
   public void init(InitParameters params) {
-    ExecutorConfig[] execConfigs =
-        new Gson().fromJson(params.getOptions().get("executors"), ExecutorConfig[].class);
 
-    List<Executor> tmpExec = new ArrayList<>();
+    if (params.getOptions().containsKey("executors")
+        && !params.getOptions().get("executors").isBlank()) {
 
-    for (ExecutorConfig executorConfig : execConfigs) {
-      Long maxSize = executorConfig.maxSize == null ? null
-          : ConfigurationTypeHelper.getFixedMemoryAsBytes(executorConfig.maxSize);
+      ExecutorConfig[] execConfigs =
+          new Gson().fromJson(params.getOptions().get("executors"), ExecutorConfig[].class);
 
-      CompactionExecutorId ceid;
+      List<Executor> tmpExec = new ArrayList<>();
 
-      // If not supplied, GSON will leave type null. Default to internal
-      if (executorConfig.type == null) {
-        executorConfig.type = "internal";
-      }
+      for (ExecutorConfig executorConfig : execConfigs) {
+        Long maxSize = executorConfig.maxSize == null ? null
+            : ConfigurationTypeHelper.getFixedMemoryAsBytes(executorConfig.maxSize);
 
-      switch (executorConfig.type) {
-        case "internal":
-          Preconditions.checkArgument(null == executorConfig.queue,
-              "'queue' should not be specified for internal compactions");
-          int numThreads = Objects.requireNonNull(executorConfig.numThreads,
-              "'numThreads' must be specified for internal type");
-          ceid = params.getExecutorManager().createExecutor(executorConfig.name, numThreads);
-          break;
-        case "external":
-          Preconditions.checkArgument(null == executorConfig.numThreads,
-              "'numThreads' should not be specified for external compactions");
-          String queue = Objects.requireNonNull(executorConfig.queue,
-              "'queue' must be specified for external type");
-          ceid = params.getExecutorManager().getExternalExecutor(queue);
-          break;
-        default:
-          throw new IllegalArgumentException("type must be 'internal' or 'external'");
-      }
-      tmpExec.add(new Executor(ceid, maxSize));
-    }
+        CompactionExecutorId ceid;
 
-    Collections.sort(tmpExec, Comparator.comparing(Executor::getMaxSize,
-        Comparator.nullsLast(Comparator.naturalOrder())));
+        // If not supplied, GSON will leave type null. Default to internal
+        if (executorConfig.type == null) {
+          executorConfig.type = "internal";
+        }
 
-    executors = List.copyOf(tmpExec);
+        switch (executorConfig.type) {
+          case "internal":
+            Preconditions.checkArgument(null == executorConfig.queue,
+                "'queue' should not be specified for internal compactions");
+            int numThreads = Objects.requireNonNull(executorConfig.numThreads,
+                "'numThreads' must be specified for internal type");
+            ceid = params.getExecutorManager().createExecutor(executorConfig.name, numThreads);
+            break;
+          case "external":
+            Preconditions.checkArgument(null == executorConfig.numThreads,
+                "'numThreads' should not be specified for external compactions");
+            String queue = Objects.requireNonNull(executorConfig.queue,
+                "'queue' must be specified for external type");
+            ceid = params.getExecutorManager().getExternalExecutor(queue);
+            break;
+          default:
+            throw new IllegalArgumentException("type must be 'internal' or 'external'");
+        }
+        tmpExec.add(new Executor(ceid, maxSize));
+      }
 
-    if (executors.stream().filter(e -> e.getMaxSize() == null).count() > 1) {
-      throw new IllegalArgumentException(
-          "Can only have one executor w/o a maxSize. " + params.getOptions().get("executors"));
-    }
+      Collections.sort(tmpExec, Comparator.comparing(Executor::getMaxSize,
+          Comparator.nullsLast(Comparator.naturalOrder())));
+
+      executors = List.copyOf(tmpExec);
 
-    // use the add method on the Set interface to check for duplicate maxSizes
-    Set<Long> maxSizes = new HashSet<>();
-    executors.forEach(e -> {
-      if (!maxSizes.add(e.getMaxSize())) {
+      if (executors.stream().filter(e -> e.getMaxSize() == null).count() > 1) {
         throw new IllegalArgumentException(
-            "Duplicate maxSize set in executors. " + params.getOptions().get("executors"));
+            "Can only have one executor w/o a maxSize. " + params.getOptions().get("executors"));
       }
-    });
 
+      // use the add method on the Set interface to check for duplicate maxSizes
+      Set<Long> maxSizes = new HashSet<>();
+      executors.forEach(e -> {
+        if (!maxSizes.add(e.getMaxSize())) {
+          throw new IllegalArgumentException(
+              "Duplicate maxSize set in executors. " + params.getOptions().get("executors"));
+        }
+      });
+    } else {
+      throw new IllegalStateException("No defined executors for this planner");
+    }
     determineMaxFilesToCompact(params);
   }
 
diff --git a/core/src/test/java/org/apache/accumulo/core/spi/compaction/DefaultCompactionPlannerTest.java b/core/src/test/java/org/apache/accumulo/core/spi/compaction/DefaultCompactionPlannerTest.java
index 302106fc95..8423ee86e6 100644
--- a/core/src/test/java/org/apache/accumulo/core/spi/compaction/DefaultCompactionPlannerTest.java
+++ b/core/src/test/java/org/apache/accumulo/core/spi/compaction/DefaultCompactionPlannerTest.java
@@ -470,6 +470,40 @@ public class DefaultCompactionPlannerTest {
     assertTrue(e.getMessage().contains("maxSize"), "Error message didn't contain maxSize");
   }
 
+  /**
+   * Tests when "executors" is defined but empty.
+   */
+  @Test
+  public void testErrorEmptyExecutors() {
+    DefaultCompactionPlanner planner = new DefaultCompactionPlanner();
+    String executors = "";
+
+    var e = assertThrows(IllegalStateException.class,
+        () -> planner.init(getInitParams(defaultConf, executors)), "Failed to throw error");
+    assertTrue(e.getMessage().contains("No defined executors"),
+        "Error message didn't contain 'No defined executors'");
+  }
+
+  /**
+   * Tests when "executors" doesn't exist
+   */
+  @Test
+  public void testErrorNoExecutors() {
+
+    ServiceEnvironment senv = EasyMock.createMock(ServiceEnvironment.class);
+    EasyMock.expect(senv.getConfiguration()).andReturn(defaultConf).anyTimes();
+    EasyMock.replay(senv);
+
+    var initParams = new CompactionPlannerInitParams(csid, new HashMap<>(), senv);
+
+    DefaultCompactionPlanner dcPlanner = new DefaultCompactionPlanner();
+
+    var e = assertThrows(IllegalStateException.class, () -> dcPlanner.init(initParams),
+        "Failed to throw error");
+    assertTrue(e.getMessage().contains("No defined executors"),
+        "Error message didn't contain 'No defined executors'");
+  }
+
   // Test cases where a tablet has more than table.file.max files, but no files were found using the
   // compaction ratio. The planner should try to find the highest ratio that will result in a
   // compaction.