You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@lucene.apache.org by ja...@apache.org on 2023/05/11 09:14:52 UTC

[lucene] branch branch_9x updated: Simplify SliceExecutor and QueueSizeBasedExecutor (#12285)

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

javanna pushed a commit to branch branch_9x
in repository https://gitbox.apache.org/repos/asf/lucene.git


The following commit(s) were added to refs/heads/branch_9x by this push:
     new 784830a6496 Simplify SliceExecutor and QueueSizeBasedExecutor (#12285)
784830a6496 is described below

commit 784830a64964f7f62c59a10f974e6f9ac28ac1c4
Author: Luca Cavanna <ja...@apache.org>
AuthorDate: Thu May 11 11:08:48 2023 +0200

    Simplify SliceExecutor and QueueSizeBasedExecutor (#12285)
    
    The only behaviour that QueueSizeBasedExecutor overrides from SliceExecutor is when to execute on the caller thread. There is no need to override the whole invokeAll method for that. Instead, this commit introduces a shouldExecuteOnCallerThread method that can be overridden.
---
 .../lucene/search/QueueSizeBasedExecutor.java      | 27 ++--------
 .../org/apache/lucene/search/SliceExecutor.java    | 57 +++++++---------------
 .../apache/lucene/search/TestIndexSearcher.java    | 13 ++---
 3 files changed, 26 insertions(+), 71 deletions(-)

diff --git a/lucene/core/src/java/org/apache/lucene/search/QueueSizeBasedExecutor.java b/lucene/core/src/java/org/apache/lucene/search/QueueSizeBasedExecutor.java
index a76b81f5da1..65ba1ea5573 100644
--- a/lucene/core/src/java/org/apache/lucene/search/QueueSizeBasedExecutor.java
+++ b/lucene/core/src/java/org/apache/lucene/search/QueueSizeBasedExecutor.java
@@ -17,7 +17,6 @@
 
 package org.apache.lucene.search;
 
-import java.util.Collection;
 import java.util.concurrent.ThreadPoolExecutor;
 
 /**
@@ -30,31 +29,15 @@ class QueueSizeBasedExecutor extends SliceExecutor {
 
   private final ThreadPoolExecutor threadPoolExecutor;
 
-  public QueueSizeBasedExecutor(ThreadPoolExecutor threadPoolExecutor) {
+  QueueSizeBasedExecutor(ThreadPoolExecutor threadPoolExecutor) {
     super(threadPoolExecutor);
     this.threadPoolExecutor = threadPoolExecutor;
   }
 
   @Override
-  public void invokeAll(Collection<? extends Runnable> tasks) {
-    int i = 0;
-
-    for (Runnable task : tasks) {
-      boolean shouldExecuteOnCallerThread = false;
-
-      // Execute last task on caller thread
-      if (i == tasks.size() - 1) {
-        shouldExecuteOnCallerThread = true;
-      }
-
-      if (threadPoolExecutor.getQueue().size()
-          >= (threadPoolExecutor.getMaximumPoolSize() * LIMITING_FACTOR)) {
-        shouldExecuteOnCallerThread = true;
-      }
-
-      processTask(task, shouldExecuteOnCallerThread);
-
-      ++i;
-    }
+  boolean shouldExecuteOnCallerThread(int index, int numTasks) {
+    return super.shouldExecuteOnCallerThread(index, numTasks)
+        || threadPoolExecutor.getQueue().size()
+            >= (threadPoolExecutor.getMaximumPoolSize() * LIMITING_FACTOR);
   }
 }
diff --git a/lucene/core/src/java/org/apache/lucene/search/SliceExecutor.java b/lucene/core/src/java/org/apache/lucene/search/SliceExecutor.java
index c84beeb5fb7..0e593740914 100644
--- a/lucene/core/src/java/org/apache/lucene/search/SliceExecutor.java
+++ b/lucene/core/src/java/org/apache/lucene/search/SliceExecutor.java
@@ -18,6 +18,7 @@
 package org.apache.lucene.search;
 
 import java.util.Collection;
+import java.util.Objects;
 import java.util.concurrent.Executor;
 import java.util.concurrent.RejectedExecutionException;
 
@@ -28,54 +29,30 @@ import java.util.concurrent.RejectedExecutionException;
 class SliceExecutor {
   private final Executor executor;
 
-  public SliceExecutor(Executor executor) {
-    this.executor = executor;
+  SliceExecutor(Executor executor) {
+    this.executor = Objects.requireNonNull(executor, "Executor is null");
   }
 
-  public void invokeAll(Collection<? extends Runnable> tasks) {
-
-    if (tasks == null) {
-      throw new IllegalArgumentException("Tasks is null");
-    }
-
-    if (executor == null) {
-      throw new IllegalArgumentException("Executor is null");
-    }
-
+  final void invokeAll(Collection<? extends Runnable> tasks) {
     int i = 0;
-
     for (Runnable task : tasks) {
-      boolean shouldExecuteOnCallerThread = false;
-
-      // Execute last task on caller thread
-      if (i == tasks.size() - 1) {
-        shouldExecuteOnCallerThread = true;
+      if (shouldExecuteOnCallerThread(i, tasks.size())) {
+        task.run();
+      } else {
+        try {
+          executor.execute(task);
+        } catch (
+            @SuppressWarnings("unused")
+            RejectedExecutionException e) {
+          task.run();
+        }
       }
-
-      processTask(task, shouldExecuteOnCallerThread);
       ++i;
     }
-    ;
   }
 
-  // Helper method to execute a single task
-  protected void processTask(final Runnable task, final boolean shouldExecuteOnCallerThread) {
-    if (task == null) {
-      throw new IllegalArgumentException("Input is null");
-    }
-
-    if (!shouldExecuteOnCallerThread) {
-      try {
-        executor.execute(task);
-
-        return;
-      } catch (
-          @SuppressWarnings("unused")
-          RejectedExecutionException e) {
-        // Execute on caller thread
-      }
-    }
-
-    task.run();
+  boolean shouldExecuteOnCallerThread(int index, int numTasks) {
+    // Execute last task on caller thread
+    return index == numTasks - 1;
   }
 }
diff --git a/lucene/core/src/test/org/apache/lucene/search/TestIndexSearcher.java b/lucene/core/src/test/org/apache/lucene/search/TestIndexSearcher.java
index c3bcda9d47a..7e8693bd8b3 100644
--- a/lucene/core/src/test/org/apache/lucene/search/TestIndexSearcher.java
+++ b/lucene/core/src/test/org/apache/lucene/search/TestIndexSearcher.java
@@ -453,20 +453,15 @@ public class TestIndexSearcher extends LuceneTestCase {
     }
   }
 
-  private class RandomBlockingSliceExecutor extends SliceExecutor {
+  private static class RandomBlockingSliceExecutor extends SliceExecutor {
 
-    public RandomBlockingSliceExecutor(Executor executor) {
+    RandomBlockingSliceExecutor(Executor executor) {
       super(executor);
     }
 
     @Override
-    public void invokeAll(Collection<? extends Runnable> tasks) {
-
-      for (Runnable task : tasks) {
-        boolean shouldExecuteOnCallerThread = random().nextBoolean();
-
-        processTask(task, shouldExecuteOnCallerThread);
-      }
+    boolean shouldExecuteOnCallerThread(int index, int numTasks) {
+      return random().nextBoolean();
     }
   }
 }