diff --git a/CHANGELOG.md b/CHANGELOG.md
index d8f2c996ef577..a8e0a6408c9c7 100644
--- a/CHANGELOG.md
+++ b/CHANGELOG.md
@@ -52,6 +52,7 @@ The format is based on [Keep a Changelog](https://keepachangelog.com/en/1.0.0/),
- Relax the join validation for Remote State publication ([#15471](https://github.com/opensearch-project/OpenSearch/pull/15471))
- Optimize NodeIndicesStats output behind flag ([#14454](https://github.com/opensearch-project/OpenSearch/pull/14454))
- Reset DiscoveryNodes in all transport node actions request ([#15131](https://github.com/opensearch-project/OpenSearch/pull/15131))
+- MultiTermQueries in keyword fields now default to `indexed` approach and gated behind cluster setting ([#15637](https://github.com/opensearch-project/OpenSearch/pull/15637))
### Dependencies
- Bump `netty` from 4.1.111.Final to 4.1.112.Final ([#15081](https://github.com/opensearch-project/OpenSearch/pull/15081))
diff --git a/server/src/main/java/org/opensearch/common/settings/ClusterSettings.java b/server/src/main/java/org/opensearch/common/settings/ClusterSettings.java
index f0ff79f5c74f6..89ef00e2f30a9 100644
--- a/server/src/main/java/org/opensearch/common/settings/ClusterSettings.java
+++ b/server/src/main/java/org/opensearch/common/settings/ClusterSettings.java
@@ -551,6 +551,7 @@ public void apply(Settings value, Settings current, Settings previous) {
SearchService.MAX_AGGREGATION_REWRITE_FILTERS,
SearchService.INDICES_MAX_CLAUSE_COUNT_SETTING,
SearchService.CARDINALITY_AGGREGATION_PRUNING_THRESHOLD,
+ SearchService.KEYWORD_INDEX_OR_DOC_VALUES_ENABLED,
CreatePitController.PIT_INIT_KEEP_ALIVE,
Node.WRITE_PORTS_FILE_SETTING,
Node.NODE_NAME_SETTING,
diff --git a/server/src/main/java/org/opensearch/index/IndexService.java b/server/src/main/java/org/opensearch/index/IndexService.java
index 6e67c8f83ee4c..83a53a48ec936 100644
--- a/server/src/main/java/org/opensearch/index/IndexService.java
+++ b/server/src/main/java/org/opensearch/index/IndexService.java
@@ -959,7 +959,8 @@ public QueryShardContext newQueryShardContext(
IndexSearcher searcher,
LongSupplier nowInMillis,
String clusterAlias,
- boolean validate
+ boolean validate,
+ boolean keywordIndexOrDocValuesEnabled
) {
final SearchIndexNameMatcher indexNameMatcher = new SearchIndexNameMatcher(
index().getName(),
@@ -985,10 +986,27 @@ public QueryShardContext newQueryShardContext(
indexNameMatcher,
allowExpensiveQueries,
valuesSourceRegistry,
- validate
+ validate,
+ keywordIndexOrDocValuesEnabled
);
}
+ /**
+ * Creates a new QueryShardContext.
+ *
+ * Passing a {@code null} {@link IndexSearcher} will return a valid context, however it won't be able to make
+ * {@link IndexReader}-specific optimizations, such as rewriting containing range queries.
+ */
+ public QueryShardContext newQueryShardContext(
+ int shardId,
+ IndexSearcher searcher,
+ LongSupplier nowInMillis,
+ String clusterAlias,
+ boolean validate
+ ) {
+ return newQueryShardContext(shardId, searcher, nowInMillis, clusterAlias, validate, false);
+ }
+
/**
* The {@link ThreadPool} to use for this index.
*/
diff --git a/server/src/main/java/org/opensearch/index/mapper/KeywordFieldMapper.java b/server/src/main/java/org/opensearch/index/mapper/KeywordFieldMapper.java
index 2116ac522b705..11ff601b3fd6d 100644
--- a/server/src/main/java/org/opensearch/index/mapper/KeywordFieldMapper.java
+++ b/server/src/main/java/org/opensearch/index/mapper/KeywordFieldMapper.java
@@ -392,6 +392,9 @@ public Query termsQuery(List> values, QueryShardContext context) {
failIfNotIndexedAndNoDocValues();
// has index and doc_values enabled
if (isSearchable() && hasDocValues()) {
+ if (!context.keywordFieldIndexOrDocValuesEnabled()) {
+ return super.termsQuery(values, context);
+ }
BytesRef[] bytesRefs = new BytesRef[values.size()];
for (int i = 0; i < bytesRefs.length; i++) {
bytesRefs[i] = indexedValueForSearch(values.get(i));
@@ -429,6 +432,9 @@ public Query prefixQuery(
}
failIfNotIndexedAndNoDocValues();
if (isSearchable() && hasDocValues()) {
+ if (!context.keywordFieldIndexOrDocValuesEnabled()) {
+ return super.prefixQuery(value, method, caseInsensitive, context);
+ }
Query indexQuery = super.prefixQuery(value, method, caseInsensitive, context);
Query dvQuery = super.prefixQuery(value, MultiTermQuery.DOC_VALUES_REWRITE, caseInsensitive, context);
return new IndexOrDocValuesQuery(indexQuery, dvQuery);
@@ -461,6 +467,9 @@ public Query regexpQuery(
}
failIfNotIndexedAndNoDocValues();
if (isSearchable() && hasDocValues()) {
+ if (!context.keywordFieldIndexOrDocValuesEnabled()) {
+ return super.regexpQuery(value, syntaxFlags, matchFlags, maxDeterminizedStates, method, context);
+ }
Query indexQuery = super.regexpQuery(value, syntaxFlags, matchFlags, maxDeterminizedStates, method, context);
Query dvQuery = super.regexpQuery(
value,
@@ -549,6 +558,9 @@ public Query fuzzyQuery(
);
}
if (isSearchable() && hasDocValues()) {
+ if (!context.keywordFieldIndexOrDocValuesEnabled()) {
+ return super.fuzzyQuery(value, fuzziness, prefixLength, maxExpansions, transpositions, method, context);
+ }
Query indexQuery = super.fuzzyQuery(value, fuzziness, prefixLength, maxExpansions, transpositions, method, context);
Query dvQuery = super.fuzzyQuery(
value,
@@ -591,6 +603,9 @@ public Query wildcardQuery(
// wildcard
// query text
if (isSearchable() && hasDocValues()) {
+ if (!context.keywordFieldIndexOrDocValuesEnabled()) {
+ return super.wildcardQuery(value, method, caseInsensitive, true, context);
+ }
Query indexQuery = super.wildcardQuery(value, method, caseInsensitive, true, context);
Query dvQuery = super.wildcardQuery(value, MultiTermQuery.DOC_VALUES_REWRITE, caseInsensitive, true, context);
return new IndexOrDocValuesQuery(indexQuery, dvQuery);
diff --git a/server/src/main/java/org/opensearch/index/query/QueryShardContext.java b/server/src/main/java/org/opensearch/index/query/QueryShardContext.java
index 05c07a0f32d5a..cc2d53cebbf69 100644
--- a/server/src/main/java/org/opensearch/index/query/QueryShardContext.java
+++ b/server/src/main/java/org/opensearch/index/query/QueryShardContext.java
@@ -126,6 +126,7 @@ public class QueryShardContext extends QueryRewriteContext {
private final ValuesSourceRegistry valuesSourceRegistry;
private BitSetProducer parentFilter;
private DerivedFieldResolver derivedFieldResolver;
+ private boolean keywordIndexOrDocValuesEnabled;
public QueryShardContext(
int shardId,
@@ -209,7 +210,55 @@ public QueryShardContext(
),
allowExpensiveQueries,
valuesSourceRegistry,
- validate
+ validate,
+ false
+ );
+ }
+
+ public QueryShardContext(
+ int shardId,
+ IndexSettings indexSettings,
+ BigArrays bigArrays,
+ BitsetFilterCache bitsetFilterCache,
+ TriFunction, IndexFieldData>> indexFieldDataLookup,
+ MapperService mapperService,
+ SimilarityService similarityService,
+ ScriptService scriptService,
+ NamedXContentRegistry xContentRegistry,
+ NamedWriteableRegistry namedWriteableRegistry,
+ Client client,
+ IndexSearcher searcher,
+ LongSupplier nowInMillis,
+ String clusterAlias,
+ Predicate indexNameMatcher,
+ BooleanSupplier allowExpensiveQueries,
+ ValuesSourceRegistry valuesSourceRegistry,
+ boolean validate,
+ boolean keywordIndexOrDocValuesEnabled
+ ) {
+ this(
+ shardId,
+ indexSettings,
+ bigArrays,
+ bitsetFilterCache,
+ indexFieldDataLookup,
+ mapperService,
+ similarityService,
+ scriptService,
+ xContentRegistry,
+ namedWriteableRegistry,
+ client,
+ searcher,
+ nowInMillis,
+ indexNameMatcher,
+ new Index(
+ RemoteClusterAware.buildRemoteIndexName(clusterAlias, indexSettings.getIndex().getName()),
+ indexSettings.getIndex().getUUID()
+ ),
+ allowExpensiveQueries,
+ valuesSourceRegistry,
+ validate,
+ keywordIndexOrDocValuesEnabled
);
}
@@ -232,7 +281,8 @@ public QueryShardContext(QueryShardContext source) {
source.fullyQualifiedIndex,
source.allowExpensiveQueries,
source.valuesSourceRegistry,
- source.validate()
+ source.validate(),
+ source.keywordIndexOrDocValuesEnabled
);
}
@@ -254,7 +304,8 @@ private QueryShardContext(
Index fullyQualifiedIndex,
BooleanSupplier allowExpensiveQueries,
ValuesSourceRegistry valuesSourceRegistry,
- boolean validate
+ boolean validate,
+ boolean keywordIndexOrDocValuesEnabled
) {
super(xContentRegistry, namedWriteableRegistry, client, nowInMillis, validate);
this.shardId = shardId;
@@ -278,6 +329,7 @@ private QueryShardContext(
emptyList(),
indexSettings.isDerivedFieldAllowed()
);
+ this.keywordIndexOrDocValuesEnabled = keywordIndexOrDocValuesEnabled;
}
private void reset() {
@@ -425,6 +477,14 @@ public MappedFieldType getDerivedFieldType(String fieldName) {
throw new UnsupportedOperationException("Use resolveDerivedFieldType() instead.");
}
+ public boolean keywordFieldIndexOrDocValuesEnabled() {
+ return keywordIndexOrDocValuesEnabled;
+ }
+
+ public void setKeywordFieldIndexOrDocValuesEnabled(boolean keywordIndexOrDocValuesEnabled) {
+ this.keywordIndexOrDocValuesEnabled = keywordIndexOrDocValuesEnabled;
+ }
+
public void setAllowUnmappedFields(boolean allowUnmappedFields) {
this.allowUnmappedFields = allowUnmappedFields;
}
diff --git a/server/src/main/java/org/opensearch/node/Node.java b/server/src/main/java/org/opensearch/node/Node.java
index 42b84d8e79d88..26150bdecdcf5 100644
--- a/server/src/main/java/org/opensearch/node/Node.java
+++ b/server/src/main/java/org/opensearch/node/Node.java
@@ -233,7 +233,7 @@
import org.opensearch.search.aggregations.support.AggregationUsageService;
import org.opensearch.search.backpressure.SearchBackpressureService;
import org.opensearch.search.backpressure.settings.SearchBackpressureSettings;
-import org.opensearch.search.deciders.ConcurrentSearchDecider;
+import org.opensearch.search.deciders.ConcurrentSearchRequestDecider;
import org.opensearch.search.fetch.FetchPhase;
import org.opensearch.search.pipeline.SearchPipelineService;
import org.opensearch.search.query.QueryPhase;
@@ -1336,7 +1336,7 @@ protected Node(
circuitBreakerService,
searchModule.getIndexSearcherExecutor(threadPool),
taskResourceTrackingService,
- searchModule.getConcurrentSearchDeciders()
+ searchModule.getConcurrentSearchRequestDeciderFactories()
);
final List> tasksExecutors = pluginsService.filterPlugins(PersistentTaskPlugin.class)
@@ -1986,7 +1986,7 @@ protected SearchService newSearchService(
CircuitBreakerService circuitBreakerService,
Executor indexSearcherExecutor,
TaskResourceTrackingService taskResourceTrackingService,
- Collection concurrentSearchDecidersList
+ Collection concurrentSearchDeciderFactories
) {
return new SearchService(
clusterService,
@@ -2000,7 +2000,7 @@ protected SearchService newSearchService(
circuitBreakerService,
indexSearcherExecutor,
taskResourceTrackingService,
- concurrentSearchDecidersList
+ concurrentSearchDeciderFactories
);
}
diff --git a/server/src/main/java/org/opensearch/plugins/SearchPlugin.java b/server/src/main/java/org/opensearch/plugins/SearchPlugin.java
index 895e6ed2971d8..60cb2184b5ab5 100644
--- a/server/src/main/java/org/opensearch/plugins/SearchPlugin.java
+++ b/server/src/main/java/org/opensearch/plugins/SearchPlugin.java
@@ -65,7 +65,7 @@
import org.opensearch.search.aggregations.pipeline.MovAvgPipelineAggregator;
import org.opensearch.search.aggregations.pipeline.PipelineAggregator;
import org.opensearch.search.aggregations.support.ValuesSourceRegistry;
-import org.opensearch.search.deciders.ConcurrentSearchDecider;
+import org.opensearch.search.deciders.ConcurrentSearchRequestDecider;
import org.opensearch.search.fetch.FetchSubPhase;
import org.opensearch.search.fetch.subphase.highlight.Highlighter;
import org.opensearch.search.query.QueryPhaseSearcher;
@@ -141,12 +141,12 @@ default Map getHighlighters() {
}
/**
- * Allows plugins to register custom decider for concurrent search
- * @return A {@link ConcurrentSearchDecider}
+ * Allows plugins to register a factory to create custom decider for concurrent search
+ * @return A {@link ConcurrentSearchRequestDecider.Factory}
*/
@ExperimentalApi
- default ConcurrentSearchDecider getConcurrentSearchDecider() {
- return null;
+ default Optional getConcurrentSearchRequestDeciderFactory() {
+ return Optional.empty();
}
/**
diff --git a/server/src/main/java/org/opensearch/search/DefaultSearchContext.java b/server/src/main/java/org/opensearch/search/DefaultSearchContext.java
index 4576921b8426e..d8fb89921213e 100644
--- a/server/src/main/java/org/opensearch/search/DefaultSearchContext.java
+++ b/server/src/main/java/org/opensearch/search/DefaultSearchContext.java
@@ -72,8 +72,8 @@
import org.opensearch.search.aggregations.SearchContextAggregations;
import org.opensearch.search.builder.SearchSourceBuilder;
import org.opensearch.search.collapse.CollapseContext;
-import org.opensearch.search.deciders.ConcurrentSearchDecider;
import org.opensearch.search.deciders.ConcurrentSearchDecision;
+import org.opensearch.search.deciders.ConcurrentSearchRequestDecider;
import org.opensearch.search.deciders.ConcurrentSearchVisitor;
import org.opensearch.search.dfs.DfsSearchResult;
import org.opensearch.search.fetch.FetchPhase;
@@ -106,13 +106,14 @@
import java.util.Collection;
import java.util.Collections;
import java.util.HashMap;
+import java.util.HashSet;
import java.util.List;
import java.util.Map;
+import java.util.Optional;
import java.util.Set;
import java.util.concurrent.Executor;
import java.util.function.Function;
import java.util.function.LongSupplier;
-import java.util.stream.Collectors;
import static org.opensearch.search.SearchService.CARDINALITY_AGGREGATION_PRUNING_THRESHOLD;
import static org.opensearch.search.SearchService.CLUSTER_CONCURRENT_SEGMENT_SEARCH_MODE;
@@ -120,6 +121,7 @@
import static org.opensearch.search.SearchService.CONCURRENT_SEGMENT_SEARCH_MODE_ALL;
import static org.opensearch.search.SearchService.CONCURRENT_SEGMENT_SEARCH_MODE_AUTO;
import static org.opensearch.search.SearchService.CONCURRENT_SEGMENT_SEARCH_MODE_NONE;
+import static org.opensearch.search.SearchService.KEYWORD_INDEX_OR_DOC_VALUES_ENABLED;
import static org.opensearch.search.SearchService.MAX_AGGREGATION_REWRITE_FILTERS;
/**
@@ -136,7 +138,7 @@ final class DefaultSearchContext extends SearchContext {
private final ShardSearchRequest request;
private final SearchShardTarget shardTarget;
private final LongSupplier relativeTimeSupplier;
- private final Collection concurrentSearchDeciders;
+ private final Collection concurrentSearchDeciderFactories;
private SearchType searchType;
private final BigArrays bigArrays;
private final IndexShard indexShard;
@@ -206,6 +208,7 @@ final class DefaultSearchContext extends SearchContext {
private final SetOnce requestShouldUseConcurrentSearch = new SetOnce<>();
private final int maxAggRewriteFilters;
private final int cardinalityAggregationPruningThreshold;
+ private final boolean keywordIndexOrDocValuesEnabled;
DefaultSearchContext(
ReaderContext readerContext,
@@ -221,7 +224,7 @@ final class DefaultSearchContext extends SearchContext {
boolean validate,
Executor executor,
Function requestToAggReduceContextBuilder,
- Collection concurrentSearchDeciders
+ Collection concurrentSearchDeciderFactories
) throws IOException {
this.readerContext = readerContext;
this.request = request;
@@ -256,7 +259,8 @@ final class DefaultSearchContext extends SearchContext {
this.searcher,
request::nowInMillis,
shardTarget.getClusterAlias(),
- validate
+ validate,
+ evaluateKeywordIndexOrDocValuesEnabled()
);
queryBoost = request.indexBoost();
this.lowLevelCancellation = lowLevelCancellation;
@@ -264,7 +268,8 @@ final class DefaultSearchContext extends SearchContext {
this.maxAggRewriteFilters = evaluateFilterRewriteSetting();
this.cardinalityAggregationPruningThreshold = evaluateCardinalityAggregationPruningThreshold();
- this.concurrentSearchDeciders = concurrentSearchDeciders;
+ this.keywordIndexOrDocValuesEnabled = evaluateKeywordIndexOrDocValuesEnabled();
+ this.concurrentSearchDeciderFactories = concurrentSearchDeciderFactories;
}
@Override
@@ -928,14 +933,21 @@ public boolean shouldUseConcurrentSearch() {
private boolean evaluateAutoMode() {
- // filter out deciders that want to opt-out of decision-making
- final Set filteredDeciders = concurrentSearchDeciders.stream()
- .filter(concurrentSearchDecider -> concurrentSearchDecider.canEvaluateForIndex(indexService.getIndexSettings()))
- .collect(Collectors.toSet());
+ final Set concurrentSearchRequestDeciders = new HashSet<>();
+
+ // create the ConcurrentSearchRequestDeciders using registered factories
+ for (ConcurrentSearchRequestDecider.Factory deciderFactory : concurrentSearchDeciderFactories) {
+ final Optional concurrentSearchRequestDecider = deciderFactory.create(
+ indexService.getIndexSettings()
+ );
+ concurrentSearchRequestDecider.ifPresent(concurrentSearchRequestDeciders::add);
+
+ }
+
// evaluate based on concurrent search query visitor
- if (filteredDeciders.size() > 0) {
+ if (concurrentSearchRequestDeciders.size() > 0) {
ConcurrentSearchVisitor concurrentSearchVisitor = new ConcurrentSearchVisitor(
- filteredDeciders,
+ concurrentSearchRequestDeciders,
indexService.getIndexSettings()
);
if (request().source() != null && request().source().query() != null) {
@@ -945,7 +957,7 @@ private boolean evaluateAutoMode() {
}
final List decisions = new ArrayList<>();
- for (ConcurrentSearchDecider decider : filteredDeciders) {
+ for (ConcurrentSearchRequestDecider decider : concurrentSearchRequestDeciders) {
ConcurrentSearchDecision decision = decider.getConcurrentSearchDecision();
if (decision != null) {
if (logger.isDebugEnabled()) {
@@ -1117,10 +1129,22 @@ public int cardinalityAggregationPruningThreshold() {
return cardinalityAggregationPruningThreshold;
}
+ @Override
+ public boolean keywordIndexOrDocValuesEnabled() {
+ return keywordIndexOrDocValuesEnabled;
+ }
+
private int evaluateCardinalityAggregationPruningThreshold() {
if (clusterService != null) {
return clusterService.getClusterSettings().get(CARDINALITY_AGGREGATION_PRUNING_THRESHOLD);
}
return 0;
}
+
+ public boolean evaluateKeywordIndexOrDocValuesEnabled() {
+ if (clusterService != null) {
+ return clusterService.getClusterSettings().get(KEYWORD_INDEX_OR_DOC_VALUES_ENABLED);
+ }
+ return false;
+ }
}
diff --git a/server/src/main/java/org/opensearch/search/SearchModule.java b/server/src/main/java/org/opensearch/search/SearchModule.java
index e9ed02828b971..b8d3a13e0df20 100644
--- a/server/src/main/java/org/opensearch/search/SearchModule.java
+++ b/server/src/main/java/org/opensearch/search/SearchModule.java
@@ -255,7 +255,7 @@
import org.opensearch.search.aggregations.pipeline.SumBucketPipelineAggregationBuilder;
import org.opensearch.search.aggregations.pipeline.SumBucketPipelineAggregator;
import org.opensearch.search.aggregations.support.ValuesSourceRegistry;
-import org.opensearch.search.deciders.ConcurrentSearchDecider;
+import org.opensearch.search.deciders.ConcurrentSearchRequestDecider;
import org.opensearch.search.fetch.FetchPhase;
import org.opensearch.search.fetch.FetchSubPhase;
import org.opensearch.search.fetch.subphase.ExplainPhase;
@@ -334,7 +334,7 @@ public class SearchModule {
private final QueryPhaseSearcher queryPhaseSearcher;
private final SearchPlugin.ExecutorServiceProvider indexSearcherExecutorProvider;
- private final Collection concurrentSearchDeciders;
+ private final Collection concurrentSearchDeciderFactories;
/**
* Constructs a new SearchModule object
@@ -364,25 +364,23 @@ public SearchModule(Settings settings, List plugins) {
queryPhaseSearcher = registerQueryPhaseSearcher(plugins);
indexSearcherExecutorProvider = registerIndexSearcherExecutorProvider(plugins);
namedWriteables.addAll(SortValue.namedWriteables());
- concurrentSearchDeciders = registerConcurrentSearchDeciders(plugins);
+ concurrentSearchDeciderFactories = registerConcurrentSearchDeciderFactories(plugins);
}
- private Collection registerConcurrentSearchDeciders(List plugins) {
- List concurrentSearchDeciders = new ArrayList<>();
+ private Collection registerConcurrentSearchDeciderFactories(List plugins) {
+ List concurrentSearchDeciderFactories = new ArrayList<>();
for (SearchPlugin plugin : plugins) {
- ConcurrentSearchDecider decider = plugin.getConcurrentSearchDecider();
- if (decider != null) {
- concurrentSearchDeciders.add(decider);
- }
+ final Optional deciderFactory = plugin.getConcurrentSearchRequestDeciderFactory();
+ deciderFactory.ifPresent(concurrentSearchDeciderFactories::add);
}
- return concurrentSearchDeciders;
+ return concurrentSearchDeciderFactories;
}
/**
- * Returns the concurrent search deciders that the plugins have registered
+ * Returns the concurrent search decider factories that the plugins have registered
*/
- public Collection getConcurrentSearchDeciders() {
- return concurrentSearchDeciders;
+ public Collection getConcurrentSearchRequestDeciderFactories() {
+ return concurrentSearchDeciderFactories;
}
public List getNamedWriteables() {
diff --git a/server/src/main/java/org/opensearch/search/SearchService.java b/server/src/main/java/org/opensearch/search/SearchService.java
index e33a47fe8e178..f8783c4b3c2da 100644
--- a/server/src/main/java/org/opensearch/search/SearchService.java
+++ b/server/src/main/java/org/opensearch/search/SearchService.java
@@ -105,7 +105,7 @@
import org.opensearch.search.aggregations.pipeline.PipelineAggregator.PipelineTree;
import org.opensearch.search.builder.SearchSourceBuilder;
import org.opensearch.search.collapse.CollapseContext;
-import org.opensearch.search.deciders.ConcurrentSearchDecider;
+import org.opensearch.search.deciders.ConcurrentSearchRequestDecider;
import org.opensearch.search.dfs.DfsPhase;
import org.opensearch.search.dfs.DfsSearchResult;
import org.opensearch.search.fetch.FetchPhase;
@@ -338,6 +338,13 @@ public class SearchService extends AbstractLifecycleComponent implements IndexEv
Property.NodeScope
);
+ public static final Setting KEYWORD_INDEX_OR_DOC_VALUES_ENABLED = Setting.boolSetting(
+ "search.keyword_index_or_doc_values_enabled",
+ false,
+ Property.Dynamic,
+ Property.NodeScope
+ );
+
public static final int DEFAULT_SIZE = 10;
public static final int DEFAULT_FROM = 0;
@@ -358,7 +365,7 @@ public class SearchService extends AbstractLifecycleComponent implements IndexEv
private final QueryPhase queryPhase;
private final FetchPhase fetchPhase;
- private final Collection concurrentSearchDeciders;
+ private final Collection concurrentSearchDeciderFactories;
private volatile long defaultKeepAlive;
@@ -404,7 +411,7 @@ public SearchService(
CircuitBreakerService circuitBreakerService,
Executor indexSearcherExecutor,
TaskResourceTrackingService taskResourceTrackingService,
- Collection concurrentSearchDeciders
+ Collection concurrentSearchDeciderFactories
) {
Settings settings = clusterService.getSettings();
this.threadPool = threadPool;
@@ -460,7 +467,7 @@ public SearchService(
allowDerivedField = CLUSTER_ALLOW_DERIVED_FIELD_SETTING.get(settings);
clusterService.getClusterSettings().addSettingsUpdateConsumer(CLUSTER_ALLOW_DERIVED_FIELD_SETTING, this::setAllowDerivedField);
- this.concurrentSearchDeciders = concurrentSearchDeciders;
+ this.concurrentSearchDeciderFactories = concurrentSearchDeciderFactories;
}
private void validateKeepAlives(TimeValue defaultKeepAlive, TimeValue maxKeepAlive) {
@@ -1161,7 +1168,7 @@ private DefaultSearchContext createSearchContext(ReaderContext reader, ShardSear
validate,
indexSearcherExecutor,
this::aggReduceContextBuilder,
- concurrentSearchDeciders
+ concurrentSearchDeciderFactories
);
// we clone the query shard context here just for rewriting otherwise we
// might end up with incorrect state since we are using now() or script services
@@ -1174,6 +1181,7 @@ private DefaultSearchContext createSearchContext(ReaderContext reader, ShardSear
context.getIndexSettings().isDerivedFieldAllowed() && allowDerivedField
);
context.setDerivedFieldResolver(derivedFieldResolver);
+ context.setKeywordFieldIndexOrDocValuesEnabled(searchContext.keywordIndexOrDocValuesEnabled());
searchContext.getQueryShardContext().setDerivedFieldResolver(derivedFieldResolver);
Rewriteable.rewrite(request.getRewriteable(), context, true);
assert searchContext.getQueryShardContext().isCacheable();
diff --git a/server/src/main/java/org/opensearch/search/deciders/ConcurrentSearchDecision.java b/server/src/main/java/org/opensearch/search/deciders/ConcurrentSearchDecision.java
index 2a30413eff9c8..4ac47221856d1 100644
--- a/server/src/main/java/org/opensearch/search/deciders/ConcurrentSearchDecision.java
+++ b/server/src/main/java/org/opensearch/search/deciders/ConcurrentSearchDecision.java
@@ -13,7 +13,7 @@
import java.util.Collection;
/**
- * This Class defines the decisions that a {@link ConcurrentSearchDecider#getConcurrentSearchDecision} can return.
+ * This Class defines the decisions that a {@link ConcurrentSearchRequestDecider#getConcurrentSearchDecision} can return.
*
*/
@ExperimentalApi
diff --git a/server/src/main/java/org/opensearch/search/deciders/ConcurrentSearchDecider.java b/server/src/main/java/org/opensearch/search/deciders/ConcurrentSearchRequestDecider.java
similarity index 50%
rename from server/src/main/java/org/opensearch/search/deciders/ConcurrentSearchDecider.java
rename to server/src/main/java/org/opensearch/search/deciders/ConcurrentSearchRequestDecider.java
index 9c588bb45b4ec..ec40527314454 100644
--- a/server/src/main/java/org/opensearch/search/deciders/ConcurrentSearchDecider.java
+++ b/server/src/main/java/org/opensearch/search/deciders/ConcurrentSearchRequestDecider.java
@@ -12,17 +12,21 @@
import org.opensearch.index.IndexSettings;
import org.opensearch.index.query.QueryBuilder;
+import java.util.Optional;
+
/**
- * {@link ConcurrentSearchDecider} allows pluggable way to evaluate if a query in the search request
+ * {@link ConcurrentSearchRequestDecider} allows pluggable way to evaluate if a query in the search request
* can use concurrent segment search using the passed in queryBuilders from query tree and index settings
* on a per shard request basis.
- * Implementations can also opt out of the evaluation process for certain indices based on the index settings.
- * For all the deciders which can evaluate query tree for an index, its evaluateForQuery method
- * will be called for each node in the query tree. After traversing of the query tree is completed, the final
- * decision from the deciders will be obtained using {@link ConcurrentSearchDecider#getConcurrentSearchDecision}
+ * Implementations will need to implement the Factory interface that can be used to create the ConcurrentSearchRequestDecider
+ * This factory will be called on each shard search request to create the ConcurrentSearchRequestDecider and get the
+ * concurrent search decision from the created decider on a per-request basis.
+ * For all the deciders the evaluateForQuery method will be called for each node in the query tree.
+ * After traversing of the query tree is completed, the final decision from the deciders will be
+ * obtained using {@link ConcurrentSearchRequestDecider#getConcurrentSearchDecision}
*/
@ExperimentalApi
-public abstract class ConcurrentSearchDecider {
+public abstract class ConcurrentSearchRequestDecider {
/**
* Evaluate for the passed in queryBuilder node in the query tree of the search request
@@ -31,14 +35,6 @@ public abstract class ConcurrentSearchDecider {
*/
public abstract void evaluateForQuery(QueryBuilder queryBuilder, IndexSettings indexSettings);
- /**
- * Provides a way for deciders to opt out of decision-making process for certain requests based on
- * index settings.
- * Return true if interested in decision making for index,
- * false, otherwise
- */
- public abstract boolean canEvaluateForIndex(IndexSettings indexSettings);
-
/**
* Provide the final decision for concurrent search based on all evaluations
* Plugins may need to maintain internal state of evaluations to provide a final decision
@@ -47,4 +43,16 @@ public abstract class ConcurrentSearchDecider {
*/
public abstract ConcurrentSearchDecision getConcurrentSearchDecision();
+ /**
+ * Factory interface that can be implemented to create the ConcurrentSearchRequestDecider object.
+ * Implementations can use the passed in indexSettings to decide whether to create the decider object or
+ * return {@link Optional#empty()}.
+ */
+ @ExperimentalApi
+ public interface Factory {
+ default Optional create(IndexSettings indexSettings) {
+ return Optional.empty();
+ }
+ }
+
}
diff --git a/server/src/main/java/org/opensearch/search/deciders/ConcurrentSearchVisitor.java b/server/src/main/java/org/opensearch/search/deciders/ConcurrentSearchVisitor.java
index 12ba1b2a9cc5f..d1a4fa982dc7e 100644
--- a/server/src/main/java/org/opensearch/search/deciders/ConcurrentSearchVisitor.java
+++ b/server/src/main/java/org/opensearch/search/deciders/ConcurrentSearchVisitor.java
@@ -19,15 +19,15 @@
/**
* Class to traverse the QueryBuilder tree and invoke the
- * {@link ConcurrentSearchDecider#evaluateForQuery} at each node of the query tree
+ * {@link ConcurrentSearchRequestDecider#evaluateForQuery} at each node of the query tree
*/
@ExperimentalApi
public class ConcurrentSearchVisitor implements QueryBuilderVisitor {
- private final Set deciders;
+ private final Set deciders;
private final IndexSettings indexSettings;
- public ConcurrentSearchVisitor(Set concurrentSearchVisitorDeciders, IndexSettings idxSettings) {
+ public ConcurrentSearchVisitor(Set concurrentSearchVisitorDeciders, IndexSettings idxSettings) {
Objects.requireNonNull(concurrentSearchVisitorDeciders, "Concurrent search deciders cannot be null");
deciders = concurrentSearchVisitorDeciders;
indexSettings = idxSettings;
diff --git a/server/src/main/java/org/opensearch/search/internal/SearchContext.java b/server/src/main/java/org/opensearch/search/internal/SearchContext.java
index fb822bcf39619..4b1b720a1aed7 100644
--- a/server/src/main/java/org/opensearch/search/internal/SearchContext.java
+++ b/server/src/main/java/org/opensearch/search/internal/SearchContext.java
@@ -530,4 +530,9 @@ public int maxAggRewriteFilters() {
public int cardinalityAggregationPruningThreshold() {
return 0;
}
+
+ public boolean keywordIndexOrDocValuesEnabled() {
+ return false;
+ }
+
}
diff --git a/server/src/test/java/org/opensearch/index/mapper/KeywordFieldTypeTests.java b/server/src/test/java/org/opensearch/index/mapper/KeywordFieldTypeTests.java
index b10035f54a0c0..f291b864beb59 100644
--- a/server/src/test/java/org/opensearch/index/mapper/KeywordFieldTypeTests.java
+++ b/server/src/test/java/org/opensearch/index/mapper/KeywordFieldTypeTests.java
@@ -136,7 +136,7 @@ public void testTermsQuery() {
new TermInSetQuery("field", terms),
new TermInSetQuery(MultiTermQuery.DOC_VALUES_REWRITE, "field", terms)
);
- assertEquals(expected, ft.termsQuery(Arrays.asList("foo", "bar"), null));
+ assertEquals(expected, ft.termsQuery(Arrays.asList("foo", "bar"), MOCK_QSC_ENABLE_INDEX_DOC_VALUES));
MappedFieldType onlyIndexed = new KeywordFieldType("field", true, false, Collections.emptyMap());
Query expectedIndex = new TermInSetQuery("field", terms);
@@ -225,7 +225,7 @@ public void testRegexpQuery() {
new RegexpQuery(new Term("field", "foo.*")),
new RegexpQuery(new Term("field", "foo.*"), 0, 0, RegexpQuery.DEFAULT_PROVIDER, 10, MultiTermQuery.DOC_VALUES_REWRITE)
),
- ft.regexpQuery("foo.*", 0, 0, 10, MultiTermQuery.CONSTANT_SCORE_BLENDED_REWRITE, MOCK_QSC)
+ ft.regexpQuery("foo.*", 0, 0, 10, MultiTermQuery.CONSTANT_SCORE_BLENDED_REWRITE, MOCK_QSC_ENABLE_INDEX_DOC_VALUES)
);
Query indexExpected = new RegexpQuery(new Term("field", "foo.*"));
@@ -267,7 +267,7 @@ public void testFuzzyQuery() {
new FuzzyQuery(new Term("field", "foo"), 2, 1, 50, true),
new FuzzyQuery(new Term("field", "foo"), 2, 1, 50, true, MultiTermQuery.DOC_VALUES_REWRITE)
),
- ft.fuzzyQuery("foo", Fuzziness.fromEdits(2), 1, 50, true, null, MOCK_QSC)
+ ft.fuzzyQuery("foo", Fuzziness.fromEdits(2), 1, 50, true, null, MOCK_QSC_ENABLE_INDEX_DOC_VALUES)
);
Query indexExpected = new FuzzyQuery(new Term("field", "foo"), 2, 1, 50, true);
@@ -308,7 +308,7 @@ public void testWildCardQuery() {
MultiTermQuery.DOC_VALUES_REWRITE
)
);
- assertEquals(expected, ft.wildcardQuery("foo*", MultiTermQuery.CONSTANT_SCORE_BLENDED_REWRITE, MOCK_QSC));
+ assertEquals(expected, ft.wildcardQuery("foo*", MultiTermQuery.CONSTANT_SCORE_BLENDED_REWRITE, MOCK_QSC_ENABLE_INDEX_DOC_VALUES));
Query indexExpected = new WildcardQuery(new Term("field", new BytesRef("foo*")));
MappedFieldType onlyIndexed = new KeywordFieldType("field", true, false, Collections.emptyMap());
diff --git a/server/src/test/java/org/opensearch/index/search/NestedHelperTests.java b/server/src/test/java/org/opensearch/index/search/NestedHelperTests.java
index 7ffcc0fb7437a..f7f921e824490 100644
--- a/server/src/test/java/org/opensearch/index/search/NestedHelperTests.java
+++ b/server/src/test/java/org/opensearch/index/search/NestedHelperTests.java
@@ -57,6 +57,8 @@
import java.io.IOException;
import java.util.Collections;
+import static org.opensearch.index.mapper.FieldTypeTestCase.MOCK_QSC_ENABLE_INDEX_DOC_VALUES;
+
public class NestedHelperTests extends OpenSearchSingleNodeTestCase {
IndexService indexService;
@@ -132,28 +134,28 @@ public void testMatchNo() {
}
public void testTermsQuery() {
- Query termsQuery = mapperService.fieldType("foo").termsQuery(Collections.singletonList("bar"), null);
+ Query termsQuery = mapperService.fieldType("foo").termsQuery(Collections.singletonList("bar"), MOCK_QSC_ENABLE_INDEX_DOC_VALUES);
assertFalse(new NestedHelper(mapperService).mightMatchNestedDocs(termsQuery));
assertTrue(new NestedHelper(mapperService).mightMatchNonNestedDocs(termsQuery, "nested1"));
assertTrue(new NestedHelper(mapperService).mightMatchNonNestedDocs(termsQuery, "nested2"));
assertTrue(new NestedHelper(mapperService).mightMatchNonNestedDocs(termsQuery, "nested3"));
assertTrue(new NestedHelper(mapperService).mightMatchNonNestedDocs(termsQuery, "nested_missing"));
- termsQuery = mapperService.fieldType("nested1.foo").termsQuery(Collections.singletonList("bar"), null);
+ termsQuery = mapperService.fieldType("nested1.foo").termsQuery(Collections.singletonList("bar"), MOCK_QSC_ENABLE_INDEX_DOC_VALUES);
assertTrue(new NestedHelper(mapperService).mightMatchNestedDocs(termsQuery));
assertFalse(new NestedHelper(mapperService).mightMatchNonNestedDocs(termsQuery, "nested1"));
assertTrue(new NestedHelper(mapperService).mightMatchNonNestedDocs(termsQuery, "nested2"));
assertTrue(new NestedHelper(mapperService).mightMatchNonNestedDocs(termsQuery, "nested3"));
assertTrue(new NestedHelper(mapperService).mightMatchNonNestedDocs(termsQuery, "nested_missing"));
- termsQuery = mapperService.fieldType("nested2.foo").termsQuery(Collections.singletonList("bar"), null);
+ termsQuery = mapperService.fieldType("nested2.foo").termsQuery(Collections.singletonList("bar"), MOCK_QSC_ENABLE_INDEX_DOC_VALUES);
assertTrue(new NestedHelper(mapperService).mightMatchNestedDocs(termsQuery));
assertTrue(new NestedHelper(mapperService).mightMatchNonNestedDocs(termsQuery, "nested1"));
assertTrue(new NestedHelper(mapperService).mightMatchNonNestedDocs(termsQuery, "nested2"));
assertTrue(new NestedHelper(mapperService).mightMatchNonNestedDocs(termsQuery, "nested3"));
assertTrue(new NestedHelper(mapperService).mightMatchNonNestedDocs(termsQuery, "nested_missing"));
- termsQuery = mapperService.fieldType("nested3.foo").termsQuery(Collections.singletonList("bar"), null);
+ termsQuery = mapperService.fieldType("nested3.foo").termsQuery(Collections.singletonList("bar"), MOCK_QSC_ENABLE_INDEX_DOC_VALUES);
assertTrue(new NestedHelper(mapperService).mightMatchNestedDocs(termsQuery));
assertTrue(new NestedHelper(mapperService).mightMatchNonNestedDocs(termsQuery, "nested1"));
assertTrue(new NestedHelper(mapperService).mightMatchNonNestedDocs(termsQuery, "nested2"));
diff --git a/server/src/test/java/org/opensearch/search/DefaultSearchContextTests.java b/server/src/test/java/org/opensearch/search/DefaultSearchContextTests.java
index 491a0377ab32e..55b30d5068daa 100644
--- a/server/src/test/java/org/opensearch/search/DefaultSearchContextTests.java
+++ b/server/src/test/java/org/opensearch/search/DefaultSearchContextTests.java
@@ -76,8 +76,8 @@
import org.opensearch.search.aggregations.MultiBucketConsumerService;
import org.opensearch.search.aggregations.SearchContextAggregations;
import org.opensearch.search.builder.SearchSourceBuilder;
-import org.opensearch.search.deciders.ConcurrentSearchDecider;
import org.opensearch.search.deciders.ConcurrentSearchDecision;
+import org.opensearch.search.deciders.ConcurrentSearchRequestDecider;
import org.opensearch.search.internal.AliasFilter;
import org.opensearch.search.internal.LegacyReaderContext;
import org.opensearch.search.internal.PitReaderContext;
@@ -96,6 +96,8 @@
import java.util.Arrays;
import java.util.Collection;
import java.util.Collections;
+import java.util.List;
+import java.util.Optional;
import java.util.UUID;
import java.util.concurrent.ExecutorService;
import java.util.concurrent.Executors;
@@ -168,9 +170,8 @@ public void testPreProcess() throws Exception {
when(indexCache.query()).thenReturn(queryCache);
when(indexService.cache()).thenReturn(indexCache);
QueryShardContext queryShardContext = mock(QueryShardContext.class);
- when(indexService.newQueryShardContext(eq(shardId.id()), any(), any(), nullable(String.class), anyBoolean())).thenReturn(
- queryShardContext
- );
+ when(indexService.newQueryShardContext(eq(shardId.id()), any(), any(), nullable(String.class), anyBoolean(), anyBoolean()))
+ .thenReturn(queryShardContext);
MapperService mapperService = mock(MapperService.class);
when(mapperService.hasNested()).thenReturn(randomBoolean());
when(indexService.mapperService()).thenReturn(mapperService);
@@ -501,9 +502,8 @@ public void testClearQueryCancellationsOnClose() throws IOException {
IndexService indexService = mock(IndexService.class);
QueryShardContext queryShardContext = mock(QueryShardContext.class);
- when(indexService.newQueryShardContext(eq(shardId.id()), any(), any(), nullable(String.class), anyBoolean())).thenReturn(
- queryShardContext
- );
+ when(indexService.newQueryShardContext(eq(shardId.id()), any(), any(), nullable(String.class), anyBoolean(), anyBoolean()))
+ .thenReturn(queryShardContext);
Settings settings = Settings.builder()
.put(IndexMetadata.SETTING_VERSION_CREATED, Version.CURRENT)
.put(IndexMetadata.SETTING_NUMBER_OF_REPLICAS, 1)
@@ -598,9 +598,8 @@ public void testSearchPathEvaluation() throws Exception {
IndexService indexService = mock(IndexService.class);
QueryShardContext queryShardContext = mock(QueryShardContext.class);
- when(indexService.newQueryShardContext(eq(shardId.id()), any(), any(), nullable(String.class), anyBoolean())).thenReturn(
- queryShardContext
- );
+ when(indexService.newQueryShardContext(eq(shardId.id()), any(), any(), nullable(String.class), anyBoolean(), anyBoolean()))
+ .thenReturn(queryShardContext);
IndexMetadata indexMetadata = IndexMetadata.builder("index").settings(settings).build();
IndexSettings indexSettings = new IndexSettings(indexMetadata, Settings.EMPTY);
@@ -830,9 +829,8 @@ public void testSearchPathEvaluationWithConcurrentSearchModeAsAuto() throws Exce
IndexService indexService = mock(IndexService.class);
QueryShardContext queryShardContext = mock(QueryShardContext.class);
- when(indexService.newQueryShardContext(eq(shardId.id()), any(), any(), nullable(String.class), anyBoolean())).thenReturn(
- queryShardContext
- );
+ when(indexService.newQueryShardContext(eq(shardId.id()), any(), any(), nullable(String.class), anyBoolean(), anyBoolean()))
+ .thenReturn(queryShardContext);
IndexMetadata indexMetadata = IndexMetadata.builder("index").settings(settings).build();
IndexSettings indexSettings = new IndexSettings(indexMetadata, Settings.EMPTY);
@@ -988,14 +986,34 @@ protected Engine.Searcher acquireSearcherInternal(String source) {
// Case4: multiple deciders are registered and all of them opt out of decision-making
// with supported agg query so concurrent path is used
- ConcurrentSearchDecider decider1 = mock(ConcurrentSearchDecider.class);
- when(decider1.canEvaluateForIndex(any())).thenReturn(false);
- ConcurrentSearchDecider decider2 = mock(ConcurrentSearchDecider.class);
- when(decider2.canEvaluateForIndex(any())).thenReturn(false);
+ ConcurrentSearchRequestDecider decider1 = mock(ConcurrentSearchRequestDecider.class);
- Collection concurrentSearchDeciders = new ArrayList<>();
- concurrentSearchDeciders.add(decider1);
- concurrentSearchDeciders.add(decider2);
+ ConcurrentSearchRequestDecider decider2 = mock(ConcurrentSearchRequestDecider.class);
+
+ ConcurrentSearchRequestDecider.Factory factory1 = new ConcurrentSearchRequestDecider.Factory() {
+ @Override
+ public Optional create(IndexSettings indexSettings) {
+ return Optional.ofNullable(decider1);
+ }
+ };
+
+ ConcurrentSearchRequestDecider.Factory factory2 = new ConcurrentSearchRequestDecider.Factory() {
+ @Override
+ public Optional create(IndexSettings indexSettings) {
+ return Optional.ofNullable(decider2);
+ }
+ };
+ ConcurrentSearchRequestDecider.Factory factory3 = new ConcurrentSearchRequestDecider.Factory() {
+ @Override
+ public Optional create(IndexSettings indexSettings) {
+ return Optional.empty();
+ }
+ };
+
+ List concurrentSearchRequestDeciders = new ArrayList<>();
+ concurrentSearchRequestDeciders.add(factory1);
+ concurrentSearchRequestDeciders.add(factory2);
+ concurrentSearchRequestDeciders.add(factory3);
context = new DefaultSearchContext(
readerContext,
@@ -1011,7 +1029,7 @@ protected Engine.Searcher acquireSearcherInternal(String source) {
false,
executor,
null,
- concurrentSearchDeciders
+ concurrentSearchRequestDeciders
);
// create a supported agg operation
context.aggregations(mockAggregations);
@@ -1025,15 +1043,9 @@ protected Engine.Searcher acquireSearcherInternal(String source) {
// Case5: multiple deciders are registered and one of them returns ConcurrentSearchDecision.DecisionStatus.NO
// use non-concurrent path even if query contains supported agg
- when(decider1.canEvaluateForIndex(any())).thenReturn(true);
when(decider1.getConcurrentSearchDecision()).thenReturn(
new ConcurrentSearchDecision(ConcurrentSearchDecision.DecisionStatus.NO, "disable concurrent search")
);
- when(decider2.canEvaluateForIndex(any())).thenReturn(false);
-
- concurrentSearchDeciders.clear();
- concurrentSearchDeciders.add(decider1);
- concurrentSearchDeciders.add(decider2);
// create a source so that query tree is parsed by visitor
SearchSourceBuilder sourceBuilder = new SearchSourceBuilder();
@@ -1055,7 +1067,7 @@ protected Engine.Searcher acquireSearcherInternal(String source) {
false,
executor,
null,
- concurrentSearchDeciders
+ concurrentSearchRequestDeciders
);
// create a supported agg operation
@@ -1071,20 +1083,17 @@ protected Engine.Searcher acquireSearcherInternal(String source) {
// Case6: multiple deciders are registered and first decider returns ConcurrentSearchDecision.DecisionStatus.YES
// while second decider returns ConcurrentSearchDecision.DecisionStatus.NO
// use non-concurrent path even if query contains supported agg
- when(decider1.canEvaluateForIndex(any())).thenReturn(true);
+
when(decider1.getConcurrentSearchDecision()).thenReturn(
new ConcurrentSearchDecision(ConcurrentSearchDecision.DecisionStatus.YES, "enable concurrent search")
);
- when(decider2.canEvaluateForIndex(any())).thenReturn(true);
+
when(decider2.getConcurrentSearchDecision()).thenReturn(
new ConcurrentSearchDecision(ConcurrentSearchDecision.DecisionStatus.NO, "disable concurrent search")
);
- concurrentSearchDeciders.clear();
- concurrentSearchDeciders.add(decider1);
- concurrentSearchDeciders.add(decider2);
-
// create a source so that query tree is parsed by visitor
+ when(shardSearchRequest.source()).thenReturn(sourceBuilder);
context = new DefaultSearchContext(
readerContext,
@@ -1100,7 +1109,7 @@ protected Engine.Searcher acquireSearcherInternal(String source) {
false,
executor,
null,
- concurrentSearchDeciders
+ concurrentSearchRequestDeciders
);
// create a supported agg operation
@@ -1115,22 +1124,19 @@ protected Engine.Searcher acquireSearcherInternal(String source) {
// Case7: multiple deciders are registered and all return ConcurrentSearchDecision.DecisionStatus.NO_OP
// but un-supported agg query is present, use non-concurrent path
- when(decider1.canEvaluateForIndex(any())).thenReturn(true);
+
when(decider1.getConcurrentSearchDecision()).thenReturn(
new ConcurrentSearchDecision(ConcurrentSearchDecision.DecisionStatus.NO_OP, "noop")
);
- when(decider2.canEvaluateForIndex(any())).thenReturn(true);
+
when(decider2.getConcurrentSearchDecision()).thenReturn(
new ConcurrentSearchDecision(ConcurrentSearchDecision.DecisionStatus.NO_OP, "noop")
);
when(mockAggregations.factories().allFactoriesSupportConcurrentSearch()).thenReturn(false);
- concurrentSearchDeciders.clear();
- concurrentSearchDeciders.add(decider1);
- concurrentSearchDeciders.add(decider2);
-
// create a source so that query tree is parsed by visitor
+ when(shardSearchRequest.source()).thenReturn(sourceBuilder);
context = new DefaultSearchContext(
readerContext,
@@ -1146,7 +1152,7 @@ protected Engine.Searcher acquireSearcherInternal(String source) {
false,
executor,
null,
- concurrentSearchDeciders
+ concurrentSearchRequestDeciders
);
// create a supported agg operation
diff --git a/server/src/test/java/org/opensearch/search/SearchModuleTests.java b/server/src/test/java/org/opensearch/search/SearchModuleTests.java
index b3483b76dee1c..81b7ca8aef30b 100644
--- a/server/src/test/java/org/opensearch/search/SearchModuleTests.java
+++ b/server/src/test/java/org/opensearch/search/SearchModuleTests.java
@@ -41,6 +41,7 @@
import org.opensearch.core.xcontent.NamedXContentRegistry;
import org.opensearch.core.xcontent.XContentBuilder;
import org.opensearch.core.xcontent.XContentParser;
+import org.opensearch.index.IndexSettings;
import org.opensearch.index.query.QueryBuilder;
import org.opensearch.index.query.QueryRewriteContext;
import org.opensearch.index.query.QueryShardContext;
@@ -70,7 +71,7 @@
import org.opensearch.search.aggregations.support.ValuesSourceConfig;
import org.opensearch.search.aggregations.support.ValuesSourceRegistry;
import org.opensearch.search.aggregations.support.ValuesSourceType;
-import org.opensearch.search.deciders.ConcurrentSearchDecider;
+import org.opensearch.search.deciders.ConcurrentSearchRequestDecider;
import org.opensearch.search.fetch.FetchSubPhase;
import org.opensearch.search.fetch.subphase.ExplainPhase;
import org.opensearch.search.fetch.subphase.highlight.CustomHighlighter;
@@ -508,12 +509,12 @@ public Optional getIndexSearcherExecutorProvider() {
expectThrows(IllegalStateException.class, () -> new SearchModule(Settings.EMPTY, searchPlugins));
}
- public void testRegisterConcurrentSearchDecidersNoExternalPlugins() {
+ public void testRegisterConcurrentSearchRequestDecidersNoExternalPlugins() {
SearchModule searchModule = new SearchModule(Settings.EMPTY, Collections.emptyList());
- assertEquals(searchModule.getConcurrentSearchDeciders().size(), 0);
+ assertEquals(searchModule.getConcurrentSearchRequestDeciderFactories().size(), 0);
}
- public void testRegisterConcurrentSearchDecidersExternalPluginsWithNoDeciders() {
+ public void testRegisterConcurrentSearchRequestDecidersExternalPluginsWithNoDeciders() {
SearchPlugin plugin1 = new SearchPlugin() {
@Override
public Optional getIndexSearcherExecutorProvider() {
@@ -528,10 +529,10 @@ public Optional getIndexSearcherExecutorProvider() {
searchPlugins.add(plugin2);
SearchModule searchModule = new SearchModule(Settings.EMPTY, searchPlugins);
- assertEquals(searchModule.getConcurrentSearchDeciders().size(), 0);
+ assertEquals(searchModule.getConcurrentSearchRequestDeciderFactories().size(), 0);
}
- public void testRegisterConcurrentSearchDecidersExternalPluginsWithDeciders() {
+ public void testRegisterConcurrentSearchRequestDecidersExternalPluginsWithDeciders() {
SearchPlugin pluginDecider1 = new SearchPlugin() {
@Override
public Optional getIndexSearcherExecutorProvider() {
@@ -539,15 +540,25 @@ public Optional getIndexSearcherExecutorProvider() {
}
@Override
- public ConcurrentSearchDecider getConcurrentSearchDecider() {
- return mock(ConcurrentSearchDecider.class);
+ public Optional getConcurrentSearchRequestDeciderFactory() {
+ return Optional.of(new ConcurrentSearchRequestDecider.Factory() {
+ @Override
+ public Optional create(IndexSettings indexSettings) {
+ return Optional.of(mock(ConcurrentSearchRequestDecider.class));
+ }
+ });
}
};
SearchPlugin pluginDecider2 = new SearchPlugin() {
@Override
- public ConcurrentSearchDecider getConcurrentSearchDecider() {
- return mock(ConcurrentSearchDecider.class);
+ public Optional getConcurrentSearchRequestDeciderFactory() {
+ return Optional.of(new ConcurrentSearchRequestDecider.Factory() {
+ @Override
+ public Optional create(IndexSettings indexSettings) {
+ return Optional.of(mock(ConcurrentSearchRequestDecider.class));
+ }
+ });
}
};
@@ -556,23 +567,7 @@ public ConcurrentSearchDecider getConcurrentSearchDecider() {
searchPlugins.add(pluginDecider2);
SearchModule searchModule = new SearchModule(Settings.EMPTY, searchPlugins);
- assertEquals(searchModule.getConcurrentSearchDeciders().size(), 2);
- }
-
- public void testRegisterConcurrentSearchDecidersPluginWithNullDecider() {
- SearchPlugin pluginWithNullDecider = new SearchPlugin() {
- @Override
- public ConcurrentSearchDecider getConcurrentSearchDecider() {
- return null;
- }
- };
-
- List searchPlugins = new ArrayList<>();
- searchPlugins.add(pluginWithNullDecider);
- SearchModule searchModule = new SearchModule(Settings.EMPTY, searchPlugins);
- // null decider is filtered out, so 0 deciders
- assertEquals(searchModule.getConcurrentSearchDeciders().size(), 0);
-
+ assertEquals(searchModule.getConcurrentSearchRequestDeciderFactories().size(), 2);
}
private static final String[] NON_DEPRECATED_QUERIES = new String[] {
diff --git a/test/framework/src/main/java/org/opensearch/index/mapper/FieldTypeTestCase.java b/test/framework/src/main/java/org/opensearch/index/mapper/FieldTypeTestCase.java
index 7ed0da8509fab..5d85844f3218d 100644
--- a/test/framework/src/main/java/org/opensearch/index/mapper/FieldTypeTestCase.java
+++ b/test/framework/src/main/java/org/opensearch/index/mapper/FieldTypeTestCase.java
@@ -46,16 +46,18 @@
/** Base test case for subclasses of MappedFieldType */
public abstract class FieldTypeTestCase extends OpenSearchTestCase {
- public static final QueryShardContext MOCK_QSC = createMockQueryShardContext(true);
- public static final QueryShardContext MOCK_QSC_DISALLOW_EXPENSIVE = createMockQueryShardContext(false);
+ public static final QueryShardContext MOCK_QSC = createMockQueryShardContext(true, false);
+ public static final QueryShardContext MOCK_QSC_DISALLOW_EXPENSIVE = createMockQueryShardContext(false, false);
+ public static final QueryShardContext MOCK_QSC_ENABLE_INDEX_DOC_VALUES = createMockQueryShardContext(true, true);
protected QueryShardContext randomMockShardContext() {
return randomFrom(MOCK_QSC, MOCK_QSC_DISALLOW_EXPENSIVE);
}
- static QueryShardContext createMockQueryShardContext(boolean allowExpensiveQueries) {
+ static QueryShardContext createMockQueryShardContext(boolean allowExpensiveQueries, boolean keywordIndexOrDocValuesEnabled) {
QueryShardContext queryShardContext = mock(QueryShardContext.class);
when(queryShardContext.allowExpensiveQueries()).thenReturn(allowExpensiveQueries);
+ when(queryShardContext.keywordFieldIndexOrDocValuesEnabled()).thenReturn(keywordIndexOrDocValuesEnabled);
return queryShardContext;
}
diff --git a/test/framework/src/main/java/org/opensearch/node/MockNode.java b/test/framework/src/main/java/org/opensearch/node/MockNode.java
index 09df9b85320f0..97c06962ca2e7 100644
--- a/test/framework/src/main/java/org/opensearch/node/MockNode.java
+++ b/test/framework/src/main/java/org/opensearch/node/MockNode.java
@@ -57,7 +57,7 @@
import org.opensearch.script.ScriptService;
import org.opensearch.search.MockSearchService;
import org.opensearch.search.SearchService;
-import org.opensearch.search.deciders.ConcurrentSearchDecider;
+import org.opensearch.search.deciders.ConcurrentSearchRequestDecider;
import org.opensearch.search.fetch.FetchPhase;
import org.opensearch.search.query.QueryPhase;
import org.opensearch.tasks.TaskResourceTrackingService;
@@ -158,7 +158,7 @@ protected SearchService newSearchService(
CircuitBreakerService circuitBreakerService,
Executor indexSearcherExecutor,
TaskResourceTrackingService taskResourceTrackingService,
- Collection concurrentSearchDecidersList
+ Collection concurrentSearchDeciderFactories
) {
if (getPluginsService().filterPlugins(MockSearchService.TestPlugin.class).isEmpty()) {
return super.newSearchService(
@@ -173,7 +173,7 @@ protected SearchService newSearchService(
circuitBreakerService,
indexSearcherExecutor,
taskResourceTrackingService,
- concurrentSearchDecidersList
+ concurrentSearchDeciderFactories
);
}
return new MockSearchService(