Browse Source

Query phase: fold collector wrappers into a single top level collector (#97030)

The query phase uses a number of different collectors and combines them together, pretty much one per feature that the search API exposes: there is a collector for post_filter, one for min_score, one for terminate_after, one for aggs. While this is very flexible, we always combine such collectors together in the same way (e.g. terminate_after must be the first one, post_filter is only applied to top docs collection, min score is applied to both aggs and top docs). This means that despite we could flexibly compose collectors, we need to apply each feature predictably which makes the composability not needed. Furthermore, composability causes complexity.

The terminate_after functionality is a clear example of complexity introduced as a consequence of having a complex collector tree: it relies on a multi collector, and throws an exception to force terminating the collection for all other collectors in the tree. If there was a single collector aware of post_filter, min_score and terminate_after at the same time, we could simply reuse Lucene mechanisms to early terminate the collection (CollectionTerminatedException) instead of forcing the termination throwing an exception that Lucene does not handle.

Furthermore, MultiCollector is a complex and generic collector to combine multiple collectors together, while we always every combine maximum two collectors with it, which are more or less fixed (e.g. top docs and aggs).

This PR introduces a new top-level collector that is inspired by MultiCollector in that it holds the top docs and the optional aggs collector and applies post_filter, min_score as well as terminate_after as part of its execution. This allows us to have a specialized collector for our needs, less flexibility and more control. This surfaced some strange behaviour that we may want to change as a follow-up in how terminate_after makes us collecting docs even when all possible collections have been early terminated. The goal of this PR though is to have feature parity with query phase before the refactoring, without any change of behaviour.

A nice benefit of this work is that it allows us to rely on CollectionTerminatedException for the terminate_after functionality. This simplifies the introduction of multi-threaded collector managers when it comes to handling exceptions.
Luca Cavanna 2 years ago
parent
commit
f5a2af6c71

+ 43 - 43
docs/reference/search/profile.asciidoc

@@ -166,9 +166,16 @@ The API returns the following result:
             "rewrite_time": 451233,
             "collector": [
               {
-                "name": "SimpleTopScoreDocCollector",
-                "reason": "search_top_hits",
-                "time_in_nanos": 775274
+                "name": "QueryPhaseCollector",
+                "reason": "search_query_phase",
+                "time_in_nanos": 775274,
+                "children" : [
+                  {
+                    "name": "SimpleTopScoreDocCollector",
+                    "reason": "search_top_hits",
+                    "time_in_nanos": 775274
+                  }
+                ]
               }
             ]
           }
@@ -509,9 +516,16 @@ Looking at the previous example:
 --------------------------------------------------
 "collector": [
   {
-    "name": "SimpleTopScoreDocCollector",
-    "reason": "search_top_hits",
-    "time_in_nanos": 775274
+    "name": "QueryPhaseCollector",
+    "reason": "search_query_phase",
+    "time_in_nanos": 775274,
+    "children" : [
+      {
+        "name": "SimpleTopScoreDocCollector",
+        "reason": "search_top_hits",
+        "time_in_nanos": 775274
+      }
+    ]
   }
 ]
 --------------------------------------------------
@@ -520,14 +534,14 @@ Looking at the previous example:
 // TESTRESPONSE[s/(?<=[" ])\d+(\.\d+)?/$body.$_path/]
 
 
-We see a single collector named `SimpleTopScoreDocCollector` wrapped into
-`CancellableCollector`. `SimpleTopScoreDocCollector` is the default "scoring and
-sorting" `Collector` used by {es}. The `reason` field attempts to give a plain
-English description of the class name. The `time_in_nanos` is similar to the
-time in the Query tree: a wall-clock time inclusive of all children. Similarly,
-`children` lists all sub-collectors. The `CancellableCollector` that wraps
-`SimpleTopScoreDocCollector` is used by {es} to detect if the current search was
-cancelled and stop collecting documents as soon as it occurs.
+We see a top-level collector named `QueryPhaseCollector` which holds a child
+`SimpleTopScoreDocCollector`. `SimpleTopScoreDocCollector` is the  default
+"scoring and sorting" `Collector` used by {es}. The `reason` field attempts
+to give a plain English description of the class name. The `time_in_nanos`
+is similar to the time in the Query tree: a wall-clock time inclusive of all
+children. Similarly, `children` lists all sub-collectors. When aggregations
+are requested, the `QueryPhaseCollector` will hold an additional child
+collector with reason `aggregation` that is the one performing aggregations.
 
 It should be noted that Collector times are **independent** from the Query
 times. They are calculated, combined, and normalized independently! Due to the
@@ -537,7 +551,7 @@ Collectors into the Query section, so they are displayed in separate portions.
 For reference, the various collector reasons are:
 
 [horizontal]
-`search_sorted`::
+`search_top_hits`::
 
     A collector that scores and sorts documents. This is the most common collector and will be seen in most
     simple searches
@@ -547,20 +561,13 @@ For reference, the various collector reasons are:
     A collector that only counts the number of documents that match the query, but does not fetch the source.
     This is seen when `size: 0` is specified
 
-`search_terminate_after_count`::
-
-    A collector that terminates search execution after `n` matching documents have been found. This is seen
-    when the `terminate_after_count` query parameter has been specified
-
-`search_min_score`::
-
-    A collector that only returns matching documents that have a score greater than `n`. This is seen when
-    the top-level parameter `min_score` has been specified.
-
-`search_multi`::
+`search_query_phase`::
 
-    A collector that wraps several other collectors. This is seen when combinations of search, aggregations,
-    global aggs, and post_filters are combined in a single search.
+    A collector that incorporates collecting top hits as well aggregations as part of the query phase.
+    It supports terminating the search execution after `n` matching documents have been found (when
+    `terminate_after` is specified), as well as only returning matching documents that have a score
+    greater than `n` (when `min_score` is provided). Additionally, it is able to filter matching top
+    hits based on the provided `post_filter`.
 
 `search_timeout`::
 
@@ -723,21 +730,14 @@ The API returns the following result:
             "rewrite_time": 4769,
             "collector": [
               {
-                "name": "MultiCollector",
-                "reason": "search_multi",
+                "name": "QueryPhaseCollector",
+                "reason": "search_query_phase",
                 "time_in_nanos": 1945072,
                 "children": [
                   {
-                    "name": "FilteredCollector",
-                    "reason": "search_post_filter",
-                    "time_in_nanos": 500850,
-                    "children": [
-                      {
-                        "name": "SimpleTopScoreDocCollector",
-                        "reason": "search_top_hits",
-                        "time_in_nanos": 22577
-                      }
-                    ]
+                    "name": "SimpleTopScoreDocCollector",
+                    "reason": "search_top_hits",
+                    "time_in_nanos": 22577
                   },
                   {
                     "name": "BucketCollectorWrapper: [BucketCollectorWrapper[bucketCollector=[my_scoped_agg, my_global_agg]]]",
@@ -772,9 +772,9 @@ major portions of the query are represented:
 2. The second `TermQuery` (message:search) represents the `post_filter` query.
 
 The Collector tree is fairly straightforward, showing how a single
-CancellableCollector wraps a MultiCollector which also wraps a FilteredCollector
-to execute the post_filter (and in turn wraps the normal scoring
-SimpleCollector), a BucketCollector to run all scoped aggregations.
+QueryPhaseCollector that holds the normal scoring SimpleTopScoreDocCollector
+used to collect top hits, as well as BucketCollectorWrapper to run all scoped
+aggregations.
 
 ===== Understanding MultiTermQuery output
 

+ 0 - 98
server/src/main/java/org/elasticsearch/common/lucene/MinimumScoreCollector.java

@@ -1,98 +0,0 @@
-/*
- * 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.common.lucene;
-
-import org.apache.lucene.index.LeafReaderContext;
-import org.apache.lucene.search.Collector;
-import org.apache.lucene.search.CollectorManager;
-import org.apache.lucene.search.LeafCollector;
-import org.apache.lucene.search.Scorable;
-import org.apache.lucene.search.ScoreCachingWrappingScorer;
-import org.apache.lucene.search.ScoreMode;
-import org.apache.lucene.search.SimpleCollector;
-import org.apache.lucene.search.Weight;
-
-import java.io.IOException;
-import java.util.Collection;
-import java.util.List;
-
-/**
- * Collector that wraps another collector and collects only documents that have a score that's greater or equal than the
- * provided minimum score. Given that this collector filters documents out, it must not propagate the {@link Weight} to its
- * inner collector, as that may lead to exposing total hit count that does not reflect the filtering.
- */
-public class MinimumScoreCollector extends SimpleCollector {
-
-    private final Collector collector;
-    private final float minimumScore;
-
-    private Scorable scorer;
-    private LeafCollector leafCollector;
-
-    public MinimumScoreCollector(Collector collector, float minimumScore) {
-        this.collector = collector;
-        this.minimumScore = minimumScore;
-    }
-
-    @Override
-    public final void setWeight(Weight weight) {
-        // no-op: this collector filters documents out hence it must not propagate the weight to its inner collector,
-        // otherwise the total hit count may not reflect the filtering
-    }
-
-    @Override
-    public void setScorer(Scorable scorer) throws IOException {
-        if ((scorer instanceof ScoreCachingWrappingScorer) == false) {
-            scorer = ScoreCachingWrappingScorer.wrap(scorer);
-        }
-        this.scorer = scorer;
-        leafCollector.setScorer(scorer);
-    }
-
-    @Override
-    public void collect(int doc) throws IOException {
-        if (scorer.score() >= minimumScore) {
-            leafCollector.collect(doc);
-        }
-    }
-
-    @Override
-    public void doSetNextReader(LeafReaderContext context) throws IOException {
-        leafCollector = collector.getLeafCollector(context);
-    }
-
-    @Override
-    public ScoreMode scoreMode() {
-        return collector.scoreMode() == ScoreMode.TOP_SCORES ? ScoreMode.TOP_SCORES : ScoreMode.COMPLETE;
-    }
-
-    /**
-     * Creates a {@link CollectorManager} for {@link MinimumScoreCollector}, which enables inter-segment search concurrency
-     * when a <code>min_score</code> is provided as part of a search request.
-     */
-    public static <C extends Collector, T> CollectorManager<MinimumScoreCollector, T> createManager(
-        CollectorManager<C, T> collectorManager,
-        float minimumScore
-    ) {
-        return new CollectorManager<>() {
-            @Override
-            public MinimumScoreCollector newCollector() throws IOException {
-                return new MinimumScoreCollector(collectorManager.newCollector(), minimumScore);
-            }
-
-            @Override
-            public T reduce(Collection<MinimumScoreCollector> collectors) throws IOException {
-                @SuppressWarnings("unchecked")
-                List<C> innerCollectors = collectors.stream().map(minimumScoreCollector -> (C) minimumScoreCollector.collector).toList();
-                return collectorManager.reduce(innerCollectors);
-            }
-        };
-    }
-
-}

+ 0 - 89
server/src/main/java/org/elasticsearch/common/lucene/search/FilteredCollector.java

@@ -1,89 +0,0 @@
-/*
- * 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.common.lucene.search;
-
-import org.apache.lucene.index.LeafReaderContext;
-import org.apache.lucene.search.Collector;
-import org.apache.lucene.search.CollectorManager;
-import org.apache.lucene.search.FilterLeafCollector;
-import org.apache.lucene.search.LeafCollector;
-import org.apache.lucene.search.ScoreMode;
-import org.apache.lucene.search.ScorerSupplier;
-import org.apache.lucene.search.Weight;
-import org.apache.lucene.util.Bits;
-import org.elasticsearch.common.lucene.Lucene;
-
-import java.io.IOException;
-import java.util.Collection;
-import java.util.List;
-
-/**
- * Collector that wraps another collector and collects only documents that match the provided filter.
- * Given that this collector filters documents out, it must not propagate the {@link Weight} to its
- * inner collector, as that may lead to exposing total hit count that does not reflect the filtering.
- */
-public class FilteredCollector implements Collector {
-
-    private final Collector collector;
-    private final Weight filter;
-
-    public FilteredCollector(Collector collector, Weight filter) {
-        this.collector = collector;
-        this.filter = filter;
-    }
-
-    @Override
-    public final void setWeight(Weight weight) {
-        // no-op: this collector filters documents out hence it must not propagate the weight to its inner collector,
-        // otherwise the total hit count may not reflect the filtering
-    }
-
-    @Override
-    public LeafCollector getLeafCollector(LeafReaderContext context) throws IOException {
-        final ScorerSupplier filterScorerSupplier = filter.scorerSupplier(context);
-        final LeafCollector in = collector.getLeafCollector(context);
-        final Bits bits = Lucene.asSequentialAccessBits(context.reader().maxDoc(), filterScorerSupplier);
-
-        return new FilterLeafCollector(in) {
-            @Override
-            public void collect(int doc) throws IOException {
-                if (bits.get(doc)) {
-                    in.collect(doc);
-                }
-            }
-        };
-    }
-
-    @Override
-    public ScoreMode scoreMode() {
-        return collector.scoreMode();
-    }
-
-    /**
-     * Creates a {@link CollectorManager} for {@link FilteredCollector}, which enables inter-segment search concurrency
-     * when a <code>post_filter</code> is provided as part of a search request.
-     */
-    public static <C extends Collector, T> CollectorManager<FilteredCollector, T> createManager(
-        CollectorManager<C, T> collectorManager,
-        Weight filter
-    ) {
-        return new CollectorManager<>() {
-            @Override
-            public FilteredCollector newCollector() throws IOException {
-                return new FilteredCollector(collectorManager.newCollector(), filter);
-            }
-
-            @Override
-            public T reduce(Collection<FilteredCollector> collectors) throws IOException {
-                @SuppressWarnings("unchecked")
-                List<C> innerCollectors = collectors.stream().map(filteredCollector -> (C) filteredCollector.collector).toList();
-                return collectorManager.reduce(innerCollectors);
-            }
-        };
-    }
-}

+ 1 - 3
server/src/main/java/org/elasticsearch/search/profile/query/CollectorResult.java

@@ -37,10 +37,8 @@ public class CollectorResult extends ProfilerCollectorResult implements ToXConte
 
     public static final String REASON_SEARCH_COUNT = "search_count";
     public static final String REASON_SEARCH_TOP_HITS = "search_top_hits";
-    public static final String REASON_SEARCH_TERMINATE_AFTER_COUNT = "search_terminate_after_count";
-    public static final String REASON_SEARCH_POST_FILTER = "search_post_filter";
-    public static final String REASON_SEARCH_MIN_SCORE = "search_min_score";
     public static final String REASON_SEARCH_MULTI = "search_multi";
+    public static final String REASON_SEARCH_QUERY_PHASE = "search_query_phase";
     public static final String REASON_AGGREGATION = "aggregation";
     public static final String REASON_AGGREGATION_GLOBAL = "aggregation_global";
 

+ 44 - 86
server/src/main/java/org/elasticsearch/search/query/QueryPhase.java

@@ -17,18 +17,14 @@ import org.apache.lucene.search.BooleanQuery;
 import org.apache.lucene.search.Collector;
 import org.apache.lucene.search.CollectorManager;
 import org.apache.lucene.search.FieldDoc;
-import org.apache.lucene.search.MultiCollector;
 import org.apache.lucene.search.Query;
 import org.apache.lucene.search.ScoreDoc;
 import org.apache.lucene.search.ScoreMode;
-import org.apache.lucene.search.SimpleCollector;
 import org.apache.lucene.search.Sort;
 import org.apache.lucene.search.TopDocs;
 import org.apache.lucene.search.TotalHits;
 import org.apache.lucene.search.Weight;
 import org.elasticsearch.common.lucene.Lucene;
-import org.elasticsearch.common.lucene.MinimumScoreCollector;
-import org.elasticsearch.common.lucene.search.FilteredCollector;
 import org.elasticsearch.common.lucene.search.TopDocsAndMaxScore;
 import org.elasticsearch.common.util.concurrent.EsThreadPoolExecutor;
 import org.elasticsearch.common.util.concurrent.TaskExecutionTimeTrackingEsThreadPoolExecutor;
@@ -40,7 +36,6 @@ import org.elasticsearch.search.aggregations.AggregationPhase;
 import org.elasticsearch.search.internal.ContextIndexSearcher;
 import org.elasticsearch.search.internal.ScrollContext;
 import org.elasticsearch.search.internal.SearchContext;
-import org.elasticsearch.search.profile.Profilers;
 import org.elasticsearch.search.profile.query.InternalProfileCollector;
 import org.elasticsearch.search.profile.query.InternalProfileCollectorManager;
 import org.elasticsearch.search.rank.RankSearchContext;
@@ -52,15 +47,11 @@ import org.elasticsearch.threadpool.ThreadPool;
 
 import java.io.IOException;
 import java.util.ArrayList;
-import java.util.Arrays;
 import java.util.List;
 import java.util.concurrent.ExecutorService;
 
 import static org.elasticsearch.search.internal.SearchContext.TRACK_TOTAL_HITS_DISABLED;
-import static org.elasticsearch.search.profile.query.CollectorResult.REASON_SEARCH_MIN_SCORE;
-import static org.elasticsearch.search.profile.query.CollectorResult.REASON_SEARCH_MULTI;
-import static org.elasticsearch.search.profile.query.CollectorResult.REASON_SEARCH_POST_FILTER;
-import static org.elasticsearch.search.profile.query.CollectorResult.REASON_SEARCH_TERMINATE_AFTER_COUNT;
+import static org.elasticsearch.search.profile.query.CollectorResult.REASON_SEARCH_QUERY_PHASE;
 import static org.elasticsearch.search.query.TopDocsCollectorManagerFactory.createTopDocsCollectorFactory;
 
 /**
@@ -193,66 +184,59 @@ public class QueryPhase {
                 }
             }
 
-            // create the top docs collector last when the other collectors are known
             final TopDocsCollectorManagerFactory topDocsFactory = createTopDocsCollectorFactory(
                 searchContext,
                 searchContext.parsedPostFilter() != null || searchContext.minimumScore() != null
             );
 
-            CollectorManager<Collector, Void> collectorManager = wrapWithProfilerCollectorManagerIfNeeded(
-                searchContext.getProfilers(),
-                topDocsFactory.collectorManager(),
-                topDocsFactory.profilerName
-            );
+            CollectorManager<Collector, Void> topDocsCollectorManager = topDocsFactory.collectorManager();
+            if (searchContext.getProfilers() != null) {
+                Collector topDocsCollector = topDocsCollectorManager.newCollector();
+                InternalProfileCollector profileCollector = new InternalProfileCollector(topDocsCollector, topDocsFactory.profilerName);
+                topDocsCollectorManager = new InternalProfileCollectorManager(profileCollector);
+            }
 
-            if (searchContext.terminateAfter() != SearchContext.DEFAULT_TERMINATE_AFTER) {
-                // add terminate_after before the filter collectors
-                // it will only be applied on documents accepted by these filter collectors
-                TerminateAfterCollector terminateAfterCollector = new TerminateAfterCollector(searchContext.terminateAfter());
-                final Collector collector = collectorManager.newCollector();
-                collectorManager = wrapWithProfilerCollectorManagerIfNeeded(
-                    searchContext.getProfilers(),
-                    new SingleThreadCollectorManager(MultiCollector.wrap(terminateAfterCollector, collector)),
-                    REASON_SEARCH_TERMINATE_AFTER_COUNT,
-                    collector
-                );
+            Collector topDocsCollector = topDocsCollectorManager.newCollector();
+            Collector aggsCollector = null;
+            Weight postFilterWeight = null;
+            if (searchContext.aggregations() != null) {
+                aggsCollector = searchContext.aggregations().getAggsCollectorManager().newCollector();
             }
             if (searchContext.parsedPostFilter() != null) {
-                // add post filters before aggregations
-                // it will only be applied to top hits
-                final Weight filterWeight = searcher.createWeight(
+                postFilterWeight = searcher.createWeight(
                     searcher.rewrite(searchContext.parsedPostFilter().query()),
                     ScoreMode.COMPLETE_NO_SCORES,
                     1f
                 );
-                final Collector collector = collectorManager.newCollector();
-                collectorManager = wrapWithProfilerCollectorManagerIfNeeded(
-                    searchContext.getProfilers(),
-                    new SingleThreadCollectorManager(new FilteredCollector(collector, filterWeight)),
-                    REASON_SEARCH_POST_FILTER,
-                    collector
-                );
-            }
-            if (searchContext.aggregations() != null) {
-                final Collector collector = collectorManager.newCollector();
-                final Collector aggsCollector = searchContext.aggregations().getAggsCollectorManager().newCollector();
-                collectorManager = wrapWithProfilerCollectorManagerIfNeeded(
-                    searchContext.getProfilers(),
-                    new SingleThreadCollectorManager(MultiCollector.wrap(collector, aggsCollector)),
-                    REASON_SEARCH_MULTI,
-                    collector,
-                    aggsCollector
-                );
             }
-            if (searchContext.minimumScore() != null) {
-                final Collector collector = collectorManager.newCollector();
-                // apply the minimum score after multi collector so we filter aggs as well
-                collectorManager = wrapWithProfilerCollectorManagerIfNeeded(
-                    searchContext.getProfilers(),
-                    new SingleThreadCollectorManager(new MinimumScoreCollector(collector, searchContext.minimumScore())),
-                    REASON_SEARCH_MIN_SCORE,
-                    collector
-                );
+            QueryPhaseCollector queryPhaseCollector = new QueryPhaseCollector(
+                topDocsCollector,
+                postFilterWeight,
+                searchContext.terminateAfter(),
+                aggsCollector,
+                searchContext.minimumScore()
+            );
+
+            SingleThreadCollectorManager collectorManager;
+            if (searchContext.getProfilers() == null) {
+                collectorManager = new SingleThreadCollectorManager(queryPhaseCollector);
+            } else {
+                InternalProfileCollector profileCollector;
+                if (aggsCollector == null) {
+                    profileCollector = new InternalProfileCollector(
+                        queryPhaseCollector,
+                        REASON_SEARCH_QUERY_PHASE,
+                        (InternalProfileCollector) topDocsCollector
+                    );
+                } else {
+                    profileCollector = new InternalProfileCollector(
+                        queryPhaseCollector,
+                        REASON_SEARCH_QUERY_PHASE,
+                        (InternalProfileCollector) topDocsCollector,
+                        (InternalProfileCollector) aggsCollector
+                    );
+                }
+                collectorManager = new InternalProfileCollectorManager(profileCollector);
             }
 
             final Runnable timeoutRunnable = getTimeoutCheck(searchContext);
@@ -262,6 +246,9 @@ public class QueryPhase {
 
             try {
                 searchWithCollectorManager(searchContext, searcher, query, collectorManager, timeoutRunnable != null);
+                if (queryPhaseCollector.isTerminatedAfter()) {
+                    queryResult.terminatedEarly(true);
+                }
                 queryResult.topDocs(topDocsFactory.topDocsAndMaxScore(), topDocsFactory.sortValueFormats);
                 ExecutorService executor = searchContext.indexShard().getThreadPool().executor(ThreadPool.Names.SEARCH);
                 assert executor instanceof TaskExecutionTimeTrackingEsThreadPoolExecutor
@@ -283,23 +270,6 @@ public class QueryPhase {
         }
     }
 
-    private static CollectorManager<Collector, Void> wrapWithProfilerCollectorManagerIfNeeded(
-        Profilers profilers,
-        CollectorManager<Collector, Void> collectorManager,
-        String profilerName,
-        Collector... children
-    ) throws IOException {
-        if (profilers == null) {
-            return collectorManager;
-        }
-        InternalProfileCollector[] childProfileCollectors = Arrays.stream(children)
-            .map(c -> (InternalProfileCollector) c)
-            .toArray(InternalProfileCollector[]::new);
-        return new InternalProfileCollectorManager(
-            new InternalProfileCollector(collectorManager.newCollector(), profilerName, childProfileCollectors)
-        );
-    }
-
     private static void searchWithCollectorManager(
         SearchContext searchContext,
         ContextIndexSearcher searcher,
@@ -315,8 +285,6 @@ public class QueryPhase {
         QuerySearchResult queryResult = searchContext.queryResult();
         try {
             searcher.search(query, collectorManager);
-        } catch (TerminateAfterCollector.EarlyTerminationException e) {
-            queryResult.terminatedEarly(true);
         } catch (TimeExceededException e) {
             assert timeoutSet : "TimeExceededException thrown even though timeout wasn't set";
             if (searchContext.request().allowPartialSearchResults() == false) {
@@ -376,14 +344,4 @@ public class QueryPhase {
             return this;
         }
     }
-
-    private static final Collector EMPTY_COLLECTOR = new SimpleCollector() {
-        @Override
-        public void collect(int doc) {}
-
-        @Override
-        public ScoreMode scoreMode() {
-            return ScoreMode.COMPLETE_NO_SCORES;
-        }
-    };
 }

+ 337 - 0
server/src/main/java/org/elasticsearch/search/query/QueryPhaseCollector.java

@@ -0,0 +1,337 @@
+/*
+ * 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.search.query;
+
+import org.apache.lucene.index.LeafReaderContext;
+import org.apache.lucene.search.CollectionTerminatedException;
+import org.apache.lucene.search.Collector;
+import org.apache.lucene.search.DocIdSetIterator;
+import org.apache.lucene.search.FilterScorable;
+import org.apache.lucene.search.LeafCollector;
+import org.apache.lucene.search.Scorable;
+import org.apache.lucene.search.ScoreCachingWrappingScorer;
+import org.apache.lucene.search.ScoreMode;
+import org.apache.lucene.search.ScorerSupplier;
+import org.apache.lucene.search.Weight;
+import org.apache.lucene.util.Bits;
+import org.elasticsearch.common.lucene.Lucene;
+
+import java.io.IOException;
+import java.util.Objects;
+
+/**
+ * Top-level collector used in the query phase to perform top hits collection as well as aggs collection.
+ * Inspired by {@link org.apache.lucene.search.MultiCollector} but specialized for wrapping two collectors and filtering collected
+ * documents as follows:
+ * - through an optional <code>post_filter</code> that is applied to the top hits collection
+ * - through an optional <code>min_score</code> threshold, which is applied to both the top hits as well as aggs.
+ * Supports also terminating the collection after a certain number of documents have been collected (<code>terminate_after</code>).
+ *
+ * When top docs as well as aggs are collected (because both collectors were provided), skipping low scoring hits via
+ * {@link Scorable#setMinCompetitiveScore(float)} is not supported for either of the collectors.
+ */
+final class QueryPhaseCollector implements Collector {
+    private final Collector aggsCollector;
+    private final Collector topDocsCollector;
+    private final int terminateAfter;
+    private final Weight postFilterWeight;
+    private final Float minScore;
+    private final boolean cacheScores;
+
+    private int numCollected;
+    private boolean terminatedAfter = false;
+
+    QueryPhaseCollector(Collector topDocsCollector, Weight postFilterWeight, int terminateAfter, Collector aggsCollector, Float minScore) {
+        this.topDocsCollector = Objects.requireNonNull(topDocsCollector);
+        this.postFilterWeight = postFilterWeight;
+        if (terminateAfter < 0) {
+            throw new IllegalArgumentException("terminateAfter must be greater than or equal to 0");
+        }
+        this.terminateAfter = terminateAfter;
+        this.aggsCollector = aggsCollector;
+        this.minScore = minScore;
+        this.cacheScores = aggsCollector != null && topDocsCollector.scoreMode().needsScores() && aggsCollector.scoreMode().needsScores();
+    }
+
+    @Override
+    public void setWeight(Weight weight) {
+        if (postFilterWeight == null && minScore == null) {
+            // propagate the weight when we do no additional filtering over the docs that are collected
+            // when post_filter or min_score are provided, the collection cannot be shortcut via Weight#count
+            topDocsCollector.setWeight(weight);
+        }
+        if (aggsCollector != null && minScore == null) {
+            // min_score is applied to aggs collection as well as top docs collection, though BucketCollectorWrapper does not override it.
+            aggsCollector.setWeight(weight);
+        }
+    }
+
+    @Override
+    public ScoreMode scoreMode() {
+        ScoreMode scoreMode;
+        if (aggsCollector == null) {
+            scoreMode = topDocsCollector.scoreMode();
+        } else {
+            assert aggsCollector.scoreMode() != ScoreMode.TOP_SCORES : "aggs never rely on setMinCompetitiveScore";
+            if (topDocsCollector.scoreMode() == aggsCollector.scoreMode()) {
+                scoreMode = topDocsCollector.scoreMode();
+            } else if (topDocsCollector.scoreMode().needsScores() || aggsCollector.scoreMode().needsScores()) {
+                scoreMode = ScoreMode.COMPLETE;
+            } else {
+                scoreMode = ScoreMode.COMPLETE_NO_SCORES;
+            }
+            // TODO for aggs that return TOP_DOCS, score mode becomes exhaustive unless top docs collector agrees on the score mode
+        }
+        if (minScore != null) {
+            // TODO if we had TOP_DOCS, shouldn't we return TOP_DOCS_WITH_SCORES instead of COMPLETE?
+            scoreMode = scoreMode == ScoreMode.TOP_SCORES ? ScoreMode.TOP_SCORES : ScoreMode.COMPLETE;
+        }
+        return scoreMode;
+    }
+
+    /**
+     * @return whether the collection was terminated based on the provided <code>terminate_after</code> value
+     */
+    boolean isTerminatedAfter() {
+        return terminatedAfter;
+    }
+
+    private boolean shouldCollectTopDocs(int doc, Scorable scorer, Bits postFilterBits) throws IOException {
+        if (isDocWithinMinScore(scorer)) {
+            if (doesDocMatchPostFilter(doc, postFilterBits)) {
+                // terminate_after is purposely applied after post_filter, and terminates aggs collection based on number of filtered
+                // top hits that have been collected. Strange feature, but that has been behaviour for a long time.
+                applyTerminateAfter();
+                return true;
+            }
+        }
+        return false;
+    }
+
+    private boolean isDocWithinMinScore(Scorable scorer) throws IOException {
+        return minScore == null || scorer.score() >= minScore;
+    }
+
+    private boolean doesDocMatchPostFilter(int doc, Bits postFilterBits) {
+        return postFilterBits == null || postFilterBits.get(doc);
+    }
+
+    private void applyTerminateAfter() {
+        if (terminateAfter > 0 && numCollected >= terminateAfter) {
+            terminatedAfter = true;
+            throw new CollectionTerminatedException();
+        }
+    }
+
+    private Bits getPostFilterBits(LeafReaderContext context) throws IOException {
+        if (postFilterWeight == null) {
+            return null;
+        }
+        final ScorerSupplier filterScorerSupplier = postFilterWeight.scorerSupplier(context);
+        return Lucene.asSequentialAccessBits(context.reader().maxDoc(), filterScorerSupplier);
+    }
+
+    @Override
+    public LeafCollector getLeafCollector(LeafReaderContext context) throws IOException {
+        applyTerminateAfter();
+        Bits postFilterBits = getPostFilterBits(context);
+
+        if (aggsCollector == null) {
+            LeafCollector topDocsLeafCollector;
+            try {
+                topDocsLeafCollector = topDocsCollector.getLeafCollector(context);
+            } catch (@SuppressWarnings("unused") CollectionTerminatedException e) {
+                // TODO we keep on collecting although we have nothing to collect (there is no top docs nor aggs leaf collector).
+                // The reason is only to set the early terminated flag to the QueryResult like some tests expect. This needs fixing.
+                if (terminateAfter == 0) {
+                    throw e;
+                }
+                topDocsLeafCollector = null;
+            }
+            return new TopDocsLeafCollector(postFilterBits, topDocsLeafCollector);
+        }
+
+        LeafCollector topDocsLeafCollector;
+        try {
+            topDocsLeafCollector = topDocsCollector.getLeafCollector(context);
+        } catch (@SuppressWarnings("unused") CollectionTerminatedException e) {
+            // top docs collector does not need this segment, but the aggs collector does.
+            topDocsLeafCollector = null;
+        }
+
+        LeafCollector aggsLeafCollector;
+        try {
+            aggsLeafCollector = aggsCollector.getLeafCollector(context);
+        } catch (@SuppressWarnings("unused") CollectionTerminatedException e) {
+            // aggs collector does not need this segment, but the top docs collector may.
+            if (topDocsLeafCollector == null) {
+                // TODO we keep on collecting although we have nothing to collect (there is no top docs nor aggs leaf collector).
+                // The reason is only to set the early terminated flag to the QueryResult. We should fix this.
+                if (terminateAfter == 0) {
+                    throw e;
+                }
+            }
+            aggsLeafCollector = null;
+        }
+        // say that the aggs collector early terminates while the top docs collector does not, we still want to wrap in the same way
+        // to enforce that setMinCompetitiveScore is a no-op. Otherwise we may allow the top docs collector to skip non competitive
+        // hits despite the score mode of the Collector did not allow it.
+        return new CompositeLeafCollector(postFilterBits, topDocsLeafCollector, aggsLeafCollector);
+    }
+
+    private class TopDocsLeafCollector implements LeafCollector {
+        private final Bits postFilterBits;
+        private LeafCollector topDocsLeafCollector;
+        private Scorable scorer;
+
+        TopDocsLeafCollector(Bits postFilterBits, LeafCollector topDocsLeafCollector) {
+            this.postFilterBits = postFilterBits;
+            this.topDocsLeafCollector = topDocsLeafCollector;
+        }
+
+        @Override
+        public void setScorer(Scorable scorer) throws IOException {
+            if (cacheScores) {
+                scorer = ScoreCachingWrappingScorer.wrap(scorer);
+            }
+            if (terminateAfter > 0) {
+                scorer = new FilterScorable(scorer) {
+                    @Override
+                    public void setMinCompetitiveScore(float minScore) {
+                        // Ignore calls to setMinCompetitiveScore when terminate_after is used, otherwise early termination
+                        // of total hits tracking makes it impossible to terminate after.
+                        // TODO the reason is only to set the early terminated flag to the QueryResult. We should fix this.
+                    }
+                };
+            }
+            if (topDocsLeafCollector != null) {
+                topDocsLeafCollector.setScorer(scorer);
+            }
+            this.scorer = scorer;
+        }
+
+        @Override
+        public DocIdSetIterator competitiveIterator() throws IOException {
+            if (topDocsLeafCollector != null) {
+                return topDocsLeafCollector.competitiveIterator();
+            }
+            return null;
+        }
+
+        @Override
+        public void collect(int doc) throws IOException {
+            if (shouldCollectTopDocs(doc, scorer, postFilterBits)) {
+                numCollected++;
+                if (topDocsLeafCollector != null) {
+                    try {
+                        topDocsLeafCollector.collect(doc);
+                    } catch (@SuppressWarnings("unused") CollectionTerminatedException e) {
+                        topDocsLeafCollector = null;
+                        // TODO we keep on collecting although we have nothing to collect (there is no top docs nor aggs leaf
+                        // collector).
+                        // The reason is only to set the early terminated flag to the QueryResult. We should fix this.
+                        if (terminateAfter == 0) {
+                            throw e;
+                        }
+                    }
+                }
+            }
+        }
+    }
+
+    private class CompositeLeafCollector implements LeafCollector {
+        private final Bits postFilterBits;
+        private LeafCollector topDocsLeafCollector;
+        private LeafCollector aggsLeafCollector;
+        private Scorable scorer;
+
+        CompositeLeafCollector(Bits postFilterBits, LeafCollector topDocsLeafCollector, LeafCollector aggsLeafCollector) {
+            this.postFilterBits = postFilterBits;
+            this.topDocsLeafCollector = topDocsLeafCollector;
+            this.aggsLeafCollector = aggsLeafCollector;
+        }
+
+        @Override
+        public void setScorer(Scorable scorer) throws IOException {
+            if (cacheScores) {
+                scorer = ScoreCachingWrappingScorer.wrap(scorer);
+            }
+            scorer = new FilterScorable(scorer) {
+                @Override
+                public void setMinCompetitiveScore(float minScore) {
+                    // Ignore calls to setMinCompetitiveScore so that if the top docs collector
+                    // wants to skip low-scoring hits, the aggs collector still sees all hits.
+                    // this is important also for terminate_after in case used when total hits tracking is early terminated.
+                }
+            };
+            if (topDocsLeafCollector != null) {
+                topDocsLeafCollector.setScorer(scorer);
+            }
+            if (aggsLeafCollector != null) {
+                aggsLeafCollector.setScorer(scorer);
+            }
+            this.scorer = scorer;
+        }
+
+        @Override
+        public void collect(int doc) throws IOException {
+            if (shouldCollectTopDocs(doc, scorer, postFilterBits)) {
+                numCollected++;
+                if (topDocsLeafCollector != null) {
+                    try {
+                        topDocsLeafCollector.collect(doc);
+                    } catch (@SuppressWarnings("unused") CollectionTerminatedException e) {
+                        topDocsLeafCollector = null;
+                        // top docs collector does not need this segment, but the aggs collector may.
+                        if (aggsLeafCollector == null) {
+                            // TODO we keep on collecting although we have nothing to collect (there is no top docs nor aggs leaf
+                            // collector).
+                            // The reason is only to set the early terminated flag to the QueryResult. We should fix this.
+                            if (terminateAfter == 0) {
+                                throw e;
+                            }
+                        }
+                    }
+                }
+            }
+            // min_score is applied to aggs as well as top hits
+            if (isDocWithinMinScore(scorer)) {
+                if (aggsLeafCollector != null) {
+                    try {
+                        aggsLeafCollector.collect(doc);
+                    } catch (@SuppressWarnings("unused") CollectionTerminatedException e) {
+                        aggsLeafCollector = null;
+                        // aggs collector does not need this segment, but the top docs collector may.
+                        if (topDocsLeafCollector == null) {
+                            // TODO we keep on collecting although we have nothing to collect (there is no top docs nor aggs leaf
+                            // collector).
+                            // The reason is only to set the early terminated flag to the QueryResult. We should fix this.
+                            if (terminateAfter == 0) {
+                                throw e;
+                            }
+                        }
+                    }
+                }
+            }
+        }
+
+        @Override
+        public DocIdSetIterator competitiveIterator() throws IOException {
+            // TODO we expose the competitive iterator only when one of the two sub-leaf collectors has early terminated,
+            // it could be a good idea to expose a disjunction of the two when both are not null
+            if (topDocsLeafCollector == null) {
+                return aggsLeafCollector.competitiveIterator();
+            }
+            if (aggsLeafCollector == null) {
+                return topDocsLeafCollector.competitiveIterator();
+            }
+            return null;
+        }
+    }
+}

+ 0 - 74
server/src/main/java/org/elasticsearch/search/query/TerminateAfterCollector.java

@@ -1,74 +0,0 @@
-/*
- * 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.search.query;
-
-import org.apache.lucene.index.LeafReaderContext;
-import org.apache.lucene.search.Collector;
-import org.apache.lucene.search.LeafCollector;
-import org.apache.lucene.search.Scorable;
-import org.apache.lucene.search.ScoreMode;
-
-import java.io.IOException;
-
-/**
- * A {@link Collector} that forcibly early terminates collection after a certain number of hits have been collected.
- * Terminates the collection across all collectors by throwing an {@link EarlyTerminationException} once the threshold is reached.
- */
-class TerminateAfterCollector implements Collector {
-    static final class EarlyTerminationException extends RuntimeException {
-        private EarlyTerminationException(String msg) {
-            super(msg);
-        }
-
-        @Override
-        public Throwable fillInStackTrace() {
-            // never re-thrown so we can save the expensive stacktrace
-            return this;
-        }
-    }
-
-    private final int maxCountHits;
-    private int numCollected;
-
-    /**
-     *
-     * @param maxCountHits the number of hits to collect, after which the collection must be early terminated
-     */
-    TerminateAfterCollector(int maxCountHits) {
-        this.maxCountHits = maxCountHits;
-    }
-
-    @Override
-    public LeafCollector getLeafCollector(LeafReaderContext context) throws IOException {
-        if (numCollected >= maxCountHits) {
-            earlyTerminate();
-        }
-        return new LeafCollector() {
-            @Override
-            public void setScorer(Scorable scorer) {}
-
-            @Override
-            public void collect(int doc) {
-                if (++numCollected > maxCountHits) {
-                    earlyTerminate();
-                }
-            }
-        };
-    }
-
-    @Override
-    public ScoreMode scoreMode() {
-        // this collector is not exhaustive, as it early terminates, and never needs scores
-        return ScoreMode.TOP_DOCS;
-    }
-
-    private void earlyTerminate() {
-        throw new EarlyTerminationException("early termination [CountBased]");
-    }
-}

+ 0 - 141
server/src/test/java/org/elasticsearch/common/lucene/MinimumScoreCollectorTests.java

@@ -1,141 +0,0 @@
-/*
- * 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.common.lucene;
-
-import org.apache.lucene.document.Document;
-import org.apache.lucene.document.Field;
-import org.apache.lucene.document.StringField;
-import org.apache.lucene.index.IndexReader;
-import org.apache.lucene.index.Term;
-import org.apache.lucene.search.BooleanClause;
-import org.apache.lucene.search.BooleanQuery;
-import org.apache.lucene.search.BoostQuery;
-import org.apache.lucene.search.CollectorManager;
-import org.apache.lucene.search.IndexSearcher;
-import org.apache.lucene.search.MatchAllDocsQuery;
-import org.apache.lucene.search.TermQuery;
-import org.apache.lucene.search.TopDocs;
-import org.apache.lucene.search.TopScoreDocCollector;
-import org.apache.lucene.search.TotalHitCountCollector;
-import org.apache.lucene.search.similarities.BM25Similarity;
-import org.apache.lucene.store.Directory;
-import org.apache.lucene.tests.index.RandomIndexWriter;
-import org.elasticsearch.core.IOUtils;
-import org.elasticsearch.test.ESTestCase;
-
-import java.io.IOException;
-
-public class MinimumScoreCollectorTests extends ESTestCase {
-
-    private Directory directory;
-    private IndexReader reader;
-    private IndexSearcher searcher;
-    private int numDocs;
-
-    @Override
-    public void setUp() throws Exception {
-        super.setUp();
-        directory = newDirectory();
-        RandomIndexWriter writer = new RandomIndexWriter(random(), directory, newIndexWriterConfig());
-        numDocs = randomIntBetween(900, 1000);
-        for (int i = 0; i < numDocs; i++) {
-            Document doc = new Document();
-            doc.add(new StringField("field1", "value", Field.Store.NO));
-            if (i == 0) {
-                doc.add(new StringField("field2", "value", Field.Store.NO));
-            }
-            writer.addDocument(doc);
-        }
-        writer.flush();
-        reader = writer.getReader();
-        searcher = newSearcher(reader);
-        searcher.setSimilarity(new BM25Similarity());
-        writer.close();
-    }
-
-    @Override
-    public void tearDown() throws Exception {
-        super.tearDown();
-        IOUtils.close(reader, directory);
-    }
-
-    public void testMinScoreFiltering() throws IOException {
-        float maxScore;
-        float thresholdScore;
-        BooleanQuery booleanQuery = new BooleanQuery.Builder().add(new TermQuery(new Term("field1", "value")), BooleanClause.Occur.MUST)
-            .add(new BoostQuery(new TermQuery(new Term("field2", "value")), 200f), BooleanClause.Occur.SHOULD)
-            .build();
-        {
-            TopScoreDocCollector topScoreDocCollector = TopScoreDocCollector.create(2, 1000);
-            searcher.search(booleanQuery, topScoreDocCollector);
-            TopDocs topDocs = topScoreDocCollector.topDocs();
-            assertEquals(numDocs, topDocs.totalHits.value);
-            maxScore = topDocs.scoreDocs[0].score;
-            thresholdScore = topDocs.scoreDocs[1].score;
-        }
-        {
-            TopScoreDocCollector topScoreDocCollector = TopScoreDocCollector.create(1, 1000);
-            searcher.search(booleanQuery, new MinimumScoreCollector(topScoreDocCollector, maxScore));
-            assertEquals(1, topScoreDocCollector.topDocs().totalHits.value);
-        }
-        {
-            TopScoreDocCollector topScoreDocCollector = TopScoreDocCollector.create(1, 1000);
-            searcher.search(booleanQuery, new MinimumScoreCollector(topScoreDocCollector, thresholdScore));
-            assertEquals(numDocs, topScoreDocCollector.topDocs().totalHits.value);
-        }
-        {
-            TopScoreDocCollector topScoreDocCollector = TopScoreDocCollector.create(1, 1000);
-            searcher.search(booleanQuery, new MinimumScoreCollector(topScoreDocCollector, maxScore + 100f));
-            assertEquals(0, topScoreDocCollector.topDocs().totalHits.value);
-        }
-    }
-
-    public void testWeightIsNotPropagated() throws IOException {
-        {
-            TotalHitCountCollector totalHitCountCollector = new TotalHitCountCollector();
-            searcher.search(new MatchAllDocsQuery(), totalHitCountCollector);
-            assertEquals(reader.maxDoc(), totalHitCountCollector.getTotalHits());
-        }
-        {
-            TotalHitCountCollector totalHitCountCollector = new TotalHitCountCollector();
-            searcher.search(new MatchAllDocsQuery(), new MinimumScoreCollector(totalHitCountCollector, 100f));
-            assertEquals(0, totalHitCountCollector.getTotalHits());
-        }
-    }
-
-    public void testManager() throws IOException {
-        float maxScore;
-        float thresholdScore;
-        BooleanQuery booleanQuery = new BooleanQuery.Builder().add(new TermQuery(new Term("field1", "value")), BooleanClause.Occur.MUST)
-            .add(new BoostQuery(new TermQuery(new Term("field2", "value")), 200f), BooleanClause.Occur.SHOULD)
-            .build();
-        {
-            CollectorManager<TopScoreDocCollector, TopDocs> topDocsManager = TopScoreDocCollector.createSharedManager(2, null, 1000);
-            TopDocs topDocs = searcher.search(booleanQuery, topDocsManager);
-            assertEquals(numDocs, topDocs.totalHits.value);
-            maxScore = topDocs.scoreDocs[0].score;
-            thresholdScore = topDocs.scoreDocs[1].score;
-        }
-        {
-            CollectorManager<TopScoreDocCollector, TopDocs> topDocsManager = TopScoreDocCollector.createSharedManager(1, null, 1000);
-            TopDocs topDocs = searcher.search(booleanQuery, MinimumScoreCollector.createManager(topDocsManager, maxScore));
-            assertEquals(1, topDocs.totalHits.value);
-        }
-        {
-            CollectorManager<TopScoreDocCollector, TopDocs> topDocsManager = TopScoreDocCollector.createSharedManager(1, null, 1000);
-            TopDocs topDocs = searcher.search(booleanQuery, MinimumScoreCollector.createManager(topDocsManager, thresholdScore));
-            assertEquals(numDocs, topDocs.totalHits.value);
-        }
-        {
-            CollectorManager<TopScoreDocCollector, TopDocs> topDocsManager = TopScoreDocCollector.createSharedManager(1, null, 1000);
-            TopDocs topDocs = searcher.search(booleanQuery, MinimumScoreCollector.createManager(topDocsManager, maxScore + 100f));
-            assertEquals(0, topDocs.totalHits.value);
-        }
-    }
-}

+ 0 - 125
server/src/test/java/org/elasticsearch/common/lucene/search/FilteredCollectorTests.java

@@ -1,125 +0,0 @@
-/*
- * 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.common.lucene.search;
-
-import org.apache.lucene.document.Document;
-import org.apache.lucene.document.Field;
-import org.apache.lucene.document.StringField;
-import org.apache.lucene.index.IndexReader;
-import org.apache.lucene.index.Term;
-import org.apache.lucene.search.CollectorManager;
-import org.apache.lucene.search.IndexSearcher;
-import org.apache.lucene.search.MatchAllDocsQuery;
-import org.apache.lucene.search.ScoreMode;
-import org.apache.lucene.search.TermQuery;
-import org.apache.lucene.search.TopDocs;
-import org.apache.lucene.search.TopScoreDocCollector;
-import org.apache.lucene.search.TotalHitCountCollector;
-import org.apache.lucene.search.Weight;
-import org.apache.lucene.store.Directory;
-import org.apache.lucene.tests.index.RandomIndexWriter;
-import org.elasticsearch.core.IOUtils;
-import org.elasticsearch.test.ESTestCase;
-
-import java.io.IOException;
-
-public class FilteredCollectorTests extends ESTestCase {
-
-    private Directory directory;
-    private IndexReader reader;
-    private IndexSearcher searcher;
-    private int numDocs;
-
-    @Override
-    public void setUp() throws Exception {
-        super.setUp();
-        directory = newDirectory();
-        RandomIndexWriter writer = new RandomIndexWriter(random(), directory, newIndexWriterConfig());
-        numDocs = randomIntBetween(900, 1000);
-        for (int i = 0; i < numDocs; i++) {
-            Document doc = new Document();
-            doc.add(new StringField("field1", "value", Field.Store.NO));
-            if (i == 0) {
-                doc.add(new StringField("field2", "value", Field.Store.NO));
-            }
-            writer.addDocument(doc);
-        }
-        writer.flush();
-        reader = writer.getReader();
-        searcher = newSearcher(reader);
-        writer.close();
-    }
-
-    @Override
-    public void tearDown() throws Exception {
-        super.tearDown();
-        IOUtils.close(reader, directory);
-    }
-
-    public void testFiltering() throws IOException {
-        {
-            TopScoreDocCollector topScoreDocCollector = TopScoreDocCollector.create(1, 1000);
-            searcher.search(new MatchAllDocsQuery(), topScoreDocCollector);
-            assertEquals(numDocs, topScoreDocCollector.topDocs().totalHits.value);
-        }
-        {
-            TopScoreDocCollector topScoreDocCollector = TopScoreDocCollector.create(1, 1000);
-            TermQuery termQuery = new TermQuery(new Term("field2", "value"));
-            Weight filterWeight = termQuery.createWeight(searcher, ScoreMode.TOP_DOCS, 1f);
-            searcher.search(new MatchAllDocsQuery(), new FilteredCollector(topScoreDocCollector, filterWeight));
-            assertEquals(1, topScoreDocCollector.topDocs().totalHits.value);
-        }
-        {
-            TopScoreDocCollector topScoreDocCollector = TopScoreDocCollector.create(1, 1000);
-            TermQuery termQuery = new TermQuery(new Term("field1", "value"));
-            Weight filterWeight = termQuery.createWeight(searcher, ScoreMode.TOP_DOCS, 1f);
-            searcher.search(new MatchAllDocsQuery(), new FilteredCollector(topScoreDocCollector, filterWeight));
-            assertEquals(numDocs, topScoreDocCollector.topDocs().totalHits.value);
-        }
-    }
-
-    public void testWeightIsNotPropagated() throws IOException {
-        {
-            TotalHitCountCollector totalHitCountCollector = new TotalHitCountCollector();
-            searcher.search(new MatchAllDocsQuery(), totalHitCountCollector);
-            assertEquals(reader.maxDoc(), totalHitCountCollector.getTotalHits());
-        }
-        {
-            TotalHitCountCollector totalHitCountCollector = new TotalHitCountCollector();
-            TermQuery termQuery = new TermQuery(new Term("field2", "value"));
-            Weight filterWeight = termQuery.createWeight(searcher, ScoreMode.TOP_DOCS, 1f);
-            searcher.search(new MatchAllDocsQuery(), new FilteredCollector(totalHitCountCollector, filterWeight));
-            assertEquals(1, totalHitCountCollector.getTotalHits());
-        }
-    }
-
-    public void testManager() throws IOException {
-        {
-            CollectorManager<TopScoreDocCollector, TopDocs> topDocsManager = TopScoreDocCollector.createSharedManager(1, null, 1000);
-            TopDocs topDocs = searcher.search(new MatchAllDocsQuery(), topDocsManager);
-            assertEquals(numDocs, topDocs.totalHits.value);
-        }
-        {
-            CollectorManager<TopScoreDocCollector, TopDocs> topDocsManager = TopScoreDocCollector.createSharedManager(1, null, 1000);
-            TermQuery termQuery = new TermQuery(new Term("field2", "value"));
-            Weight filterWeight = termQuery.createWeight(searcher, ScoreMode.TOP_DOCS, 1f);
-            CollectorManager<FilteredCollector, TopDocs> filteredManager = FilteredCollector.createManager(topDocsManager, filterWeight);
-            TopDocs topDocs = searcher.search(new MatchAllDocsQuery(), filteredManager);
-            assertEquals(1, topDocs.totalHits.value);
-        }
-        {
-            CollectorManager<TopScoreDocCollector, TopDocs> topDocsManager = TopScoreDocCollector.createSharedManager(1, null, 1000);
-            TermQuery termQuery = new TermQuery(new Term("field1", "value"));
-            Weight filterWeight = termQuery.createWeight(searcher, ScoreMode.TOP_DOCS, 1f);
-            CollectorManager<FilteredCollector, TopDocs> filteredManager = FilteredCollector.createManager(topDocsManager, filterWeight);
-            TopDocs topDocs = searcher.search(new MatchAllDocsQuery(), filteredManager);
-            assertEquals(numDocs, topDocs.totalHits.value);
-        }
-    }
-}

+ 1206 - 0
server/src/test/java/org/elasticsearch/search/query/QueryPhaseCollectorTests.java

@@ -0,0 +1,1206 @@
+/*
+ * 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.search.query;
+
+import org.apache.lucene.document.Document;
+import org.apache.lucene.document.Field;
+import org.apache.lucene.document.StringField;
+import org.apache.lucene.index.IndexReader;
+import org.apache.lucene.index.LeafReaderContext;
+import org.apache.lucene.index.Term;
+import org.apache.lucene.search.BooleanClause;
+import org.apache.lucene.search.BooleanQuery;
+import org.apache.lucene.search.BoostQuery;
+import org.apache.lucene.search.CollectionTerminatedException;
+import org.apache.lucene.search.Collector;
+import org.apache.lucene.search.DocIdSetIterator;
+import org.apache.lucene.search.FilterCollector;
+import org.apache.lucene.search.FilterLeafCollector;
+import org.apache.lucene.search.FilterScorable;
+import org.apache.lucene.search.IndexSearcher;
+import org.apache.lucene.search.LeafCollector;
+import org.apache.lucene.search.MatchAllDocsQuery;
+import org.apache.lucene.search.Scorable;
+import org.apache.lucene.search.ScoreCachingWrappingScorer;
+import org.apache.lucene.search.ScoreMode;
+import org.apache.lucene.search.SimpleCollector;
+import org.apache.lucene.search.TermQuery;
+import org.apache.lucene.search.TopDocs;
+import org.apache.lucene.search.TopScoreDocCollector;
+import org.apache.lucene.search.Weight;
+import org.apache.lucene.search.similarities.BM25Similarity;
+import org.apache.lucene.store.Directory;
+import org.apache.lucene.tests.index.RandomIndexWriter;
+import org.apache.lucene.tests.search.DummyTotalHitCountCollector;
+import org.elasticsearch.core.IOUtils;
+import org.elasticsearch.test.ESTestCase;
+import org.hamcrest.Matchers;
+
+import java.io.IOException;
+import java.util.Arrays;
+
+public class QueryPhaseCollectorTests extends ESTestCase {
+
+    private Directory directory;
+    private IndexReader reader;
+    private IndexSearcher searcher;
+    private int numDocs;
+    private int numField2Docs;
+    private int numField3Docs;
+    private int numField2AndField3Docs;
+
+    @Override
+    public void setUp() throws Exception {
+        super.setUp();
+        directory = newDirectory();
+        RandomIndexWriter writer = new RandomIndexWriter(random(), directory, newIndexWriterConfig());
+        numDocs = randomIntBetween(900, 1000);
+        for (int i = 0; i < numDocs; i++) {
+            Document doc = new Document();
+            doc.add(new StringField("field1", "value", Field.Store.NO));
+            boolean field2 = randomBoolean();
+            if (field2) {
+                doc.add(new StringField("field2", "value", Field.Store.NO));
+                numField2Docs++;
+            }
+            boolean field3 = randomBoolean();
+            if (field3) {
+                doc.add(new StringField("field3", "value", Field.Store.NO));
+                numField3Docs++;
+            }
+            if (field2 && field3) {
+                numField2AndField3Docs++;
+            }
+            writer.addDocument(doc);
+        }
+        writer.flush();
+        reader = writer.getReader();
+        searcher = newSearcher(reader);
+        writer.close();
+    }
+
+    @Override
+    public void tearDown() throws Exception {
+        super.tearDown();
+        IOUtils.close(reader, directory);
+    }
+
+    public void testNullTopDocsCollector() {
+        expectThrows(NullPointerException.class, () -> new QueryPhaseCollector(null, null, 0, null, null));
+    }
+
+    public void testNegativeTerminateAfter() {
+        expectThrows(
+            IllegalArgumentException.class,
+            () -> new QueryPhaseCollector(new DummyTotalHitCountCollector(), null, randomIntBetween(Integer.MIN_VALUE, -1), null, null)
+        );
+    }
+
+    public void testTopDocsOnly() throws IOException {
+        {
+            TopScoreDocCollector topScoreDocCollector = TopScoreDocCollector.create(1, 1000);
+            QueryPhaseCollector queryPhaseCollector = new QueryPhaseCollector(topScoreDocCollector, null, 0, null, null);
+            searcher.search(new MatchAllDocsQuery(), queryPhaseCollector);
+            assertFalse(queryPhaseCollector.isTerminatedAfter());
+            assertEquals(numDocs, topScoreDocCollector.topDocs().totalHits.value);
+        }
+        {
+            TopScoreDocCollector topScoreDocCollector = TopScoreDocCollector.create(1, 1000);
+            QueryPhaseCollector queryPhaseCollector = new QueryPhaseCollector(topScoreDocCollector, null, 0, null, null);
+            searcher.search(new TermQuery(new Term("field2", "value")), queryPhaseCollector);
+            assertFalse(queryPhaseCollector.isTerminatedAfter());
+            assertEquals(numField2Docs, topScoreDocCollector.topDocs().totalHits.value);
+        }
+    }
+
+    public void testWithAggs() throws IOException {
+        {
+            TopScoreDocCollector topScoreDocCollector = TopScoreDocCollector.create(1, 1000);
+            DummyTotalHitCountCollector aggsCollector = new DummyTotalHitCountCollector();
+            QueryPhaseCollector queryPhaseCollector = new QueryPhaseCollector(topScoreDocCollector, null, 0, aggsCollector, null);
+            searcher.search(new MatchAllDocsQuery(), queryPhaseCollector);
+            assertFalse(queryPhaseCollector.isTerminatedAfter());
+            assertEquals(numDocs, topScoreDocCollector.topDocs().totalHits.value);
+            assertEquals(numDocs, aggsCollector.getTotalHits());
+        }
+        {
+            TopScoreDocCollector topScoreDocCollector = TopScoreDocCollector.create(1, 1000);
+            DummyTotalHitCountCollector aggsCollector = new DummyTotalHitCountCollector();
+            QueryPhaseCollector queryPhaseCollector = new QueryPhaseCollector(topScoreDocCollector, null, 0, aggsCollector, null);
+            searcher.search(new TermQuery(new Term("field2", "value")), queryPhaseCollector);
+            assertFalse(queryPhaseCollector.isTerminatedAfter());
+            assertEquals(numField2Docs, topScoreDocCollector.topDocs().totalHits.value);
+            assertEquals(numField2Docs, aggsCollector.getTotalHits());
+        }
+    }
+
+    public void testPostFilterTopDocsOnly() throws IOException {
+        {
+            TopScoreDocCollector topScoreDocCollector = TopScoreDocCollector.create(1, 1000);
+            TermQuery termQuery = new TermQuery(new Term("field2", "value"));
+            Weight filterWeight = termQuery.createWeight(searcher, ScoreMode.TOP_DOCS, 1f);
+            QueryPhaseCollector queryPhaseCollector = new QueryPhaseCollector(topScoreDocCollector, filterWeight, 0, null, null);
+            searcher.search(new MatchAllDocsQuery(), queryPhaseCollector);
+            assertFalse(queryPhaseCollector.isTerminatedAfter());
+            assertEquals(numField2Docs, topScoreDocCollector.topDocs().totalHits.value);
+        }
+        {
+            TopScoreDocCollector topScoreDocCollector = TopScoreDocCollector.create(1, 1000);
+            TermQuery termQuery = new TermQuery(new Term("field1", "value"));
+            Weight filterWeight = termQuery.createWeight(searcher, ScoreMode.TOP_DOCS, 1f);
+            QueryPhaseCollector queryPhaseCollector = new QueryPhaseCollector(topScoreDocCollector, filterWeight, 0, null, null);
+            searcher.search(new MatchAllDocsQuery(), queryPhaseCollector);
+            assertFalse(queryPhaseCollector.isTerminatedAfter());
+            assertEquals(numDocs, topScoreDocCollector.topDocs().totalHits.value);
+        }
+    }
+
+    public void testPostFilterWithAggs() throws IOException {
+        {
+            TopScoreDocCollector topScoreDocCollector = TopScoreDocCollector.create(1, 1000);
+            DummyTotalHitCountCollector aggsCollector = new DummyTotalHitCountCollector();
+            TermQuery termQuery = new TermQuery(new Term("field1", "value"));
+            Weight filterWeight = termQuery.createWeight(searcher, ScoreMode.TOP_DOCS, 1f);
+            QueryPhaseCollector queryPhaseCollector = new QueryPhaseCollector(topScoreDocCollector, filterWeight, 0, aggsCollector, null);
+            searcher.search(new MatchAllDocsQuery(), queryPhaseCollector);
+            assertFalse(queryPhaseCollector.isTerminatedAfter());
+            assertEquals(numDocs, topScoreDocCollector.topDocs().totalHits.value);
+            assertEquals(numDocs, aggsCollector.getTotalHits());
+        }
+        {
+            TopScoreDocCollector topScoreDocCollector = TopScoreDocCollector.create(1, 1000);
+            DummyTotalHitCountCollector aggsCollector = new DummyTotalHitCountCollector();
+            TermQuery termQuery = new TermQuery(new Term("field2", "value"));
+            Weight filterWeight = termQuery.createWeight(searcher, ScoreMode.TOP_DOCS, 1f);
+            QueryPhaseCollector queryPhaseCollector = new QueryPhaseCollector(topScoreDocCollector, filterWeight, 0, aggsCollector, null);
+            searcher.search(new MatchAllDocsQuery(), queryPhaseCollector);
+            assertFalse(queryPhaseCollector.isTerminatedAfter());
+            assertEquals(numField2Docs, topScoreDocCollector.topDocs().totalHits.value);
+            // post_filter is not applied to aggs
+            assertEquals(reader.maxDoc(), aggsCollector.getTotalHits());
+        }
+    }
+
+    public void testMinScoreTopDocsOnly() throws IOException {
+        searcher.setSimilarity(new BM25Similarity());
+        float maxScore;
+        float thresholdScore;
+        BooleanQuery booleanQuery = new BooleanQuery.Builder().add(new TermQuery(new Term("field1", "value")), BooleanClause.Occur.MUST)
+            .add(new BoostQuery(new TermQuery(new Term("field2", "value")), 200f), BooleanClause.Occur.SHOULD)
+            .build();
+        {
+            TopScoreDocCollector topScoreDocCollector = TopScoreDocCollector.create(numField2Docs + 1, 1000);
+            searcher.search(booleanQuery, topScoreDocCollector);
+            TopDocs topDocs = topScoreDocCollector.topDocs();
+            assertEquals(numDocs, topDocs.totalHits.value);
+            maxScore = topDocs.scoreDocs[0].score;
+            thresholdScore = topDocs.scoreDocs[numField2Docs].score;
+        }
+        {
+            TopScoreDocCollector topScoreDocCollector = TopScoreDocCollector.create(1, 1000);
+            QueryPhaseCollector queryPhaseCollector = new QueryPhaseCollector(topScoreDocCollector, null, 0, null, maxScore);
+            searcher.search(booleanQuery, queryPhaseCollector);
+            assertFalse(queryPhaseCollector.isTerminatedAfter());
+            assertEquals(numField2Docs, topScoreDocCollector.topDocs().totalHits.value);
+        }
+        {
+            TopScoreDocCollector topScoreDocCollector = TopScoreDocCollector.create(1, 1000);
+            QueryPhaseCollector queryPhaseCollector = new QueryPhaseCollector(topScoreDocCollector, null, 0, null, thresholdScore);
+            searcher.search(booleanQuery, queryPhaseCollector);
+            assertFalse(queryPhaseCollector.isTerminatedAfter());
+            assertEquals(numDocs, topScoreDocCollector.topDocs().totalHits.value);
+        }
+        {
+            TopScoreDocCollector topScoreDocCollector = TopScoreDocCollector.create(1, 1000);
+            QueryPhaseCollector queryPhaseCollector = new QueryPhaseCollector(topScoreDocCollector, null, 0, null, maxScore + 100f);
+            searcher.search(booleanQuery, queryPhaseCollector);
+            assertFalse(queryPhaseCollector.isTerminatedAfter());
+            assertEquals(0, topScoreDocCollector.topDocs().totalHits.value);
+        }
+    }
+
+    public void testMinScoreWithAggs() throws IOException {
+        searcher.setSimilarity(new BM25Similarity());
+        float maxScore;
+        float thresholdScore;
+        BooleanQuery booleanQuery = new BooleanQuery.Builder().add(new TermQuery(new Term("field1", "value")), BooleanClause.Occur.MUST)
+            .add(new BoostQuery(new TermQuery(new Term("field2", "value")), 200f), BooleanClause.Occur.SHOULD)
+            .build();
+        {
+            TopScoreDocCollector topScoreDocCollector = TopScoreDocCollector.create(numField2Docs + 1, 1000);
+            searcher.search(booleanQuery, topScoreDocCollector);
+            TopDocs topDocs = topScoreDocCollector.topDocs();
+            assertEquals(numDocs, topDocs.totalHits.value);
+            maxScore = topDocs.scoreDocs[0].score;
+            thresholdScore = topDocs.scoreDocs[numField2Docs].score;
+        }
+        {
+            TopScoreDocCollector topScoreDocCollector = TopScoreDocCollector.create(1, 1000);
+            DummyTotalHitCountCollector aggsCollector = new DummyTotalHitCountCollector();
+            QueryPhaseCollector queryPhaseCollector = new QueryPhaseCollector(topScoreDocCollector, null, 0, aggsCollector, maxScore);
+            searcher.search(booleanQuery, queryPhaseCollector);
+            assertFalse(queryPhaseCollector.isTerminatedAfter());
+            assertEquals(numField2Docs, topScoreDocCollector.topDocs().totalHits.value);
+            // min_score is applied to aggs as well as top docs
+            assertEquals(numField2Docs, aggsCollector.getTotalHits());
+        }
+        {
+            TopScoreDocCollector topScoreDocCollector = TopScoreDocCollector.create(1, 1000);
+            DummyTotalHitCountCollector aggsCollector = new DummyTotalHitCountCollector();
+            QueryPhaseCollector queryPhaseCollector = new QueryPhaseCollector(topScoreDocCollector, null, 0, aggsCollector, thresholdScore);
+            searcher.search(booleanQuery, queryPhaseCollector);
+            assertFalse(queryPhaseCollector.isTerminatedAfter());
+            assertEquals(numDocs, topScoreDocCollector.topDocs().totalHits.value);
+            assertEquals(numDocs, aggsCollector.getTotalHits());
+        }
+        {
+            TopScoreDocCollector topScoreDocCollector = TopScoreDocCollector.create(1, 1000);
+            DummyTotalHitCountCollector aggsCollector = new DummyTotalHitCountCollector();
+            QueryPhaseCollector queryPhaseCollector = new QueryPhaseCollector(
+                topScoreDocCollector,
+                null,
+                0,
+                aggsCollector,
+                maxScore + 100f
+            );
+            searcher.search(booleanQuery, queryPhaseCollector);
+            assertFalse(queryPhaseCollector.isTerminatedAfter());
+            assertEquals(0, topScoreDocCollector.topDocs().totalHits.value);
+            assertEquals(0, aggsCollector.getTotalHits());
+        }
+    }
+
+    public void testPostFilterAndMinScoreTopDocsOnly() throws IOException {
+        searcher.setSimilarity(new BM25Similarity());
+        float maxScore;
+        float thresholdScore;
+        BooleanQuery booleanQuery = new BooleanQuery.Builder().add(new TermQuery(new Term("field1", "value")), BooleanClause.Occur.MUST)
+            .add(new BoostQuery(new TermQuery(new Term("field3", "value")), 200f), BooleanClause.Occur.SHOULD)
+            .build();
+        TermQuery termQuery = new TermQuery(new Term("field2", "value"));
+        Weight filterWeight = termQuery.createWeight(searcher, ScoreMode.TOP_DOCS, 1f);
+        {
+            TopScoreDocCollector topScoreDocCollector = TopScoreDocCollector.create(numField3Docs + 1, 1000);
+            searcher.search(booleanQuery, topScoreDocCollector);
+            TopDocs topDocs = topScoreDocCollector.topDocs();
+            assertEquals(numDocs, topDocs.totalHits.value);
+            maxScore = topDocs.scoreDocs[0].score;
+            thresholdScore = topDocs.scoreDocs[numField3Docs].score;
+        }
+        {
+            TopScoreDocCollector topScoreDocCollector = TopScoreDocCollector.create(1, 1000);
+            QueryPhaseCollector queryPhaseCollector = new QueryPhaseCollector(topScoreDocCollector, filterWeight, 0, null, maxScore);
+            searcher.search(booleanQuery, queryPhaseCollector);
+            assertFalse(queryPhaseCollector.isTerminatedAfter());
+            assertEquals(numField2AndField3Docs, topScoreDocCollector.topDocs().totalHits.value);
+        }
+        {
+            TopScoreDocCollector topScoreDocCollector = TopScoreDocCollector.create(1, 1000);
+            QueryPhaseCollector queryPhaseCollector = new QueryPhaseCollector(topScoreDocCollector, filterWeight, 0, null, thresholdScore);
+            searcher.search(booleanQuery, queryPhaseCollector);
+            assertFalse(queryPhaseCollector.isTerminatedAfter());
+            assertEquals(numField2Docs, topScoreDocCollector.topDocs().totalHits.value);
+        }
+        {
+            TopScoreDocCollector topScoreDocCollector = TopScoreDocCollector.create(1, 1000);
+            QueryPhaseCollector queryPhaseCollector = new QueryPhaseCollector(topScoreDocCollector, filterWeight, 0, null, maxScore + 100f);
+            searcher.search(booleanQuery, queryPhaseCollector);
+            assertFalse(queryPhaseCollector.isTerminatedAfter());
+            assertEquals(0, topScoreDocCollector.topDocs().totalHits.value);
+        }
+    }
+
+    public void testPostFilterAndMinScoreWithAggs() throws IOException {
+        searcher.setSimilarity(new BM25Similarity());
+        float maxScore;
+        float thresholdScore;
+        BooleanQuery booleanQuery = new BooleanQuery.Builder().add(new TermQuery(new Term("field1", "value")), BooleanClause.Occur.MUST)
+            .add(new BoostQuery(new TermQuery(new Term("field3", "value")), 200f), BooleanClause.Occur.SHOULD)
+            .build();
+        TermQuery termQuery = new TermQuery(new Term("field2", "value"));
+        Weight filterWeight = termQuery.createWeight(searcher, ScoreMode.TOP_DOCS, 1f);
+        {
+            TopScoreDocCollector topScoreDocCollector = TopScoreDocCollector.create(numField3Docs + 1, 1000);
+            searcher.search(booleanQuery, topScoreDocCollector);
+            TopDocs topDocs = topScoreDocCollector.topDocs();
+            assertEquals(numDocs, topDocs.totalHits.value);
+            maxScore = topDocs.scoreDocs[0].score;
+            thresholdScore = topDocs.scoreDocs[numField3Docs].score;
+        }
+        {
+            TopScoreDocCollector topScoreDocCollector = TopScoreDocCollector.create(1, 1000);
+            DummyTotalHitCountCollector aggs = new DummyTotalHitCountCollector();
+            QueryPhaseCollector queryPhaseCollector = new QueryPhaseCollector(topScoreDocCollector, filterWeight, 0, aggs, maxScore);
+            searcher.search(booleanQuery, queryPhaseCollector);
+            assertFalse(queryPhaseCollector.isTerminatedAfter());
+            assertEquals(numField2AndField3Docs, topScoreDocCollector.topDocs().totalHits.value);
+            assertEquals(numField3Docs, aggs.getTotalHits());
+        }
+        {
+            TopScoreDocCollector topScoreDocCollector = TopScoreDocCollector.create(1, 1000);
+            DummyTotalHitCountCollector aggsCollector = new DummyTotalHitCountCollector();
+            QueryPhaseCollector queryPhaseCollector = new QueryPhaseCollector(
+                topScoreDocCollector,
+                filterWeight,
+                0,
+                aggsCollector,
+                thresholdScore
+            );
+            searcher.search(booleanQuery, queryPhaseCollector);
+            assertFalse(queryPhaseCollector.isTerminatedAfter());
+            assertEquals(numField2Docs, topScoreDocCollector.topDocs().totalHits.value);
+            assertEquals(numDocs, aggsCollector.getTotalHits());
+        }
+        {
+            TopScoreDocCollector topScoreDocCollector = TopScoreDocCollector.create(1, 1000);
+            DummyTotalHitCountCollector aggsCollector = new DummyTotalHitCountCollector();
+            QueryPhaseCollector queryPhaseCollector = new QueryPhaseCollector(
+                topScoreDocCollector,
+                filterWeight,
+                0,
+                aggsCollector,
+                maxScore + 100f
+            );
+            searcher.search(booleanQuery, queryPhaseCollector);
+            assertFalse(queryPhaseCollector.isTerminatedAfter());
+            assertEquals(0, topScoreDocCollector.topDocs().totalHits.value);
+            assertEquals(0, aggsCollector.getTotalHits());
+        }
+    }
+
+    public void testTerminateAfterTopDocsOnly() throws IOException {
+        {
+            int terminateAfter = randomIntBetween(1, numDocs - 1);
+            DummyTotalHitCountCollector topDocs = new DummyTotalHitCountCollector();
+            QueryPhaseCollector queryPhaseCollector = new QueryPhaseCollector(topDocs, null, terminateAfter, null, null);
+            searcher.search(new MatchAllDocsQuery(), queryPhaseCollector);
+            assertTrue(queryPhaseCollector.isTerminatedAfter());
+            assertEquals(terminateAfter, topDocs.getTotalHits());
+        }
+        {
+            DummyTotalHitCountCollector topDocs = new DummyTotalHitCountCollector();
+            QueryPhaseCollector queryPhaseCollector = new QueryPhaseCollector(topDocs, null, numDocs, null, null);
+            searcher.search(new MatchAllDocsQuery(), queryPhaseCollector);
+            assertFalse(queryPhaseCollector.isTerminatedAfter());
+            assertEquals(numDocs, topDocs.getTotalHits());
+        }
+    }
+
+    public void testTerminateAfterWithAggs() throws IOException {
+        {
+            int terminateAfter = randomIntBetween(1, numDocs - 1);
+            DummyTotalHitCountCollector topDocs = new DummyTotalHitCountCollector();
+            DummyTotalHitCountCollector aggs = new DummyTotalHitCountCollector();
+            QueryPhaseCollector queryPhaseCollector = new QueryPhaseCollector(topDocs, null, terminateAfter, aggs, null);
+            searcher.search(new MatchAllDocsQuery(), queryPhaseCollector);
+            assertTrue(queryPhaseCollector.isTerminatedAfter());
+            assertEquals(terminateAfter, topDocs.getTotalHits());
+            assertEquals(terminateAfter, aggs.getTotalHits());
+        }
+        {
+            DummyTotalHitCountCollector topDocs = new DummyTotalHitCountCollector();
+            DummyTotalHitCountCollector aggs = new DummyTotalHitCountCollector();
+            QueryPhaseCollector queryPhaseCollector = new QueryPhaseCollector(topDocs, null, numDocs, aggs, null);
+            searcher.search(new MatchAllDocsQuery(), queryPhaseCollector);
+            assertFalse(queryPhaseCollector.isTerminatedAfter());
+            assertEquals(numDocs, topDocs.getTotalHits());
+            assertEquals(numDocs, aggs.getTotalHits());
+        }
+    }
+
+    public void testTerminateAfterTopDocsOnlyWithPostFilter() throws IOException {
+        TermQuery termQuery = new TermQuery(new Term("field2", "value"));
+        Weight filterWeight = termQuery.createWeight(searcher, ScoreMode.TOP_DOCS, 1f);
+        {
+            int terminateAfter = randomIntBetween(1, numField2Docs - 1);
+            DummyTotalHitCountCollector topDocs = new DummyTotalHitCountCollector();
+            QueryPhaseCollector queryPhaseCollector = new QueryPhaseCollector(topDocs, filterWeight, terminateAfter, null, null);
+            searcher.search(new MatchAllDocsQuery(), queryPhaseCollector);
+            assertTrue(queryPhaseCollector.isTerminatedAfter());
+            assertEquals(terminateAfter, topDocs.getTotalHits());
+        }
+        {
+            int terminateAfter = randomIntBetween(numField2Docs, Integer.MAX_VALUE);
+            DummyTotalHitCountCollector topDocs = new DummyTotalHitCountCollector();
+            QueryPhaseCollector queryPhaseCollector = new QueryPhaseCollector(topDocs, filterWeight, terminateAfter, null, null);
+            searcher.search(new MatchAllDocsQuery(), queryPhaseCollector);
+            assertFalse(queryPhaseCollector.isTerminatedAfter());
+            assertEquals(numField2Docs, topDocs.getTotalHits());
+        }
+    }
+
+    public void testTerminateAfterWithAggsAndPostFilter() throws IOException {
+        TermQuery termQuery = new TermQuery(new Term("field2", "value"));
+        Weight filterWeight = termQuery.createWeight(searcher, ScoreMode.TOP_DOCS, 1f);
+        {
+            int terminateAfter = randomIntBetween(1, numField2Docs - 1);
+            DummyTotalHitCountCollector topDocs = new DummyTotalHitCountCollector();
+            DummyTotalHitCountCollector aggs = new DummyTotalHitCountCollector();
+            QueryPhaseCollector queryPhaseCollector = new QueryPhaseCollector(topDocs, filterWeight, terminateAfter, aggs, null);
+            searcher.search(new MatchAllDocsQuery(), queryPhaseCollector);
+            assertTrue(queryPhaseCollector.isTerminatedAfter());
+            assertEquals(terminateAfter, topDocs.getTotalHits());
+            // aggs see more docs because they are not filtered
+            assertThat(aggs.getTotalHits(), Matchers.greaterThanOrEqualTo(terminateAfter));
+        }
+        {
+            int terminateAfter = randomIntBetween(numField2Docs, Integer.MAX_VALUE);
+            DummyTotalHitCountCollector topDocs = new DummyTotalHitCountCollector();
+            DummyTotalHitCountCollector aggs = new DummyTotalHitCountCollector();
+            QueryPhaseCollector queryPhaseCollector = new QueryPhaseCollector(topDocs, filterWeight, terminateAfter, aggs, null);
+            searcher.search(new MatchAllDocsQuery(), queryPhaseCollector);
+            assertFalse(queryPhaseCollector.isTerminatedAfter());
+            assertEquals(numField2Docs, topDocs.getTotalHits());
+            // aggs see more docs because they are not filtered
+            assertThat(aggs.getTotalHits(), Matchers.greaterThanOrEqualTo(numField2Docs));
+        }
+    }
+
+    public void testTerminateAfterTopDocsOnlyWithMinScore() throws IOException {
+        searcher.setSimilarity(new BM25Similarity());
+        float maxScore;
+        BooleanQuery booleanQuery = new BooleanQuery.Builder().add(new TermQuery(new Term("field1", "value")), BooleanClause.Occur.MUST)
+            .add(new BoostQuery(new TermQuery(new Term("field2", "value")), 200f), BooleanClause.Occur.SHOULD)
+            .build();
+        {
+            TopScoreDocCollector topScoreDocCollector = TopScoreDocCollector.create(numField2Docs + 1, 1000);
+            searcher.search(booleanQuery, topScoreDocCollector);
+            TopDocs topDocs = topScoreDocCollector.topDocs();
+            assertEquals(numDocs, topDocs.totalHits.value);
+            maxScore = topDocs.scoreDocs[0].score;
+        }
+        {
+            int terminateAfter = randomIntBetween(1, numField2Docs - 1);
+            TopScoreDocCollector topScoreDocCollector = TopScoreDocCollector.create(1, 1000);
+            QueryPhaseCollector queryPhaseCollector = new QueryPhaseCollector(topScoreDocCollector, null, terminateAfter, null, maxScore);
+            searcher.search(booleanQuery, queryPhaseCollector);
+            assertTrue(queryPhaseCollector.isTerminatedAfter());
+            assertEquals(terminateAfter, topScoreDocCollector.topDocs().totalHits.value);
+        }
+    }
+
+    public void testTerminateAfterWithAggsAndMinScore() throws IOException {
+        searcher.setSimilarity(new BM25Similarity());
+        float maxScore;
+        BooleanQuery booleanQuery = new BooleanQuery.Builder().add(new TermQuery(new Term("field1", "value")), BooleanClause.Occur.MUST)
+            .add(new BoostQuery(new TermQuery(new Term("field2", "value")), 200f), BooleanClause.Occur.SHOULD)
+            .build();
+        {
+            TopScoreDocCollector topScoreDocCollector = TopScoreDocCollector.create(numField2Docs + 1, 1000);
+            searcher.search(booleanQuery, topScoreDocCollector);
+            TopDocs topDocs = topScoreDocCollector.topDocs();
+            assertEquals(numDocs, topDocs.totalHits.value);
+            maxScore = topDocs.scoreDocs[0].score;
+        }
+        {
+            int terminateAfter = randomIntBetween(1, numField2Docs - 1);
+            TopScoreDocCollector topScoreDocCollector = TopScoreDocCollector.create(1, 1000);
+            DummyTotalHitCountCollector aggs = new DummyTotalHitCountCollector();
+            QueryPhaseCollector queryPhaseCollector = new QueryPhaseCollector(topScoreDocCollector, null, terminateAfter, aggs, maxScore);
+            searcher.search(booleanQuery, queryPhaseCollector);
+            assertTrue(queryPhaseCollector.isTerminatedAfter());
+            assertEquals(terminateAfter, topScoreDocCollector.topDocs().totalHits.value);
+            assertEquals(terminateAfter, aggs.getTotalHits());
+        }
+    }
+
+    public void testTerminateAfterAndPostFilterAndMinScoreTopDocsOnly() throws IOException {
+        searcher.setSimilarity(new BM25Similarity());
+        float maxScore;
+        BooleanQuery booleanQuery = new BooleanQuery.Builder().add(new TermQuery(new Term("field1", "value")), BooleanClause.Occur.MUST)
+            .add(new BoostQuery(new TermQuery(new Term("field3", "value")), 200f), BooleanClause.Occur.SHOULD)
+            .build();
+        TermQuery termQuery = new TermQuery(new Term("field2", "value"));
+        Weight filterWeight = termQuery.createWeight(searcher, ScoreMode.TOP_DOCS, 1f);
+        {
+            TopScoreDocCollector topScoreDocCollector = TopScoreDocCollector.create(numField3Docs + 1, 1000);
+            searcher.search(booleanQuery, topScoreDocCollector);
+            TopDocs topDocs = topScoreDocCollector.topDocs();
+            assertEquals(numDocs, topDocs.totalHits.value);
+            maxScore = topDocs.scoreDocs[0].score;
+        }
+        {
+            int terminateAfter = randomIntBetween(1, numField2AndField3Docs - 1);
+            TopScoreDocCollector topScoreDocCollector = TopScoreDocCollector.create(1, 1000);
+            QueryPhaseCollector queryPhaseCollector = new QueryPhaseCollector(
+                topScoreDocCollector,
+                filterWeight,
+                terminateAfter,
+                null,
+                maxScore
+            );
+            searcher.search(booleanQuery, queryPhaseCollector);
+            assertTrue(queryPhaseCollector.isTerminatedAfter());
+            assertEquals(terminateAfter, topScoreDocCollector.topDocs().totalHits.value);
+        }
+    }
+
+    public void testTerminateAfterAndPostFilterAndMinScoreWithAggs() throws IOException {
+        searcher.setSimilarity(new BM25Similarity());
+        float maxScore;
+        BooleanQuery booleanQuery = new BooleanQuery.Builder().add(new TermQuery(new Term("field1", "value")), BooleanClause.Occur.MUST)
+            .add(new BoostQuery(new TermQuery(new Term("field3", "value")), 200f), BooleanClause.Occur.SHOULD)
+            .build();
+        TermQuery termQuery = new TermQuery(new Term("field2", "value"));
+        Weight filterWeight = termQuery.createWeight(searcher, ScoreMode.TOP_DOCS, 1f);
+        {
+            TopScoreDocCollector topScoreDocCollector = TopScoreDocCollector.create(numField3Docs + 1, 1000);
+            searcher.search(booleanQuery, topScoreDocCollector);
+            TopDocs topDocs = topScoreDocCollector.topDocs();
+            assertEquals(numDocs, topDocs.totalHits.value);
+            maxScore = topDocs.scoreDocs[0].score;
+        }
+        {
+            int terminateAfter = randomIntBetween(1, numField2AndField3Docs - 1);
+            TopScoreDocCollector topScoreDocCollector = TopScoreDocCollector.create(1, 1000);
+            DummyTotalHitCountCollector aggs = new DummyTotalHitCountCollector();
+            QueryPhaseCollector queryPhaseCollector = new QueryPhaseCollector(
+                topScoreDocCollector,
+                filterWeight,
+                terminateAfter,
+                aggs,
+                maxScore
+            );
+            searcher.search(booleanQuery, queryPhaseCollector);
+            assertTrue(queryPhaseCollector.isTerminatedAfter());
+            assertEquals(terminateAfter, topScoreDocCollector.topDocs().totalHits.value);
+            // aggs see more documents because the filter is not applied to them
+            assertThat(aggs.getTotalHits(), Matchers.greaterThanOrEqualTo(terminateAfter));
+        }
+    }
+
+    public void testScoreModeTopDocsOnly() throws IOException {
+        ScoreMode scoreMode = randomFrom(ScoreMode.values());
+        Weight weight = randomBoolean() ? searcher.createWeight(new MatchAllDocsQuery(), ScoreMode.COMPLETE, 1.0f) : null;
+        int terminateAfter = randomBoolean() ? 0 : randomIntBetween(1, Integer.MAX_VALUE);
+        QueryPhaseCollector qpc = new QueryPhaseCollector(new MockCollector(scoreMode), weight, terminateAfter, null, null);
+        assertEquals(scoreMode, qpc.scoreMode());
+    }
+
+    public void testScoreModeTopDocsOnlyWithMinScore() throws IOException {
+        Weight weight = randomBoolean() ? searcher.createWeight(new MatchAllDocsQuery(), ScoreMode.COMPLETE, 1.0f) : null;
+        int terminateAfter = randomBoolean() ? 0 : randomIntBetween(1, Integer.MAX_VALUE);
+        {
+            QueryPhaseCollector qpc = new QueryPhaseCollector(new MockCollector(ScoreMode.TOP_SCORES), weight, terminateAfter, null, 100f);
+            assertEquals(ScoreMode.TOP_SCORES, qpc.scoreMode());
+        }
+        {
+            ScoreMode scoreMode = randomScoreModeExceptTopScores();
+            QueryPhaseCollector qpc = new QueryPhaseCollector(new MockCollector(scoreMode), weight, terminateAfter, null, 100f);
+            assertEquals(ScoreMode.COMPLETE, qpc.scoreMode());
+        }
+    }
+
+    public void testScoreModeWithAggsSameScoreMode() throws IOException {
+        Weight weight = randomBoolean() ? searcher.createWeight(new MatchAllDocsQuery(), ScoreMode.COMPLETE, 1.0f) : null;
+        int terminateAfter = randomBoolean() ? 0 : randomIntBetween(1, Integer.MAX_VALUE);
+        ScoreMode scoreMode = randomScoreModeExceptTopScores();
+        QueryPhaseCollector qpc = new QueryPhaseCollector(
+            new MockCollector(scoreMode),
+            weight,
+            terminateAfter,
+            new MockCollector(scoreMode),
+            null
+        );
+        assertEquals(scoreMode, qpc.scoreMode());
+    }
+
+    public void testScoreModeWithAggsSameScoreModeWithMinScore() throws IOException {
+        Weight weight = randomBoolean() ? searcher.createWeight(new MatchAllDocsQuery(), ScoreMode.COMPLETE, 1.0f) : null;
+        int terminateAfter = randomBoolean() ? 0 : randomIntBetween(1, Integer.MAX_VALUE);
+        ScoreMode scoreMode = randomScoreModeExceptTopScores();
+        QueryPhaseCollector qpc = new QueryPhaseCollector(
+            new MockCollector(scoreMode),
+            weight,
+            terminateAfter,
+            new MockCollector(scoreMode),
+            100f
+        );
+        assertEquals(ScoreMode.COMPLETE, qpc.scoreMode());
+    }
+
+    public void testScoreModeWithAggsExhaustive() throws IOException {
+        Weight weight = randomBoolean() ? searcher.createWeight(new MatchAllDocsQuery(), ScoreMode.COMPLETE, 1.0f) : null;
+        int terminateAfter = randomBoolean() ? 0 : randomIntBetween(1, Integer.MAX_VALUE);
+        Float minScore = randomBoolean() ? 100f : null;
+        Collector complete = new MockCollector(ScoreMode.COMPLETE);
+        Collector completeNoScores = new MockCollector(ScoreMode.COMPLETE_NO_SCORES);
+        {
+            QueryPhaseCollector qpc = new QueryPhaseCollector(complete, weight, terminateAfter, completeNoScores, minScore);
+            assertEquals(ScoreMode.COMPLETE, qpc.scoreMode());
+        }
+        {
+            QueryPhaseCollector qpc = new QueryPhaseCollector(completeNoScores, weight, terminateAfter, complete, minScore);
+            assertEquals(ScoreMode.COMPLETE, qpc.scoreMode());
+        }
+        {
+            QueryPhaseCollector qpc = new QueryPhaseCollector(
+                complete,
+                weight,
+                terminateAfter,
+                new MockCollector(ScoreMode.TOP_DOCS),
+                minScore
+            );
+            assertEquals(ScoreMode.COMPLETE, qpc.scoreMode());
+        }
+        {
+            QueryPhaseCollector qpc = new QueryPhaseCollector(
+                complete,
+                weight,
+                terminateAfter,
+                new MockCollector(ScoreMode.TOP_DOCS_WITH_SCORES),
+                minScore
+            );
+            assertEquals(ScoreMode.COMPLETE, qpc.scoreMode());
+        }
+        {
+            QueryPhaseCollector qpc = new QueryPhaseCollector(
+                completeNoScores,
+                weight,
+                terminateAfter,
+                new MockCollector(ScoreMode.TOP_DOCS_WITH_SCORES),
+                minScore
+            );
+            assertEquals(ScoreMode.COMPLETE, qpc.scoreMode());
+        }
+        {
+            QueryPhaseCollector qpc = new QueryPhaseCollector(
+                completeNoScores,
+                weight,
+                terminateAfter,
+                new MockCollector(ScoreMode.TOP_DOCS),
+                null
+            );
+            assertEquals(ScoreMode.COMPLETE_NO_SCORES, qpc.scoreMode());
+        }
+        {
+            QueryPhaseCollector qpc = new QueryPhaseCollector(
+                completeNoScores,
+                weight,
+                terminateAfter,
+                new MockCollector(ScoreMode.TOP_DOCS),
+                100f
+            );
+            assertEquals(ScoreMode.COMPLETE, qpc.scoreMode());
+        }
+    }
+
+    public void testScoreModeWithAggsTopScores() throws IOException {
+        Weight weight = randomBoolean() ? searcher.createWeight(new MatchAllDocsQuery(), ScoreMode.COMPLETE, 1.0f) : null;
+        int terminateAfter = randomBoolean() ? 0 : randomIntBetween(1, Integer.MAX_VALUE);
+        Float minScore = randomBoolean() ? 100f : null;
+        Collector topScores = new MockCollector(ScoreMode.TOP_SCORES);
+        {
+            QueryPhaseCollector qpc = new QueryPhaseCollector(
+                topScores,
+                weight,
+                terminateAfter,
+                new MockCollector(ScoreMode.COMPLETE),
+                minScore
+            );
+            assertEquals(ScoreMode.COMPLETE, qpc.scoreMode());
+        }
+        {
+            QueryPhaseCollector qpc = new QueryPhaseCollector(
+                topScores,
+                weight,
+                terminateAfter,
+                new MockCollector(ScoreMode.COMPLETE_NO_SCORES),
+                minScore
+            );
+            assertEquals(ScoreMode.COMPLETE, qpc.scoreMode());
+        }
+        {
+            QueryPhaseCollector qpc = new QueryPhaseCollector(
+                topScores,
+                weight,
+                terminateAfter,
+                new MockCollector(ScoreMode.TOP_DOCS),
+                minScore
+            );
+            assertEquals(ScoreMode.COMPLETE, qpc.scoreMode());
+        }
+        {
+            QueryPhaseCollector qpc = new QueryPhaseCollector(
+                topScores,
+                weight,
+                terminateAfter,
+                new MockCollector(ScoreMode.TOP_DOCS_WITH_SCORES),
+                minScore
+            );
+            assertEquals(ScoreMode.COMPLETE, qpc.scoreMode());
+        }
+    }
+
+    public void testScoreModeWithAggsTopDocs() throws IOException {
+        Weight weight = randomBoolean() ? searcher.createWeight(new MatchAllDocsQuery(), ScoreMode.COMPLETE, 1.0f) : null;
+        int terminateAfter = randomBoolean() ? 0 : randomIntBetween(1, Integer.MAX_VALUE);
+        Float minScore = randomBoolean() ? 100f : null;
+        Collector topDocs = new MockCollector(ScoreMode.TOP_DOCS);
+        {
+            QueryPhaseCollector qpc = new QueryPhaseCollector(
+                topDocs,
+                weight,
+                terminateAfter,
+                new MockCollector(ScoreMode.COMPLETE),
+                minScore
+            );
+            assertEquals(ScoreMode.COMPLETE, qpc.scoreMode());
+        }
+        {
+            QueryPhaseCollector qpc = new QueryPhaseCollector(
+                topDocs,
+                weight,
+                terminateAfter,
+                new MockCollector(ScoreMode.TOP_DOCS_WITH_SCORES),
+                minScore
+            );
+            assertEquals(ScoreMode.COMPLETE, qpc.scoreMode());
+        }
+        {
+            QueryPhaseCollector qpc = new QueryPhaseCollector(
+                topDocs,
+                weight,
+                terminateAfter,
+                new MockCollector(ScoreMode.COMPLETE_NO_SCORES),
+                null
+            );
+            assertEquals(ScoreMode.COMPLETE_NO_SCORES, qpc.scoreMode());
+        }
+        {
+            QueryPhaseCollector qpc = new QueryPhaseCollector(
+                topDocs,
+                weight,
+                terminateAfter,
+                new MockCollector(ScoreMode.COMPLETE_NO_SCORES),
+                100f
+            );
+            assertEquals(ScoreMode.COMPLETE, qpc.scoreMode());
+        }
+    }
+
+    public void testScoreModeWithAggsTopDocsWithScores() throws IOException {
+        Weight weight = randomBoolean() ? searcher.createWeight(new MatchAllDocsQuery(), ScoreMode.COMPLETE, 1.0f) : null;
+        int terminateAfter = randomBoolean() ? 0 : randomIntBetween(1, Integer.MAX_VALUE);
+        Float minScore = randomBoolean() ? 100f : null;
+        Collector topDocsWithScores = new MockCollector(ScoreMode.TOP_DOCS_WITH_SCORES);
+        {
+            QueryPhaseCollector qpc = new QueryPhaseCollector(
+                topDocsWithScores,
+                weight,
+                terminateAfter,
+                new MockCollector(ScoreMode.COMPLETE),
+                minScore
+            );
+            assertEquals(ScoreMode.COMPLETE, qpc.scoreMode());
+        }
+        {
+            QueryPhaseCollector qpc = new QueryPhaseCollector(
+                topDocsWithScores,
+                weight,
+                terminateAfter,
+                new MockCollector(ScoreMode.COMPLETE_NO_SCORES),
+                minScore
+            );
+            assertEquals(ScoreMode.COMPLETE, qpc.scoreMode());
+        }
+        {
+            QueryPhaseCollector qpc = new QueryPhaseCollector(
+                topDocsWithScores,
+                weight,
+                terminateAfter,
+                new MockCollector(ScoreMode.TOP_DOCS),
+                minScore
+            );
+            assertEquals(ScoreMode.COMPLETE, qpc.scoreMode());
+        }
+    }
+
+    public void testWeightIsPropagatedTopDocsOnly() throws IOException {
+        MockCollector topDocsCollector = new MockCollector(randomFrom(ScoreMode.values()));
+        QueryPhaseCollector queryPhaseCollector = new QueryPhaseCollector(topDocsCollector, null, 0, null, null);
+        searcher.search(new MatchAllDocsQuery(), queryPhaseCollector);
+        assertTrue(topDocsCollector.setWeightCalled);
+    }
+
+    public void testWeightIsPropagatedWithAggs() throws IOException {
+        MockCollector topDocsCollector = new MockCollector(randomFrom(ScoreMode.values()));
+        MockCollector aggsCollector = new MockCollector(randomScoreModeExceptTopScores());
+        QueryPhaseCollector queryPhaseCollector = new QueryPhaseCollector(topDocsCollector, null, 0, aggsCollector, null);
+        searcher.search(new MatchAllDocsQuery(), queryPhaseCollector);
+        assertTrue(topDocsCollector.setWeightCalled);
+        assertTrue(aggsCollector.setWeightCalled);
+    }
+
+    public void testWeightPropagationWithPostFilterTopDocsOnly() throws IOException {
+        // the weight is not propagated because docs collection is filtered
+        MockCollector mockCollector = new MockCollector(randomFrom(ScoreMode.values()));
+        TermQuery termQuery = new TermQuery(new Term("field2", "value"));
+        Weight filterWeight = termQuery.createWeight(searcher, ScoreMode.TOP_DOCS, 1f);
+        QueryPhaseCollector queryPhaseCollector = new QueryPhaseCollector(mockCollector, filterWeight, 0, null, null);
+        searcher.search(new MatchAllDocsQuery(), queryPhaseCollector);
+        assertFalse(mockCollector.setWeightCalled);
+    }
+
+    public void testWeightPropagationWithPostFilterWithAggs() throws IOException {
+        // the weight is propagated only to the aggs collector because docs collection is filtered
+        MockCollector topDocsCollector = new MockCollector(randomFrom(ScoreMode.values()));
+        MockCollector aggsCollector = new MockCollector(randomScoreModeExceptTopScores());
+        TermQuery termQuery = new TermQuery(new Term("field2", "value"));
+        Weight filterWeight = termQuery.createWeight(searcher, ScoreMode.TOP_DOCS, 1f);
+        QueryPhaseCollector queryPhaseCollector = new QueryPhaseCollector(topDocsCollector, filterWeight, 0, aggsCollector, null);
+        searcher.search(new MatchAllDocsQuery(), queryPhaseCollector);
+        assertFalse(topDocsCollector.setWeightCalled);
+        assertTrue(aggsCollector.setWeightCalled);
+    }
+
+    public void testWeightPropagationWithMinScoreTopDocsOnly() throws IOException {
+        // the weight is not propagated to the top docs collector
+        MockCollector topDocsCollector = new MockCollector(randomFrom(ScoreMode.values()));
+        QueryPhaseCollector queryPhaseCollector = new QueryPhaseCollector(topDocsCollector, null, 0, null, 100f);
+        searcher.search(new MatchAllDocsQuery(), queryPhaseCollector);
+        assertFalse(queryPhaseCollector.isTerminatedAfter());
+        assertFalse(topDocsCollector.setWeightCalled);
+    }
+
+    public void testWeightPropagationWithMinScoreWithAggs() throws IOException {
+        // the weight is not propagated to either of the collectors
+        MockCollector topDocsCollector = new MockCollector(randomFrom(ScoreMode.values()));
+        MockCollector aggsCollector = new MockCollector(randomScoreModeExceptTopScores());
+        QueryPhaseCollector queryPhaseCollector = new QueryPhaseCollector(topDocsCollector, null, 0, aggsCollector, 100f);
+        searcher.search(new MatchAllDocsQuery(), queryPhaseCollector);
+        assertFalse(queryPhaseCollector.isTerminatedAfter());
+        assertFalse(topDocsCollector.setWeightCalled);
+        assertFalse(aggsCollector.setWeightCalled);
+    }
+
+    public void testCollectionTerminatedExceptionHandling() throws IOException {
+        final int terminateAfter1 = random().nextInt(numDocs + 10);
+        final int expectedCount1 = Math.min(terminateAfter1, numDocs);
+        DummyTotalHitCountCollector collector1 = new DummyTotalHitCountCollector();
+
+        final int terminateAfter2 = random().nextInt(numDocs + 10);
+        final int expectedCount2 = Math.min(terminateAfter2, numDocs);
+        DummyTotalHitCountCollector collector2 = new DummyTotalHitCountCollector();
+
+        QueryPhaseCollector queryPhaseCollector = new QueryPhaseCollector(
+            new TerminateAfterCollector(collector1, terminateAfter1),
+            null,
+            0,
+            new TerminateAfterCollector(collector2, terminateAfter2),
+            null
+        );
+        searcher.search(new MatchAllDocsQuery(), queryPhaseCollector);
+        assertEquals(expectedCount1, collector1.getTotalHits());
+        assertEquals(expectedCount2, collector2.getTotalHits());
+    }
+
+    public void testSetScorerAfterCollectionTerminated() throws IOException {
+        MockCollector mockCollector1 = new MockCollector(randomFrom(ScoreMode.values()));
+        Collector collector1 = new TerminateAfterCollector(mockCollector1, 1);
+
+        MockCollector mockCollector2 = new MockCollector(randomScoreModeExceptTopScores());
+        Collector collector2 = new TerminateAfterCollector(mockCollector2, 2);
+
+        Scorable scorer = new Scorable() {
+            @Override
+            public float score() {
+                return 0;
+            }
+
+            @Override
+            public int docID() {
+                return 0;
+            }
+        };
+
+        QueryPhaseCollector queryPhaseCollector = new QueryPhaseCollector(collector1, null, 0, collector2, null);
+
+        LeafCollector leafCollector = queryPhaseCollector.getLeafCollector(null);
+        leafCollector.setScorer(scorer);
+        assertTrue(mockCollector1.setScorerCalled);
+        assertTrue(mockCollector2.setScorerCalled);
+
+        leafCollector.collect(0);
+        leafCollector.collect(1);
+
+        mockCollector1.setScorerCalled = false;
+        mockCollector2.setScorerCalled = false;
+        leafCollector.setScorer(scorer);
+        assertFalse(mockCollector1.setScorerCalled);
+        assertTrue(mockCollector2.setScorerCalled);
+
+        expectThrows(CollectionTerminatedException.class, () -> leafCollector.collect(1));
+
+        mockCollector1.setScorerCalled = false;
+        mockCollector2.setScorerCalled = false;
+        leafCollector.setScorer(scorer);
+        assertFalse(mockCollector1.setScorerCalled);
+        assertFalse(mockCollector2.setScorerCalled);
+    }
+
+    public void testSetMinCompetitiveScoreIsEnabledTopDocsOnly() throws IOException {
+        // without aggs no need to disable set min competitive score
+        TopScoresCollector topDocs = new TopScoresCollector();
+        Collector queryPhaseCollector = new QueryPhaseCollector(topDocs, null, 0, null, null);
+        LeafReaderContext leafReaderContext = searcher.getLeafContexts().get(0);
+        LeafCollector leafCollector = queryPhaseCollector.getLeafCollector(leafReaderContext);
+        MinCompetitiveScoreScorable scorer = new MinCompetitiveScoreScorable();
+        leafCollector.setScorer(scorer);
+        leafCollector.collect(0);
+        assertTrue(scorer.setMinCompetitiveScoreCalled);
+    }
+
+    public void testSetMinCompetitiveScoreIsDisabledWithAggs() throws IOException {
+        TopScoresCollector topDocs = new TopScoresCollector();
+        Collector aggs = new MockCollector(randomBoolean() ? ScoreMode.COMPLETE : ScoreMode.COMPLETE_NO_SCORES);
+        Collector queryPhaseCollector = new QueryPhaseCollector(topDocs, null, 0, aggs, null);
+        LeafReaderContext leafReaderContext = searcher.getLeafContexts().get(0);
+        LeafCollector leafCollector = queryPhaseCollector.getLeafCollector(leafReaderContext);
+        MinCompetitiveScoreScorable scorer = new MinCompetitiveScoreScorable();
+        leafCollector.setScorer(scorer);
+        leafCollector.collect(0);
+        assertFalse(scorer.setMinCompetitiveScoreCalled);
+    }
+
+    public void testSetMinCompetitiveScoreIsDisabledWithEarlyTerminatedAggs() throws IOException {
+        // aggs don't support top_scores: even if their collection terminated, we can't skip low scoring hits despite the
+        // top docs collector may support it, because the top-level score mode wasn't TOP_SCORES
+        TopScoresCollector topDocs = new TopScoresCollector();
+        Collector aggs = new TerminateAfterCollector(new DummyTotalHitCountCollector(), 0);
+        Collector queryPhaseCollector = new QueryPhaseCollector(topDocs, null, 0, aggs, null);
+        LeafReaderContext leafReaderContext = searcher.getLeafContexts().get(0);
+        LeafCollector leafCollector = queryPhaseCollector.getLeafCollector(leafReaderContext);
+        MinCompetitiveScoreScorable scorer = new MinCompetitiveScoreScorable();
+        leafCollector.setScorer(scorer);
+        leafCollector.collect(0);
+        assertFalse(scorer.setMinCompetitiveScoreCalled);
+    }
+
+    public void testCacheScoresIfNecessary() throws IOException {
+        final LeafReaderContext ctx = searcher.getLeafContexts().get(0);
+        {
+            // single collector => no caching
+            Collector c1 = new MockCollector(ScoreMode.COMPLETE, MinCompetitiveScoreScorable.class);
+            QueryPhaseCollector queryPhaseCollector = new QueryPhaseCollector(c1, null, 0, null, null);
+            queryPhaseCollector.getLeafCollector(ctx).setScorer(new MinCompetitiveScoreScorable());
+        }
+        {
+            // no collector needs scores => no caching
+            Collector c1 = new MockCollector(ScoreMode.COMPLETE_NO_SCORES, MinCompetitiveScoreScorable.class);
+            Collector c2 = new MockCollector(ScoreMode.COMPLETE_NO_SCORES, MinCompetitiveScoreScorable.class);
+            QueryPhaseCollector queryPhaseCollector = new QueryPhaseCollector(c1, null, 0, c2, null);
+            queryPhaseCollector.getLeafCollector(ctx).setScorer(new MinCompetitiveScoreScorable());
+        }
+        {
+            // only one collector needs scores => no caching
+            Collector c1 = new MockCollector(ScoreMode.COMPLETE, MinCompetitiveScoreScorable.class);
+            Collector c2 = new MockCollector(ScoreMode.COMPLETE_NO_SCORES, MinCompetitiveScoreScorable.class);
+            QueryPhaseCollector queryPhaseCollector = new QueryPhaseCollector(c1, null, 0, c2, null);
+            queryPhaseCollector.getLeafCollector(ctx).setScorer(new MinCompetitiveScoreScorable());
+        }
+        {
+            // both collectors need scores => caching
+            Collector c1 = new MockCollector(ScoreMode.COMPLETE, ScoreCachingWrappingScorer.class);
+            Collector c2 = new MockCollector(ScoreMode.COMPLETE, ScoreCachingWrappingScorer.class);
+            QueryPhaseCollector queryPhaseCollector = new QueryPhaseCollector(c1, null, 0, c2, null);
+            queryPhaseCollector.getLeafCollector(ctx).setScorer(new MinCompetitiveScoreScorable());
+        }
+    }
+
+    public void testCompetitiveIteratorTopDocsOnly() throws IOException {
+        MockCollector mockCollector = new MockCollector(randomFrom(ScoreMode.values()));
+        QueryPhaseCollector queryPhaseCollector = new QueryPhaseCollector(mockCollector, null, 0, null, null);
+        LeafReaderContext context = searcher.getLeafContexts().get(0);
+        LeafCollector leafCollector = queryPhaseCollector.getLeafCollector(context);
+        leafCollector.competitiveIterator();
+        assertTrue(mockCollector.competitiveIteratorCalled);
+    }
+
+    public void testCompetitiveIteratorTopDocsOnlyCollectionTerminated() throws IOException {
+        MockCollector mockCollector = new MockCollector(randomFrom(ScoreMode.values()));
+        TerminateAfterCollector terminateAfterCollector = new TerminateAfterCollector(mockCollector, 1);
+        QueryPhaseCollector queryPhaseCollector = new QueryPhaseCollector(terminateAfterCollector, null, 0, null, null);
+        LeafReaderContext context = searcher.getLeafContexts().get(0);
+        LeafCollector leafCollector = queryPhaseCollector.getLeafCollector(context);
+        leafCollector.competitiveIterator();
+        assertTrue(mockCollector.competitiveIteratorCalled);
+        mockCollector.competitiveIteratorCalled = false;
+        leafCollector.collect(0);
+        expectThrows(CollectionTerminatedException.class, () -> leafCollector.collect(1));
+        leafCollector.competitiveIterator();
+        assertFalse(mockCollector.competitiveIteratorCalled);
+    }
+
+    public void testCompetitiveIteratorWithAggs() throws IOException {
+        MockCollector topDocs = new MockCollector(randomFrom(ScoreMode.values()));
+        MockCollector aggs = new MockCollector(randomScoreModeExceptTopScores());
+        QueryPhaseCollector queryPhaseCollector = new QueryPhaseCollector(topDocs, null, 0, aggs, null);
+        LeafReaderContext context = searcher.getLeafContexts().get(0);
+        LeafCollector leafCollector = queryPhaseCollector.getLeafCollector(context);
+        leafCollector.competitiveIterator();
+        assertFalse(topDocs.competitiveIteratorCalled);
+        assertFalse(aggs.competitiveIteratorCalled);
+    }
+
+    public void testCompetitiveIteratorWithAggsCollectionTerminated() throws IOException {
+        MockCollector topDocsMockCollector = new MockCollector(randomFrom(ScoreMode.values()));
+        TerminateAfterCollector collector1 = new TerminateAfterCollector(topDocsMockCollector, 1);
+        MockCollector aggsMockCollector = new MockCollector(randomScoreModeExceptTopScores());
+        TerminateAfterCollector collector2 = new TerminateAfterCollector(aggsMockCollector, 2);
+        QueryPhaseCollector queryPhaseCollector;
+        if (randomBoolean()) {
+            queryPhaseCollector = new QueryPhaseCollector(collector1, null, 0, collector2, null);
+        } else {
+            queryPhaseCollector = new QueryPhaseCollector(collector2, null, 0, collector1, null);
+        }
+        LeafReaderContext context = searcher.getLeafContexts().get(0);
+        LeafCollector leafCollector = queryPhaseCollector.getLeafCollector(context);
+        leafCollector.competitiveIterator();
+        assertFalse(topDocsMockCollector.competitiveIteratorCalled);
+        assertFalse(aggsMockCollector.competitiveIteratorCalled);
+        leafCollector.collect(0);
+        leafCollector.competitiveIterator();
+        assertFalse(topDocsMockCollector.competitiveIteratorCalled);
+        assertFalse(aggsMockCollector.competitiveIteratorCalled);
+        leafCollector.collect(1);
+        leafCollector.competitiveIterator();
+        assertFalse(topDocsMockCollector.competitiveIteratorCalled);
+        // when top docs collection has terminated, we forward competitive iterator to aggs collection
+        assertTrue(aggsMockCollector.competitiveIteratorCalled);
+        aggsMockCollector.competitiveIteratorCalled = false;
+        expectThrows(CollectionTerminatedException.class, () -> leafCollector.collect(2));
+        assertFalse(topDocsMockCollector.competitiveIteratorCalled);
+        assertFalse(aggsMockCollector.competitiveIteratorCalled);
+    }
+
+    private static ScoreMode randomScoreModeExceptTopScores() {
+        return randomFrom(Arrays.stream(ScoreMode.values()).filter(scoreMode -> scoreMode != ScoreMode.TOP_SCORES).toList());
+    }
+
+    private static class TerminateAfterCollector extends FilterCollector {
+        private final int terminateAfter;
+        private int count = 0;
+
+        TerminateAfterCollector(Collector in, int terminateAfter) {
+            super(in);
+            this.terminateAfter = terminateAfter;
+        }
+
+        @Override
+        public LeafCollector getLeafCollector(LeafReaderContext context) throws IOException {
+            if (count >= terminateAfter) {
+                throw new CollectionTerminatedException();
+            }
+            final LeafCollector in = super.getLeafCollector(context);
+            return new FilterLeafCollector(in) {
+                @Override
+                public void collect(int doc) throws IOException {
+                    if (count >= terminateAfter) {
+                        throw new CollectionTerminatedException();
+                    }
+                    super.collect(doc);
+                    count++;
+                }
+
+                @Override
+                public DocIdSetIterator competitiveIterator() throws IOException {
+                    return in.competitiveIterator();
+                }
+            };
+        }
+    }
+
+    private static class TopScoresCollector extends SimpleCollector {
+        private Scorable scorer;
+        float minScore = 0;
+
+        @Override
+        public ScoreMode scoreMode() {
+            return ScoreMode.TOP_SCORES;
+        }
+
+        @Override
+        public void setScorer(Scorable scorer) {
+            this.scorer = scorer;
+        }
+
+        @Override
+        public void collect(int doc) throws IOException {
+            minScore = Math.nextUp(minScore);
+            scorer.setMinCompetitiveScore(minScore);
+        }
+    }
+
+    private static class MinCompetitiveScoreScorable extends Scorable {
+        boolean setMinCompetitiveScoreCalled = false;
+
+        @Override
+        public float score() throws IOException {
+            return 0;
+        }
+
+        @Override
+        public int docID() {
+            throw new UnsupportedOperationException();
+        }
+
+        @Override
+        public void setMinCompetitiveScore(float minScore) {
+            setMinCompetitiveScoreCalled = true;
+        }
+    }
+
+    private static class MockCollector extends SimpleCollector {
+        private final ScoreMode scoreMode;
+        private final Class<?> expectedScorable;
+        private boolean setScorerCalled = false;
+        private boolean setWeightCalled = false;
+        private boolean competitiveIteratorCalled = false;
+
+        MockCollector(ScoreMode scoreMode) {
+            this(scoreMode, null);
+        }
+
+        MockCollector(ScoreMode scoreMode, Class<?> expectedScorable) {
+            this.scoreMode = scoreMode;
+            this.expectedScorable = expectedScorable;
+        }
+
+        @Override
+        public void setWeight(Weight weight) {
+            setWeightCalled = true;
+        }
+
+        @Override
+        public ScoreMode scoreMode() {
+            return scoreMode;
+        }
+
+        @Override
+        public void setScorer(Scorable scorer) throws IOException {
+            setScorerCalled = true;
+            if (expectedScorable != null) {
+                while (expectedScorable.equals(scorer.getClass()) == false && scorer instanceof FilterScorable) {
+                    scorer = scorer.getChildren().iterator().next().child;
+                }
+                assertEquals(expectedScorable, scorer.getClass());
+            }
+        }
+
+        @Override
+        public DocIdSetIterator competitiveIterator() {
+            competitiveIteratorCalled = true;
+            return null;
+        }
+
+        @Override
+        public void collect(int doc) throws IOException {}
+    }
+}

+ 4 - 2
server/src/test/java/org/elasticsearch/search/query/QueryPhaseTests.java

@@ -232,6 +232,7 @@ public class QueryPhaseTests extends IndexShardTestCase {
             assertEquals(TotalHits.Relation.EQUAL_TO, context.queryResult().topDocs().topDocs.totalHits.relation);
         }
         {
+            // shortcutTotalHitCount makes us not track total hits as part of the top docs collection, hence size is the threshold
             TestSearchContext context = createContext(earlyTerminationContextSearcher(reader, 10), new MatchAllDocsQuery());
             context.setSize(10);
             QueryPhase.addCollectorsAndSearch(context);
@@ -239,7 +240,7 @@ public class QueryPhaseTests extends IndexShardTestCase {
             assertEquals(TotalHits.Relation.EQUAL_TO, context.queryResult().topDocs().topDocs.totalHits.relation);
         }
         {
-            // FilteredCollector does not propagate Weight#count, hence it forces collection despite
+            // QueryPhaseCollector does not propagate Weight#count when a post_filter is provided, hence it forces collection despite
             // the inner TotalHitCountCollector can shortcut
             TestSearchContext context = createContext(newContextSearcher(reader), new MatchAllDocsQuery());
             context.setSize(0);
@@ -281,6 +282,7 @@ public class QueryPhaseTests extends IndexShardTestCase {
             assertEquals(TotalHits.Relation.EQUAL_TO, context.queryResult().topDocs().topDocs.totalHits.relation);
         }
         {
+            // shortcutTotalHitCount makes us not track total hits as part of the top docs collection, hence size is the threshold
             TestSearchContext context = createContext(earlyTerminationContextSearcher(reader, 10), new MatchAllDocsQuery());
             context.setSize(10);
             QueryPhase.addCollectorsAndSearch(context);
@@ -288,7 +290,7 @@ public class QueryPhaseTests extends IndexShardTestCase {
             assertEquals(TotalHits.Relation.EQUAL_TO, context.queryResult().topDocs().topDocs.totalHits.relation);
         }
         {
-            // MinimumScoreCollector does not propagate Weight#count, hence it forces collection despite
+            // QueryPhaseCollector does not propagate Weight#count when min_score is provided, hence it forces collection despite
             // the inner TotalHitCountCollector can shortcut
             TestSearchContext context = createContext(newContextSearcher(reader), new MatchAllDocsQuery());
             context.setSize(0);