2424import org .elasticsearch .action .ResolvedIndices ;
2525import org .elasticsearch .action .search .CanMatchNodeRequest ;
2626import org .elasticsearch .action .search .CanMatchNodeResponse ;
27+ import org .elasticsearch .action .search .OnlinePrewarmingService ;
2728import org .elasticsearch .action .search .SearchShardTask ;
2829import org .elasticsearch .action .search .SearchType ;
2930import org .elasticsearch .action .support .TransportActions ;
147148import java .util .Set ;
148149import java .util .concurrent .ExecutionException ;
149150import java .util .concurrent .Executor ;
151+ import java .util .concurrent .ThreadPoolExecutor ;
150152import java .util .concurrent .TimeoutException ;
151153import java .util .concurrent .atomic .AtomicBoolean ;
152154import java .util .concurrent .atomic .AtomicInteger ;
@@ -283,6 +285,16 @@ public class SearchService extends AbstractLifecycleComponent implements IndexEv
283285 Property .NodeScope
284286 );
285287
288+ // This setting ensures that we skip online prewarming tasks if the queuing in the search thread pool
289+ // reaches the configured factor X number of max threads in the search thread pool, such that
290+ // the system has a chance to catch up and prewarming doesn't take over the network bandwidth
291+ public static final Setting <Integer > PREWARMING_THRESHOLD_THREADPOOL_SIZE_FACTOR_POOL_SIZE = Setting .intSetting (
292+ "search.online_prewarming_threshold_poolsize_factor" ,
293+ 10 , // we will only execute online prewarming if there are less than 10 queued up items/ search thread
294+ 0 , // 0 would mean we only execute online prewarming if there's no queuing in the search tp
295+ Setting .Property .NodeScope
296+ );
297+
286298 private static final boolean BATCHED_QUERY_PHASE_FEATURE_FLAG = new FeatureFlag ("batched_query_phase" ).isEnabled ();
287299
288300 /**
@@ -317,6 +329,8 @@ public class SearchService extends AbstractLifecycleComponent implements IndexEv
317329
318330 private final FetchPhase fetchPhase ;
319331 private final CircuitBreaker circuitBreaker ;
332+ private final OnlinePrewarmingService onlinePrewarmingService ;
333+ private final int prewarmingMaxPoolFactorThreshold ;
320334 private volatile Executor searchExecutor ;
321335 private volatile boolean enableQueryPhaseParallelCollection ;
322336
@@ -362,7 +376,8 @@ public SearchService(
362376 FetchPhase fetchPhase ,
363377 CircuitBreakerService circuitBreakerService ,
364378 ExecutorSelector executorSelector ,
365- Tracer tracer
379+ Tracer tracer ,
380+ OnlinePrewarmingService onlinePrewarmingService
366381 ) {
367382 Settings settings = clusterService .getSettings ();
368383 this .threadPool = threadPool ;
@@ -375,7 +390,7 @@ public SearchService(
375390 this .multiBucketConsumerService = new MultiBucketConsumerService (clusterService , settings , circuitBreaker );
376391 this .executorSelector = executorSelector ;
377392 this .tracer = tracer ;
378-
393+ this . onlinePrewarmingService = onlinePrewarmingService ;
379394 TimeValue keepAliveInterval = KEEPALIVE_INTERVAL_SETTING .get (settings );
380395 setKeepAlives (DEFAULT_KEEPALIVE_SETTING .get (settings ), MAX_KEEPALIVE_SETTING .get (settings ));
381396
@@ -427,6 +442,7 @@ public SearchService(
427442 memoryAccountingBufferSize = MEMORY_ACCOUNTING_BUFFER_SIZE .get (settings ).getBytes ();
428443 clusterService .getClusterSettings ()
429444 .addSettingsUpdateConsumer (MEMORY_ACCOUNTING_BUFFER_SIZE , newValue -> this .memoryAccountingBufferSize = newValue .getBytes ());
445+ prewarmingMaxPoolFactorThreshold = PREWARMING_THRESHOLD_THREADPOOL_SIZE_FACTOR_POOL_SIZE .get (settings );
430446 }
431447
432448 public CircuitBreaker getCircuitBreaker () {
@@ -702,6 +718,10 @@ private <T extends RefCounted> void ensureAfterSeqNoRefreshed(
702718 try {
703719 if (waitForCheckpoint <= UNASSIGNED_SEQ_NO ) {
704720 runAsync (executor , executable , listener );
721+ // we successfully submitted the async task to the search pool so let's prewarm the shard
722+ if (isExecutorQueuedBeyondPrewarmingFactor (executor , prewarmingMaxPoolFactorThreshold ) == false ) {
723+ onlinePrewarmingService .prewarm (shard );
724+ }
705725 return ;
706726 }
707727 if (shard .indexSettings ().getRefreshInterval ().getMillis () <= 0 ) {
@@ -778,6 +798,10 @@ private void searchReady() {
778798 timeoutTask .cancel ();
779799 }
780800 runAsync (executor , executable , listener );
801+ // we successfully submitted the async task to the search pool so let's prewarm the shard
802+ if (isExecutorQueuedBeyondPrewarmingFactor (executor , prewarmingMaxPoolFactorThreshold ) == false ) {
803+ onlinePrewarmingService .prewarm (shard );
804+ }
781805 }
782806 }
783807 });
@@ -786,6 +810,28 @@ private void searchReady() {
786810 }
787811 }
788812
813+ /**
814+ * Checks if the executor is queued beyond the prewarming factor threshold, relative to the
815+ * number of threads in the pool.
816+ * This is used to determine if we should prewarm the shard - i.e. if the executor doesn't
817+ * contain queued tasks beyond the prewarming factor threshold X max pool size.
818+ *
819+ * @param searchOperationsExecutor the executor that executes the search operations
820+ * @param prewarmingMaxPoolFactorThreshold maximum number of queued up items / thread in the search pool
821+ */
822+ // visible for testing
823+ static boolean isExecutorQueuedBeyondPrewarmingFactor (Executor searchOperationsExecutor , int prewarmingMaxPoolFactorThreshold ) {
824+ if (searchOperationsExecutor instanceof ThreadPoolExecutor tpe ) {
825+ return (tpe .getMaximumPoolSize () * prewarmingMaxPoolFactorThreshold ) < tpe .getQueue ().size ();
826+ } else {
827+ logger .trace (
828+ "received executor [{}] that we can't inspect for queueing. allowing online prewarming for all searches" ,
829+ searchOperationsExecutor
830+ );
831+ return false ;
832+ }
833+ }
834+
789835 private IndexShard getShard (ShardSearchRequest request ) {
790836 final ShardSearchContextId contextId = request .readerId ();
791837 if (contextId != null && sessionId .equals (contextId .getSessionId ())) {
@@ -939,7 +985,8 @@ public void executeQueryPhase(
939985 freeReaderContext (readerContext .id ());
940986 throw e ;
941987 }
942- runAsync (getExecutor (readerContext .indexShard ()), () -> {
988+ Executor executor = getExecutor (readerContext .indexShard ());
989+ runAsync (executor , () -> {
943990 final ShardSearchRequest shardSearchRequest = readerContext .getShardSearchRequest (null );
944991 try (SearchContext searchContext = createContext (readerContext , shardSearchRequest , task , ResultsType .QUERY , false );) {
945992 var opsListener = searchContext .indexShard ().getSearchOperationListener ();
@@ -965,6 +1012,10 @@ public void executeQueryPhase(
9651012 throw e ;
9661013 }
9671014 }, wrapFailureListener (listener , readerContext , markAsUsed ));
1015+ // we successfully submitted the async task to the search pool so let's prewarm the shard
1016+ if (isExecutorQueuedBeyondPrewarmingFactor (executor , prewarmingMaxPoolFactorThreshold ) == false ) {
1017+ onlinePrewarmingService .prewarm (readerContext .indexShard ());
1018+ }
9681019 }
9691020
9701021 /**
@@ -991,7 +1042,8 @@ public void executeQueryPhase(
9911042 final Releasable markAsUsed = readerContext .markAsUsed (getKeepAlive (shardSearchRequest ));
9921043 rewriteAndFetchShardRequest (readerContext .indexShard (), shardSearchRequest , listener .delegateFailure ((l , rewritten ) -> {
9931044 // fork the execution in the search thread pool
994- runAsync (getExecutor (readerContext .indexShard ()), () -> {
1045+ Executor executor = getExecutor (readerContext .indexShard ());
1046+ runAsync (executor , () -> {
9951047 readerContext .setAggregatedDfs (request .dfs ());
9961048 try (SearchContext searchContext = createContext (readerContext , shardSearchRequest , task , ResultsType .QUERY , true );) {
9971049 final QuerySearchResult queryResult ;
@@ -1029,6 +1081,10 @@ public void executeQueryPhase(
10291081 throw e ;
10301082 }
10311083 }, wrapFailureListener (l , readerContext , markAsUsed ));
1084+ // we successfully submitted the async task to the search pool so let's prewarm the shard
1085+ if (isExecutorQueuedBeyondPrewarmingFactor (executor , prewarmingMaxPoolFactorThreshold ) == false ) {
1086+ onlinePrewarmingService .prewarm (readerContext .indexShard ());
1087+ }
10321088 }));
10331089 }
10341090
0 commit comments