Use FallbackSyntheticSourceBlockLoader for point and geo_point (#125816)

This commit is contained in:
Oleksandr Kolomiiets 2025-04-01 12:55:18 -07:00 committed by GitHub
parent 7b46621c19
commit f3ccde6959
No known key found for this signature in database
GPG key ID: B5690EEEBB952194
27 changed files with 864 additions and 92 deletions

View file

@ -0,0 +1,5 @@
pr: 125816
summary: Use `FallbackSyntheticSourceBlockLoader` for point and `geo_point`
area: Mapping
type: enhancement
issues: []

View file

@ -22,8 +22,6 @@ import java.util.Objects;
import java.util.function.Function;
import java.util.function.Supplier;
import static org.elasticsearch.index.mapper.MappedFieldType.FieldExtractPreference.DOC_VALUES;
/** Base class for spatial fields that only support indexing points */
public abstract class AbstractPointGeometryFieldMapper<T> extends AbstractGeometryFieldMapper<T> {
@ -148,7 +146,6 @@ public abstract class AbstractPointGeometryFieldMapper<T> extends AbstractGeomet
}
public abstract static class AbstractPointFieldType<T extends SpatialPoint> extends AbstractGeometryFieldType<T> {
protected AbstractPointFieldType(
String name,
boolean indexed,
@ -165,13 +162,5 @@ public abstract class AbstractPointGeometryFieldMapper<T> extends AbstractGeomet
protected Object nullValueAsSource(T nullValue) {
return nullValue == null ? null : nullValue.toWKT();
}
@Override
public BlockLoader blockLoader(BlockLoaderContext blContext) {
if (blContext.fieldExtractPreference() == DOC_VALUES && hasDocValues()) {
return new BlockDocValuesReader.LongsBlockLoader(name());
}
return blockLoaderFromSource(blContext);
}
}
}

View file

@ -67,6 +67,8 @@ import java.util.Objects;
import java.util.Set;
import java.util.function.Function;
import static org.elasticsearch.index.mapper.MappedFieldType.FieldExtractPreference.DOC_VALUES;
/**
* Field Mapper for geo_point types.
*
@ -224,7 +226,8 @@ public class GeoPointFieldMapper extends AbstractPointGeometryFieldMapper<GeoPoi
scriptValues(),
meta.get(),
metric.get(),
indexMode
indexMode,
context.isSourceSynthetic()
);
hasScript = script.get() != null;
onScriptError = onScriptErrorParam.get();
@ -370,6 +373,7 @@ public class GeoPointFieldMapper extends AbstractPointGeometryFieldMapper<GeoPoi
private final FieldValues<GeoPoint> scriptValues;
private final IndexMode indexMode;
private final boolean isSyntheticSource;
private GeoPointFieldType(
String name,
@ -381,17 +385,19 @@ public class GeoPointFieldMapper extends AbstractPointGeometryFieldMapper<GeoPoi
FieldValues<GeoPoint> scriptValues,
Map<String, String> meta,
TimeSeriesParams.MetricType metricType,
IndexMode indexMode
IndexMode indexMode,
boolean isSyntheticSource
) {
super(name, indexed, stored, hasDocValues, parser, nullValue, meta);
this.scriptValues = scriptValues;
this.metricType = metricType;
this.indexMode = indexMode;
this.isSyntheticSource = isSyntheticSource;
}
// only used in test
public GeoPointFieldType(String name, TimeSeriesParams.MetricType metricType, IndexMode indexMode) {
this(name, true, false, true, null, null, null, Collections.emptyMap(), metricType, indexMode);
this(name, true, false, true, null, null, null, Collections.emptyMap(), metricType, indexMode, false);
}
// only used in test
@ -524,6 +530,28 @@ public class GeoPointFieldMapper extends AbstractPointGeometryFieldMapper<GeoPoi
public TimeSeriesParams.MetricType getMetricType() {
return metricType;
}
@Override
public BlockLoader blockLoader(BlockLoaderContext blContext) {
if (blContext.fieldExtractPreference() == DOC_VALUES && hasDocValues()) {
return new BlockDocValuesReader.LongsBlockLoader(name());
}
// There are two scenarios possible once we arrive here:
//
// * Stored source - we'll just use blockLoaderFromSource
// * Synthetic source. However, because of the fieldExtractPreference() check above it is still possible that doc_values are
// present here.
// So we have two subcases:
// - doc_values are enabled - _ignored_source field does not exist since we have doc_values. We will use
// blockLoaderFromSource which reads "native" synthetic source.
// - doc_values are disabled - we know that _ignored_source field is present and use a special block loader.
if (isSyntheticSource && hasDocValues() == false) {
return blockLoaderFromFallbackSyntheticSource(blContext);
}
return blockLoaderFromSource(blContext);
}
}
/** GeoPoint parser implementation */

View file

@ -23,7 +23,7 @@ public class BooleanFieldBlockLoaderTests extends BlockLoaderTestCase {
@Override
@SuppressWarnings("unchecked")
protected Object expected(Map<String, Object> fieldMapping, Object value) {
protected Object expected(Map<String, Object> fieldMapping, Object value, TestContext testContext) {
var nullValue = switch (fieldMapping.get("null_value")) {
case Boolean b -> b;
case String s -> Boolean.parseBoolean(s);

View file

@ -29,7 +29,7 @@ public class DateFieldBlockLoaderTests extends BlockLoaderTestCase {
@Override
@SuppressWarnings("unchecked")
protected Object expected(Map<String, Object> fieldMapping, Object value) {
protected Object expected(Map<String, Object> fieldMapping, Object value, TestContext testContext) {
var format = (String) fieldMapping.get("format");
var nullValue = fieldMapping.get("null_value") != null ? format(fieldMapping.get("null_value"), format) : null;

View file

@ -0,0 +1,207 @@
/*
* 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", the "GNU Affero General Public License v3.0 only", 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", the "GNU Affero General Public
* License v3.0 only", or the "Server Side Public License, v 1".
*/
package org.elasticsearch.index.mapper.blockloader;
import org.apache.lucene.util.BytesRef;
import org.elasticsearch.common.geo.GeoPoint;
import org.elasticsearch.geometry.Point;
import org.elasticsearch.geometry.utils.WellKnownBinary;
import org.elasticsearch.index.mapper.BlockLoaderTestCase;
import org.elasticsearch.index.mapper.MappedFieldType;
import java.nio.ByteOrder;
import java.util.ArrayList;
import java.util.Comparator;
import java.util.List;
import java.util.Map;
import java.util.Objects;
public class GeoPointFieldBlockLoaderTests extends BlockLoaderTestCase {
public GeoPointFieldBlockLoaderTests(BlockLoaderTestCase.Params params) {
super("geo_point", params);
}
@Override
@SuppressWarnings("unchecked")
protected Object expected(Map<String, Object> fieldMapping, Object value, TestContext testContext) {
var extractedFieldValues = (ExtractedFieldValues) value;
var values = extractedFieldValues.values();
var nullValue = switch (fieldMapping.get("null_value")) {
case String s -> convert(s, null);
case null -> null;
default -> throw new IllegalStateException("Unexpected null_value format");
};
if (params.preference() == MappedFieldType.FieldExtractPreference.DOC_VALUES && hasDocValues(fieldMapping, true)) {
if (values instanceof List<?> == false) {
var point = convert(values, nullValue);
return point != null ? point.getEncoded() : null;
}
var resultList = ((List<Object>) values).stream()
.map(v -> convert(v, nullValue))
.filter(Objects::nonNull)
.map(GeoPoint::getEncoded)
.sorted()
.toList();
return maybeFoldList(resultList);
}
if (params.syntheticSource() == false) {
return exactValuesFromSource(values, nullValue);
}
// Usually implementation of block loader from source adjusts values read from source
// so that they look the same as doc_values would (like reducing precision).
// geo_point does not do that and because of that we need to handle all these cases below.
// If we are reading from stored source or fallback synthetic source we get the same exact data as source.
// But if we are using "normal" synthetic source we get lesser precision data from doc_values.
// That is unless "synthetic_source_keep" forces fallback synthetic source again.
if (testContext.forceFallbackSyntheticSource()) {
return exactValuesFromSource(values, nullValue);
}
String syntheticSourceKeep = (String) fieldMapping.getOrDefault("synthetic_source_keep", "none");
if (syntheticSourceKeep.equals("all")) {
return exactValuesFromSource(values, nullValue);
}
if (syntheticSourceKeep.equals("arrays") && extractedFieldValues.documentHasObjectArrays()) {
return exactValuesFromSource(values, nullValue);
}
// synthetic source and doc_values are present
if (hasDocValues(fieldMapping, true)) {
if (values instanceof List<?> == false) {
return toWKB(normalize(convert(values, nullValue)));
}
var resultList = ((List<Object>) values).stream()
.map(v -> convert(v, nullValue))
.filter(Objects::nonNull)
.sorted(Comparator.comparingLong(GeoPoint::getEncoded))
.map(p -> toWKB(normalize(p)))
.toList();
return maybeFoldList(resultList);
}
// synthetic source but no doc_values so using fallback synthetic source
return exactValuesFromSource(values, nullValue);
}
@SuppressWarnings("unchecked")
private Object exactValuesFromSource(Object value, GeoPoint nullValue) {
if (value instanceof List<?> == false) {
return toWKB(convert(value, nullValue));
}
var resultList = ((List<Object>) value).stream().map(v -> convert(v, nullValue)).filter(Objects::nonNull).map(this::toWKB).toList();
return maybeFoldList(resultList);
}
private record ExtractedFieldValues(Object values, boolean documentHasObjectArrays) {}
@Override
protected Object getFieldValue(Map<String, Object> document, String fieldName) {
var extracted = new ArrayList<>();
var documentHasObjectArrays = processLevel(document, fieldName, extracted, false);
if (extracted.size() == 1) {
return new ExtractedFieldValues(extracted.get(0), documentHasObjectArrays);
}
return new ExtractedFieldValues(extracted, documentHasObjectArrays);
}
@SuppressWarnings("unchecked")
private boolean processLevel(Map<String, Object> level, String field, ArrayList<Object> extracted, boolean documentHasObjectArrays) {
if (field.contains(".") == false) {
var value = level.get(field);
processLeafLevel(value, extracted);
return documentHasObjectArrays;
}
var nameInLevel = field.split("\\.")[0];
var entry = level.get(nameInLevel);
if (entry instanceof Map<?, ?> m) {
return processLevel((Map<String, Object>) m, field.substring(field.indexOf('.') + 1), extracted, documentHasObjectArrays);
}
if (entry instanceof List<?> l) {
for (var object : l) {
processLevel((Map<String, Object>) object, field.substring(field.indexOf('.') + 1), extracted, true);
}
return true;
}
assert false : "unexpected document structure";
return false;
}
private void processLeafLevel(Object value, ArrayList<Object> extracted) {
if (value instanceof List<?> l) {
if (l.size() > 0 && l.get(0) instanceof Double) {
// this must be a single point in array form
// we'll put it into a different form here to make our lives a bit easier while implementing `expected`
extracted.add(Map.of("type", "point", "coordinates", l));
} else {
// this is actually an array of points but there could still be points in array form inside
for (var arrayValue : l) {
processLeafLevel(arrayValue, extracted);
}
}
} else {
extracted.add(value);
}
}
@SuppressWarnings("unchecked")
private GeoPoint convert(Object value, GeoPoint nullValue) {
if (value == null) {
return nullValue;
}
if (value instanceof String s) {
try {
return new GeoPoint(s);
} catch (Exception e) {
return null;
}
}
if (value instanceof Map<?, ?> m) {
if (m.get("type") != null) {
var coordinates = (List<Double>) m.get("coordinates");
// Order is GeoJSON is lon,lat
return new GeoPoint(coordinates.get(1), coordinates.get(0));
} else {
return new GeoPoint((Double) m.get("lat"), (Double) m.get("lon"));
}
}
// Malformed values are excluded
return null;
}
private GeoPoint normalize(GeoPoint point) {
if (point == null) {
return null;
}
return point.resetFromEncoded(point.getEncoded());
}
private BytesRef toWKB(GeoPoint point) {
if (point == null) {
return null;
}
return new BytesRef(WellKnownBinary.toWKB(new Point(point.getX(), point.getY()), ByteOrder.LITTLE_ENDIAN));
}
}

View file

@ -27,7 +27,7 @@ public class KeywordFieldBlockLoaderTests extends BlockLoaderTestCase {
@SuppressWarnings("unchecked")
@Override
protected Object expected(Map<String, Object> fieldMapping, Object value) {
protected Object expected(Map<String, Object> fieldMapping, Object value, TestContext testContext) {
var nullValue = (String) fieldMapping.get("null_value");
var ignoreAbove = fieldMapping.get("ignore_above") == null

View file

@ -44,6 +44,7 @@ import java.util.stream.Stream;
public abstract class BlockLoaderTestCase extends MapperServiceTestCase {
private static final MappedFieldType.FieldExtractPreference[] PREFERENCES = new MappedFieldType.FieldExtractPreference[] {
MappedFieldType.FieldExtractPreference.NONE,
MappedFieldType.FieldExtractPreference.DOC_VALUES,
MappedFieldType.FieldExtractPreference.STORED };
@ParametersFactory(argumentFormatting = "preference=%s")
@ -59,6 +60,8 @@ public abstract class BlockLoaderTestCase extends MapperServiceTestCase {
public record Params(boolean syntheticSource, MappedFieldType.FieldExtractPreference preference) {}
public record TestContext(boolean forceFallbackSyntheticSource) {}
private final String fieldType;
protected final Params params;
@ -73,6 +76,7 @@ public abstract class BlockLoaderTestCase extends MapperServiceTestCase {
protected BlockLoaderTestCase(String fieldType, Collection<DataSourceHandler> customHandlers, Params params) {
this.fieldType = fieldType;
this.params = params;
this.fieldName = randomAlphaOfLengthBetween(5, 10);
var specification = DataGeneratorSpecification.builder()
@ -112,7 +116,7 @@ public abstract class BlockLoaderTestCase extends MapperServiceTestCase {
var template = new Template(Map.of(fieldName, new Template.Leaf(fieldName, fieldType)));
var mapping = mappingGenerator.generate(template);
runTest(template, mapping, fieldName);
runTest(template, mapping, fieldName, new TestContext(false));
}
@SuppressWarnings("unchecked")
@ -138,17 +142,21 @@ public abstract class BlockLoaderTestCase extends MapperServiceTestCase {
var mapping = mappingGenerator.generate(template);
TestContext testContext = new TestContext(false);
if (params.syntheticSource && randomBoolean()) {
// force fallback synthetic source in the hierarchy
var docMapping = (Map<String, Object>) mapping.raw().get("_doc");
var topLevelMapping = (Map<String, Object>) ((Map<String, Object>) docMapping.get("properties")).get("top");
topLevelMapping.put("synthetic_source_keep", "all");
testContext = new TestContext(true);
}
runTest(template, mapping, fullFieldName.toString());
runTest(template, mapping, fullFieldName.toString(), testContext);
}
private void runTest(Template template, Mapping mapping, String fieldName) throws IOException {
private void runTest(Template template, Mapping mapping, String fieldName, TestContext testContext) throws IOException {
var mappingXContent = XContentBuilder.builder(XContentType.JSON.xContent()).map(mapping.raw());
var mapperService = params.syntheticSource
@ -159,13 +167,13 @@ public abstract class BlockLoaderTestCase extends MapperServiceTestCase {
var documentXContent = XContentBuilder.builder(XContentType.JSON.xContent()).map(document);
Object blockLoaderResult = setupAndInvokeBlockLoader(mapperService, documentXContent, fieldName);
Object expected = expected(mapping.lookup().get(fieldName), getFieldValue(document, fieldName));
Object expected = expected(mapping.lookup().get(fieldName), getFieldValue(document, fieldName), testContext);
assertEquals(expected, blockLoaderResult);
}
protected abstract Object expected(Map<String, Object> fieldMapping, Object value);
protected abstract Object expected(Map<String, Object> fieldMapping, Object value, TestContext testContext);
private Object getFieldValue(Map<String, Object> document, String fieldName) {
protected Object getFieldValue(Map<String, Object> document, String fieldName) {
var rawValues = new ArrayList<>();
processLevel(document, fieldName, rawValues);

View file

@ -22,7 +22,7 @@ public abstract class NumberFieldBlockLoaderTestCase<T extends Number> extends B
@Override
@SuppressWarnings("unchecked")
protected Object expected(Map<String, Object> fieldMapping, Object value) {
protected Object expected(Map<String, Object> fieldMapping, Object value, TestContext testContext) {
var nullValue = fieldMapping.get("null_value") != null ? convert((Number) fieldMapping.get("null_value"), fieldMapping) : null;
if (value instanceof List<?> == false) {
@ -30,7 +30,9 @@ public abstract class NumberFieldBlockLoaderTestCase<T extends Number> extends B
}
boolean hasDocValues = hasDocValues(fieldMapping, true);
boolean useDocValues = params.preference() == MappedFieldType.FieldExtractPreference.NONE || params.syntheticSource();
boolean useDocValues = params.preference() == MappedFieldType.FieldExtractPreference.NONE
|| params.preference() == MappedFieldType.FieldExtractPreference.DOC_VALUES
|| params.syntheticSource();
if (hasDocValues && useDocValues) {
// Sorted
var resultList = ((List<Object>) value).stream()

View file

@ -16,6 +16,7 @@ import org.elasticsearch.logsdb.datageneration.fields.leaf.CountedKeywordFieldDa
import org.elasticsearch.logsdb.datageneration.fields.leaf.DateFieldDataGenerator;
import org.elasticsearch.logsdb.datageneration.fields.leaf.DoubleFieldDataGenerator;
import org.elasticsearch.logsdb.datageneration.fields.leaf.FloatFieldDataGenerator;
import org.elasticsearch.logsdb.datageneration.fields.leaf.GeoPointFieldDataGenerator;
import org.elasticsearch.logsdb.datageneration.fields.leaf.HalfFloatFieldDataGenerator;
import org.elasticsearch.logsdb.datageneration.fields.leaf.IntegerFieldDataGenerator;
import org.elasticsearch.logsdb.datageneration.fields.leaf.KeywordFieldDataGenerator;
@ -40,7 +41,8 @@ public enum FieldType {
SCALED_FLOAT("scaled_float"),
COUNTED_KEYWORD("counted_keyword"),
BOOLEAN("boolean"),
DATE("date");
DATE("date"),
GEO_POINT("geo_point");
private final String name;
@ -63,6 +65,7 @@ public enum FieldType {
case COUNTED_KEYWORD -> new CountedKeywordFieldDataGenerator(fieldName, dataSource);
case BOOLEAN -> new BooleanFieldDataGenerator(dataSource);
case DATE -> new DateFieldDataGenerator(dataSource);
case GEO_POINT -> new GeoPointFieldDataGenerator(dataSource);
};
}
@ -81,6 +84,7 @@ public enum FieldType {
case "counted_keyword" -> FieldType.COUNTED_KEYWORD;
case "boolean" -> FieldType.BOOLEAN;
case "date" -> FieldType.DATE;
case "geo_point" -> FieldType.GEO_POINT;
default -> null;
};
}

View file

@ -66,6 +66,14 @@ public interface DataSourceHandler {
return null;
}
default DataSourceResponse.GeoPointGenerator handle(DataSourceRequest.GeoPointGenerator request) {
return null;
}
default DataSourceResponse.PointGenerator handle(DataSourceRequest.PointGenerator request) {
return null;
}
default DataSourceResponse.NullWrapper handle(DataSourceRequest.NullWrapper request) {
return null;
}
@ -86,6 +94,10 @@ public interface DataSourceHandler {
return null;
}
default DataSourceResponse.TransformWeightedWrapper handle(DataSourceRequest.TransformWeightedWrapper<?> request) {
return null;
}
default DataSourceResponse.ChildFieldGenerator handle(DataSourceRequest.ChildFieldGenerator request) {
return null;
}

View file

@ -9,10 +9,12 @@
package org.elasticsearch.logsdb.datageneration.datasource;
import org.elasticsearch.core.Tuple;
import org.elasticsearch.index.mapper.ObjectMapper;
import org.elasticsearch.logsdb.datageneration.DataGeneratorSpecification;
import org.elasticsearch.logsdb.datageneration.fields.DynamicMapping;
import java.util.List;
import java.util.Set;
import java.util.function.Function;
import java.util.function.Supplier;
@ -106,6 +108,18 @@ public interface DataSourceRequest<TResponse extends DataSourceResponse> {
}
}
record GeoPointGenerator() implements DataSourceRequest<DataSourceResponse.GeoPointGenerator> {
public DataSourceResponse.GeoPointGenerator accept(DataSourceHandler handler) {
return handler.handle(this);
}
}
record PointGenerator() implements DataSourceRequest<DataSourceResponse.PointGenerator> {
public DataSourceResponse.PointGenerator accept(DataSourceHandler handler) {
return handler.handle(this);
}
}
record NullWrapper() implements DataSourceRequest<DataSourceResponse.NullWrapper> {
public DataSourceResponse.NullWrapper accept(DataSourceHandler handler) {
return handler.handle(this);
@ -138,6 +152,14 @@ public interface DataSourceRequest<TResponse extends DataSourceResponse> {
}
}
record TransformWeightedWrapper<T>(List<Tuple<Double, Function<T, Object>>> transformations)
implements
DataSourceRequest<DataSourceResponse.TransformWeightedWrapper> {
public DataSourceResponse.TransformWeightedWrapper accept(DataSourceHandler handler) {
return handler.handle(this);
}
}
record ChildFieldGenerator(DataGeneratorSpecification specification)
implements
DataSourceRequest<DataSourceResponse.ChildFieldGenerator> {

View file

@ -46,6 +46,10 @@ public interface DataSourceResponse {
record ShapeGenerator(Supplier<Geometry> generator) implements DataSourceResponse {}
record PointGenerator(Supplier<Object> generator) implements DataSourceResponse {}
record GeoPointGenerator(Supplier<Object> generator) implements DataSourceResponse {}
record NullWrapper(Function<Supplier<Object>, Supplier<Object>> wrapper) implements DataSourceResponse {}
record ArrayWrapper(Function<Supplier<Object>, Supplier<Object>> wrapper) implements DataSourceResponse {}
@ -56,6 +60,8 @@ public interface DataSourceResponse {
record TransformWrapper(Function<Supplier<Object>, Supplier<Object>> wrapper) implements DataSourceResponse {}
record TransformWeightedWrapper(Function<Supplier<Object>, Supplier<Object>> wrapper) implements DataSourceResponse {}
interface ChildFieldGenerator extends DataSourceResponse {
int generateChildFieldCount();

View file

@ -9,6 +9,8 @@
package org.elasticsearch.logsdb.datageneration.datasource;
import org.elasticsearch.geo.GeometryTestUtils;
import org.elasticsearch.geometry.utils.WellKnownText;
import org.elasticsearch.index.mapper.Mapper;
import org.elasticsearch.index.mapper.ObjectMapper;
import org.elasticsearch.logsdb.datageneration.FieldType;
@ -48,6 +50,7 @@ public class DefaultMappingParametersHandler implements DataSourceHandler {
case COUNTED_KEYWORD -> plain(Map.of("index", ESTestCase.randomBoolean()));
case BOOLEAN -> booleanMapping(map);
case DATE -> dateMapping(map);
case GEO_POINT -> geoPointMapping(map);
});
}
@ -172,6 +175,21 @@ public class DefaultMappingParametersHandler implements DataSourceHandler {
};
}
private Supplier<Map<String, Object>> geoPointMapping(Map<String, Object> injected) {
return () -> {
if (ESTestCase.randomDouble() <= 0.2) {
var point = GeometryTestUtils.randomPoint(false);
injected.put("null_value", WellKnownText.toWKT(point));
}
if (ESTestCase.randomBoolean()) {
injected.put("ignore_malformed", ESTestCase.randomBoolean());
}
return injected;
};
}
@Override
public DataSourceResponse.ObjectMappingParametersGenerator handle(DataSourceRequest.ObjectMappingParametersGenerator request) {
if (request.isNested()) {

View file

@ -60,7 +60,6 @@ public class DefaultObjectGenerationHandler implements DataSourceHandler {
@Override
public DataSourceResponse.FieldTypeGenerator handle(DataSourceRequest.FieldTypeGenerator request) {
return new DataSourceResponse.FieldTypeGenerator(
() -> new DataSourceResponse.FieldTypeGenerator.FieldTypeInfo(ESTestCase.randomFrom(FieldType.values()).toString())
);

View file

@ -10,6 +10,7 @@
package org.elasticsearch.logsdb.datageneration.datasource;
import org.elasticsearch.test.ESTestCase;
import org.elasticsearch.test.geo.RandomGeoGenerator;
import java.math.BigInteger;
import java.time.Instant;
@ -72,4 +73,9 @@ public class DefaultPrimitiveTypesHandler implements DataSourceHandler {
public DataSourceResponse.InstantGenerator handle(DataSourceRequest.InstantGenerator request) {
return new DataSourceResponse.InstantGenerator(() -> ESTestCase.randomInstantBetween(Instant.ofEpochMilli(1), MAX_INSTANT));
}
@Override
public DataSourceResponse.GeoPointGenerator handle(DataSourceRequest.GeoPointGenerator request) {
return new DataSourceResponse.GeoPointGenerator(() -> RandomGeoGenerator.randomPoint(ESTestCase.random()));
}
}

View file

@ -9,9 +9,12 @@
package org.elasticsearch.logsdb.datageneration.datasource;
import org.elasticsearch.core.Tuple;
import org.elasticsearch.test.ESTestCase;
import java.util.ArrayList;
import java.util.HashSet;
import java.util.List;
import java.util.function.Function;
import java.util.function.Supplier;
import java.util.stream.IntStream;
@ -42,6 +45,11 @@ public class DefaultWrappersHandler implements DataSourceHandler {
return new DataSourceResponse.TransformWrapper(transform(request.transformedProportion(), request.transformation()));
}
@Override
public DataSourceResponse.TransformWeightedWrapper handle(DataSourceRequest.TransformWeightedWrapper<?> request) {
return new DataSourceResponse.TransformWeightedWrapper(transformWeighted(request.transformations()));
}
private static Function<Supplier<Object>, Supplier<Object>> injectNulls() {
// Inject some nulls but majority of data should be non-null (as it likely is in reality).
return transform(0.05, ignored -> null);
@ -83,4 +91,36 @@ public class DefaultWrappersHandler implements DataSourceHandler {
) {
return (values) -> () -> ESTestCase.randomDouble() <= transformedProportion ? transformation.apply(values.get()) : values.get();
}
@SuppressWarnings("unchecked")
public static <T> Function<Supplier<Object>, Supplier<Object>> transformWeighted(
List<Tuple<Double, Function<T, Object>>> transformations
) {
double totalWeight = transformations.stream().mapToDouble(Tuple::v1).sum();
if (totalWeight != 1.0) {
throw new IllegalArgumentException("Sum of weights must be equal to 1");
}
List<Tuple<Double, Double>> lookup = new ArrayList<>();
Double leftBound = 0d;
for (var tuple : transformations) {
lookup.add(Tuple.tuple(leftBound, leftBound + tuple.v1()));
leftBound += tuple.v1();
}
return values -> {
var roll = ESTestCase.randomDouble();
for (int i = 0; i < lookup.size(); i++) {
var bounds = lookup.get(i);
if (roll >= bounds.v1() && roll <= bounds.v2()) {
var transformation = transformations.get(i).v2();
return () -> transformation.apply((T) values.get());
}
}
assert false : "Should not get here if weights add up to 1";
return null;
};
}
}

View file

@ -0,0 +1,64 @@
/*
* 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", the "GNU Affero General Public License v3.0 only", 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", the "GNU Affero General Public
* License v3.0 only", or the "Server Side Public License, v 1".
*/
package org.elasticsearch.logsdb.datageneration.fields.leaf;
import org.elasticsearch.common.geo.GeoPoint;
import org.elasticsearch.core.Tuple;
import org.elasticsearch.logsdb.datageneration.FieldDataGenerator;
import org.elasticsearch.logsdb.datageneration.datasource.DataSource;
import org.elasticsearch.logsdb.datageneration.datasource.DataSourceRequest;
import java.util.List;
import java.util.Map;
import java.util.function.Supplier;
public class GeoPointFieldDataGenerator implements FieldDataGenerator {
private final Supplier<Object> formattedPoints;
private final Supplier<Object> formattedPointsWithMalformed;
public GeoPointFieldDataGenerator(DataSource dataSource) {
var points = dataSource.get(new DataSourceRequest.GeoPointGenerator()).generator();
var representations = dataSource.get(
new DataSourceRequest.TransformWeightedWrapper<GeoPoint>(
List.of(
Tuple.tuple(0.2, p -> Map.of("type", "point", "coordinates", List.of(p.getLon(), p.getLat()))),
Tuple.tuple(0.2, p -> "POINT( " + p.getLon() + " " + p.getLat() + " )"),
Tuple.tuple(0.2, p -> Map.of("lon", p.getLon(), "lat", p.getLat())),
// this triggers a bug in stored source block loader, see #125710
// Tuple.tuple(0.2, p -> List.of(p.getLon(), p.getLat())),
Tuple.tuple(0.2, p -> p.getLat() + "," + p.getLon()),
Tuple.tuple(0.2, GeoPoint::getGeohash)
)
)
);
var pointRepresentations = representations.wrapper().apply(points);
this.formattedPoints = Wrappers.defaults(pointRepresentations, dataSource);
var strings = dataSource.get(new DataSourceRequest.StringGenerator()).generator();
this.formattedPointsWithMalformed = Wrappers.defaultsWithMalformed(pointRepresentations, strings::get, dataSource);
}
@Override
public Object generateValue(Map<String, Object> fieldMapping) {
if (fieldMapping == null) {
// dynamically mapped and dynamic mapping does not play well with this type (it sometimes gets mapped as an object)
// return null to skip indexing this field
return null;
}
if ((Boolean) fieldMapping.getOrDefault("ignore_malformed", false)) {
return formattedPointsWithMalformed.get();
}
return formattedPoints.get();
}
}

View file

@ -10,6 +10,7 @@
package org.elasticsearch.logsdb.datageneration.matchers.source;
import org.apache.lucene.sandbox.document.HalfFloatPoint;
import org.elasticsearch.common.geo.GeoPoint;
import org.elasticsearch.common.settings.Settings;
import org.elasticsearch.index.mapper.DateFieldMapper;
import org.elasticsearch.logsdb.datageneration.matchers.MatchResult;
@ -20,6 +21,7 @@ import java.time.Instant;
import java.time.ZoneId;
import java.time.ZoneOffset;
import java.time.format.DateTimeFormatter;
import java.util.HashMap;
import java.util.List;
import java.util.Locale;
import java.util.Map;
@ -35,6 +37,34 @@ import static org.elasticsearch.logsdb.datageneration.matchers.Messages.prettyPr
interface FieldSpecificMatcher {
MatchResult match(List<Object> actual, List<Object> expected, Map<String, Object> actualMapping, Map<String, Object> expectedMapping);
static Map<String, FieldSpecificMatcher> matchers(
XContentBuilder actualMappings,
Settings.Builder actualSettings,
XContentBuilder expectedMappings,
Settings.Builder expectedSettings
) {
return new HashMap<>() {
{
put("keyword", new KeywordMatcher(actualMappings, actualSettings, expectedMappings, expectedSettings));
put("date", new DateMatcher(actualMappings, actualSettings, expectedMappings, expectedSettings));
put("long", new NumberMatcher("long", actualMappings, actualSettings, expectedMappings, expectedSettings));
put("unsigned_long", new UnsignedLongMatcher(actualMappings, actualSettings, expectedMappings, expectedSettings));
put("integer", new NumberMatcher("integer", actualMappings, actualSettings, expectedMappings, expectedSettings));
put("short", new NumberMatcher("short", actualMappings, actualSettings, expectedMappings, expectedSettings));
put("byte", new NumberMatcher("byte", actualMappings, actualSettings, expectedMappings, expectedSettings));
put("double", new NumberMatcher("double", actualMappings, actualSettings, expectedMappings, expectedSettings));
put("float", new NumberMatcher("float", actualMappings, actualSettings, expectedMappings, expectedSettings));
put("half_float", new HalfFloatMatcher(actualMappings, actualSettings, expectedMappings, expectedSettings));
put("scaled_float", new ScaledFloatMatcher(actualMappings, actualSettings, expectedMappings, expectedSettings));
put("counted_keyword", new CountedKeywordMatcher(actualMappings, actualSettings, expectedMappings, expectedSettings));
put("boolean", new BooleanMatcher(actualMappings, actualSettings, expectedMappings, expectedSettings));
put("geo_shape", new ExactMatcher("geo_shape", actualMappings, actualSettings, expectedMappings, expectedSettings));
put("shape", new ExactMatcher("shape", actualMappings, actualSettings, expectedMappings, expectedSettings));
put("geo_point", new GeoPointMatcher(actualMappings, actualSettings, expectedMappings, expectedSettings));
}
};
}
class CountedKeywordMatcher implements FieldSpecificMatcher {
private final XContentBuilder actualMappings;
private final Settings.Builder actualSettings;
@ -165,12 +195,12 @@ interface FieldSpecificMatcher {
Map<String, Object> actualMapping,
Map<String, Object> expectedMapping
) {
var scalingFactor = FieldSpecificMatcher.getMappingParameter("scaling_factor", actualMapping, expectedMapping);
var scalingFactor = getMappingParameter("scaling_factor", actualMapping, expectedMapping);
assert scalingFactor instanceof Number;
double scalingFactorDouble = ((Number) scalingFactor).doubleValue();
var nullValue = (Number) FieldSpecificMatcher.getNullValue(actualMapping, expectedMapping);
var nullValue = (Number) getNullValue(actualMapping, expectedMapping);
// It is possible that we receive a mix of reduced precision values and original values.
// F.e. in case of `synthetic_source_keep: "arrays"` in nested objects only arrays are preserved as is
@ -473,18 +503,70 @@ interface FieldSpecificMatcher {
}
}
class ShapeMatcher implements FieldSpecificMatcher {
private final XContentBuilder actualMappings;
private final Settings.Builder actualSettings;
private final XContentBuilder expectedMappings;
private final Settings.Builder expectedSettings;
ShapeMatcher(
class GeoPointMatcher extends GenericMappingAwareMatcher {
GeoPointMatcher(
XContentBuilder actualMappings,
Settings.Builder actualSettings,
XContentBuilder expectedMappings,
Settings.Builder expectedSettings
) {
super("geo_point", actualMappings, actualSettings, expectedMappings, expectedSettings);
}
@Override
@SuppressWarnings("unchecked")
Object convert(Object value, Object nullValue) {
if (value == null) {
if (nullValue != null) {
return normalizePoint(new GeoPoint((String) nullValue));
}
return null;
}
if (value instanceof String s) {
try {
return normalizePoint(new GeoPoint(s));
} catch (Exception e) {
// malformed
return value;
}
}
if (value instanceof Map<?, ?> m) {
if (m.get("type") != null) {
var coordinates = (List<Double>) m.get("coordinates");
// Order in GeoJSON is lon,lat
return normalizePoint(new GeoPoint(coordinates.get(1), coordinates.get(0)));
} else {
return normalizePoint(new GeoPoint((Double) m.get("lat"), (Double) m.get("lon")));
}
}
if (value instanceof List<?> l) {
// Order in arrays is lon,lat
return normalizePoint(new GeoPoint((Double) l.get(1), (Double) l.get(0)));
}
return value;
}
private static GeoPoint normalizePoint(GeoPoint point) {
return point.resetFromEncoded(point.getEncoded());
}
}
class ExactMatcher implements FieldSpecificMatcher {
private final String fieldType;
private final XContentBuilder actualMappings;
private final Settings.Builder actualSettings;
private final XContentBuilder expectedMappings;
private final Settings.Builder expectedSettings;
ExactMatcher(
String fieldType,
XContentBuilder actualMappings,
Settings.Builder actualSettings,
XContentBuilder expectedMappings,
Settings.Builder expectedSettings
) {
this.fieldType = fieldType;
this.actualMappings = actualMappings;
this.actualSettings = actualSettings;
this.expectedMappings = expectedMappings;
@ -498,7 +580,6 @@ interface FieldSpecificMatcher {
Map<String, Object> actualMapping,
Map<String, Object> expectedMapping
) {
// Since fallback synthetic source is used, should always match exactly.
return actual.equals(expected)
? MatchResult.match()
: MatchResult.noMatch(
@ -507,7 +588,11 @@ interface FieldSpecificMatcher {
actualSettings,
expectedMappings,
expectedSettings,
"Values of type [geo_shape] don't match, values " + prettyPrintCollections(actual, expected)
"Values of type ["
+ fieldType
+ "] were expected to match exactly "
+ "but don't match, values "
+ prettyPrintCollections(actual, expected)
)
);
}

View file

@ -16,7 +16,6 @@ import org.elasticsearch.logsdb.datageneration.matchers.GenericEqualsMatcher;
import org.elasticsearch.logsdb.datageneration.matchers.MatchResult;
import org.elasticsearch.xcontent.XContentBuilder;
import java.util.HashMap;
import java.util.List;
import java.util.Map;
import java.util.Objects;
@ -55,55 +54,7 @@ public class SourceMatcher extends GenericEqualsMatcher<List<Map<String, Object>
.v2();
this.expectedNormalizedMapping = MappingTransforms.normalizeMapping(expectedMappingAsMap);
this.fieldSpecificMatchers = new HashMap<>() {
{
put("keyword", new FieldSpecificMatcher.KeywordMatcher(actualMappings, actualSettings, expectedMappings, expectedSettings));
put("date", new FieldSpecificMatcher.DateMatcher(actualMappings, actualSettings, expectedMappings, expectedSettings));
put(
"long",
new FieldSpecificMatcher.NumberMatcher("long", actualMappings, actualSettings, expectedMappings, expectedSettings)
);
put(
"unsigned_long",
new FieldSpecificMatcher.UnsignedLongMatcher(actualMappings, actualSettings, expectedMappings, expectedSettings)
);
put(
"integer",
new FieldSpecificMatcher.NumberMatcher("integer", actualMappings, actualSettings, expectedMappings, expectedSettings)
);
put(
"short",
new FieldSpecificMatcher.NumberMatcher("short", actualMappings, actualSettings, expectedMappings, expectedSettings)
);
put(
"byte",
new FieldSpecificMatcher.NumberMatcher("byte", actualMappings, actualSettings, expectedMappings, expectedSettings)
);
put(
"double",
new FieldSpecificMatcher.NumberMatcher("double", actualMappings, actualSettings, expectedMappings, expectedSettings)
);
put(
"float",
new FieldSpecificMatcher.NumberMatcher("float", actualMappings, actualSettings, expectedMappings, expectedSettings)
);
put(
"half_float",
new FieldSpecificMatcher.HalfFloatMatcher(actualMappings, actualSettings, expectedMappings, expectedSettings)
);
put(
"scaled_float",
new FieldSpecificMatcher.ScaledFloatMatcher(actualMappings, actualSettings, expectedMappings, expectedSettings)
);
put(
"counted_keyword",
new FieldSpecificMatcher.CountedKeywordMatcher(actualMappings, actualSettings, expectedMappings, expectedSettings)
);
put("boolean", new FieldSpecificMatcher.BooleanMatcher(actualMappings, actualSettings, expectedMappings, expectedSettings));
put("geo_shape", new FieldSpecificMatcher.ShapeMatcher(actualMappings, actualSettings, expectedMappings, expectedSettings));
put("shape", new FieldSpecificMatcher.ShapeMatcher(actualMappings, actualSettings, expectedMappings, expectedSettings));
}
};
this.fieldSpecificMatchers = FieldSpecificMatcher.matchers(actualMappings, actualSettings, expectedMappings, expectedSettings);
this.dynamicFieldMatcher = new DynamicFieldMatcher(actualMappings, actualSettings, expectedMappings, expectedSettings);
}

View file

@ -26,6 +26,8 @@ import org.elasticsearch.geometry.Point;
import org.elasticsearch.index.fielddata.FieldDataContext;
import org.elasticsearch.index.fielddata.IndexFieldData;
import org.elasticsearch.index.mapper.AbstractPointGeometryFieldMapper;
import org.elasticsearch.index.mapper.BlockDocValuesReader;
import org.elasticsearch.index.mapper.BlockLoader;
import org.elasticsearch.index.mapper.DocumentParserContext;
import org.elasticsearch.index.mapper.FieldMapper;
import org.elasticsearch.index.mapper.MappedFieldType;
@ -45,6 +47,8 @@ import java.util.List;
import java.util.Map;
import java.util.function.Function;
import static org.elasticsearch.index.mapper.MappedFieldType.FieldExtractPreference.DOC_VALUES;
/**
* Field Mapper for point type.
*
@ -124,6 +128,7 @@ public class PointFieldMapper extends AbstractPointGeometryFieldMapper<Cartesian
hasDocValues.get(),
parser,
nullValue.get(),
context.isSourceSynthetic(),
meta.get()
);
return new PointFieldMapper(leafName(), ft, builderParams(this, context), parser, this);
@ -187,6 +192,7 @@ public class PointFieldMapper extends AbstractPointGeometryFieldMapper<Cartesian
}
public static class PointFieldType extends AbstractPointFieldType<CartesianPoint> implements ShapeQueryable {
private final boolean isSyntheticSource;
private PointFieldType(
String name,
@ -195,14 +201,16 @@ public class PointFieldMapper extends AbstractPointGeometryFieldMapper<Cartesian
boolean hasDocValues,
CartesianPointParser parser,
CartesianPoint nullValue,
boolean isSyntheticSource,
Map<String, String> meta
) {
super(name, indexed, stored, hasDocValues, parser, nullValue, meta);
this.isSyntheticSource = isSyntheticSource;
}
// only used in test
public PointFieldType(String name) {
this(name, true, false, true, null, null, Collections.emptyMap());
this(name, true, false, true, null, null, false, Collections.emptyMap());
}
@Override
@ -230,6 +238,19 @@ public class PointFieldMapper extends AbstractPointGeometryFieldMapper<Cartesian
protected Function<List<CartesianPoint>, List<Object>> getFormatter(String format) {
return GeometryFormatterFactory.getFormatter(format, p -> new Point(p.getX(), p.getY()));
}
@Override
public BlockLoader blockLoader(BlockLoaderContext blContext) {
if (blContext.fieldExtractPreference() == DOC_VALUES && hasDocValues()) {
return new BlockDocValuesReader.LongsBlockLoader(name());
}
if (isSyntheticSource) {
return blockLoaderFromFallbackSyntheticSource(blContext);
}
return blockLoaderFromSource(blContext);
}
}
/** CartesianPoint parser implementation */

View file

@ -48,7 +48,7 @@ public class GeoShapeFieldDataGenerator implements FieldDataGenerator {
return null;
}
if (fieldMapping != null && (Boolean) fieldMapping.getOrDefault("ignore_malformed", false)) {
if ((Boolean) fieldMapping.getOrDefault("ignore_malformed", false)) {
return formattedGeoShapesWithMalformed.get();
}

View file

@ -0,0 +1,65 @@
/*
* 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; you may not use this file except in compliance with the Elastic License
* 2.0.
*/
package org.elasticsearch.xpack.spatial.datageneration;
import org.elasticsearch.geo.GeometryTestUtils;
import org.elasticsearch.logsdb.datageneration.datasource.DataSourceHandler;
import org.elasticsearch.logsdb.datageneration.datasource.DataSourceRequest;
import org.elasticsearch.logsdb.datageneration.datasource.DataSourceResponse;
import org.elasticsearch.test.ESTestCase;
import org.elasticsearch.xpack.spatial.common.CartesianPoint;
import java.util.HashMap;
import java.util.Map;
public class PointDataSourceHandler implements DataSourceHandler {
@Override
public DataSourceResponse.PointGenerator handle(DataSourceRequest.PointGenerator request) {
return new DataSourceResponse.PointGenerator(this::generateValidPoint);
}
@Override
public DataSourceResponse.LeafMappingParametersGenerator handle(DataSourceRequest.LeafMappingParametersGenerator request) {
if (request.fieldType().equals("point") == false) {
return null;
}
return new DataSourceResponse.LeafMappingParametersGenerator(() -> {
var map = new HashMap<String, Object>();
map.put("index", ESTestCase.randomBoolean());
map.put("doc_values", ESTestCase.randomBoolean());
map.put("store", ESTestCase.randomBoolean());
if (ESTestCase.randomBoolean()) {
map.put("ignore_malformed", ESTestCase.randomBoolean());
}
if (ESTestCase.randomDouble() <= 0.2) {
var point = generateValidPoint();
map.put("null_value", Map.of("x", point.getX(), "y", point.getY()));
}
return map;
});
}
@Override
public DataSourceResponse.FieldDataGenerator handle(DataSourceRequest.FieldDataGenerator request) {
if (request.fieldType().equals("point") == false) {
return null;
}
return new DataSourceResponse.FieldDataGenerator(new PointFieldDataGenerator(request.dataSource()));
}
private CartesianPoint generateValidPoint() {
var point = GeometryTestUtils.randomPoint(false);
return new CartesianPoint(point.getLat(), point.getLon());
}
}

View file

@ -0,0 +1,62 @@
/*
* 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; you may not use this file except in compliance with the Elastic License
* 2.0.
*/
package org.elasticsearch.xpack.spatial.datageneration;
import org.elasticsearch.core.Tuple;
import org.elasticsearch.logsdb.datageneration.FieldDataGenerator;
import org.elasticsearch.logsdb.datageneration.datasource.DataSource;
import org.elasticsearch.logsdb.datageneration.datasource.DataSourceRequest;
import org.elasticsearch.logsdb.datageneration.fields.leaf.Wrappers;
import org.elasticsearch.xpack.spatial.common.CartesianPoint;
import java.util.List;
import java.util.Map;
import java.util.function.Supplier;
public class PointFieldDataGenerator implements FieldDataGenerator {
private final Supplier<Object> formattedPoints;
private final Supplier<Object> formattedPointsWithMalformed;
public PointFieldDataGenerator(DataSource dataSource) {
var points = dataSource.get(new DataSourceRequest.PointGenerator()).generator();
var representations = dataSource.get(
new DataSourceRequest.TransformWeightedWrapper<CartesianPoint>(
List.of(
Tuple.tuple(0.25, cp -> Map.of("type", "point", "coordinates", List.of(cp.getX(), cp.getY()))),
Tuple.tuple(0.25, cp -> "POINT( " + cp.getX() + " " + cp.getY() + " )"),
Tuple.tuple(0.25, cp -> Map.of("x", cp.getX(), "y", cp.getY())),
// this triggers a bug in stored source block loader, see #125710
// Tuple.tuple(0.2, cp -> List.of(cp.getX(), cp.getY())),
Tuple.tuple(0.25, cp -> cp.getX() + "," + cp.getY())
)
)
);
var pointRepresentations = representations.wrapper().apply(points);
this.formattedPoints = Wrappers.defaults(pointRepresentations, dataSource);
var strings = dataSource.get(new DataSourceRequest.StringGenerator()).generator();
this.formattedPointsWithMalformed = Wrappers.defaultsWithMalformed(pointRepresentations, strings::get, dataSource);
}
@Override
public Object generateValue(Map<String, Object> fieldMapping) {
if (fieldMapping == null) {
// dynamically mapped and dynamic mapping does not play well with this type (it sometimes gets mapped as an object)
// return null to skip indexing this field
return null;
}
if ((Boolean) fieldMapping.getOrDefault("ignore_malformed", false)) {
return formattedPointsWithMalformed.get();
}
return formattedPoints.get();
}
}

View file

@ -38,7 +38,7 @@ public class GeoShapeFieldBlockLoaderTests extends BlockLoaderTestCase {
@Override
@SuppressWarnings("unchecked")
protected Object expected(Map<String, Object> fieldMapping, Object value) {
protected Object expected(Map<String, Object> fieldMapping, Object value, TestContext testContext) {
if (value instanceof List<?> == false) {
return convert(value);
}

View file

@ -0,0 +1,178 @@
/*
* 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; you may not use this file except in compliance with the Elastic License
* 2.0.
*/
package org.elasticsearch.xpack.spatial.index.mapper;
import org.apache.lucene.document.XYDocValuesField;
import org.apache.lucene.util.BytesRef;
import org.elasticsearch.geometry.Point;
import org.elasticsearch.geometry.utils.WellKnownBinary;
import org.elasticsearch.index.mapper.BlockLoaderTestCase;
import org.elasticsearch.index.mapper.MappedFieldType;
import org.elasticsearch.plugins.ExtensiblePlugin;
import org.elasticsearch.plugins.Plugin;
import org.elasticsearch.xpack.spatial.LocalStateSpatialPlugin;
import org.elasticsearch.xpack.spatial.common.CartesianPoint;
import org.elasticsearch.xpack.spatial.datageneration.PointDataSourceHandler;
import java.nio.ByteOrder;
import java.util.ArrayList;
import java.util.Collection;
import java.util.Collections;
import java.util.List;
import java.util.Map;
import java.util.Objects;
public class PointFieldBlockLoaderTests extends BlockLoaderTestCase {
public PointFieldBlockLoaderTests(Params params) {
super("point", List.of(new PointDataSourceHandler()), params);
}
@Override
@SuppressWarnings("unchecked")
protected Object expected(Map<String, Object> fieldMapping, Object value, TestContext testContext) {
var nullValue = switch (fieldMapping.get("null_value")) {
case Map<?, ?> m -> convert(m, null);
case null -> null;
default -> throw new IllegalStateException("Unexpected null_value format");
};
if (params.preference() == MappedFieldType.FieldExtractPreference.DOC_VALUES && hasDocValues(fieldMapping, true)) {
if (value instanceof List<?> == false) {
return encode(convert(value, nullValue));
}
var resultList = ((List<Object>) value).stream()
.map(v -> convert(v, nullValue))
.filter(Objects::nonNull)
.map(this::encode)
.sorted()
.toList();
return maybeFoldList(resultList);
}
if (value instanceof List<?> == false) {
return toWKB(convert(value, nullValue));
}
// As a result we always load from source (stored or fallback synthetic) and they should work the same.
var resultList = ((List<Object>) value).stream().map(v -> convert(v, nullValue)).filter(Objects::nonNull).map(this::toWKB).toList();
return maybeFoldList(resultList);
}
@Override
protected Object getFieldValue(Map<String, Object> document, String fieldName) {
var extracted = new ArrayList<>();
processLevel(document, fieldName, extracted);
if (extracted.size() == 1) {
return extracted.get(0);
}
return extracted;
}
@SuppressWarnings("unchecked")
private void processLevel(Map<String, Object> level, String field, ArrayList<Object> extracted) {
if (field.contains(".") == false) {
var value = level.get(field);
processLeafLevel(value, extracted);
return;
}
var nameInLevel = field.split("\\.")[0];
var entry = level.get(nameInLevel);
if (entry instanceof Map<?, ?> m) {
processLevel((Map<String, Object>) m, field.substring(field.indexOf('.') + 1), extracted);
}
if (entry instanceof List<?> l) {
for (var object : l) {
processLevel((Map<String, Object>) object, field.substring(field.indexOf('.') + 1), extracted);
}
}
}
private void processLeafLevel(Object value, ArrayList<Object> extracted) {
if (value instanceof List<?> l) {
if (l.size() > 0 && l.get(0) instanceof Double) {
// this must be a single point in array form
// we'll put it into a different form here to make our lives a bit easier while implementing `expected`
extracted.add(Map.of("type", "point", "coordinates", l));
} else {
// this is actually an array of points but there could still be points in array form inside
for (var arrayValue : l) {
processLeafLevel(arrayValue, extracted);
}
}
} else {
extracted.add(value);
}
}
@SuppressWarnings("unchecked")
private CartesianPoint convert(Object value, CartesianPoint nullValue) {
if (value == null) {
return nullValue;
}
var point = new CartesianPoint();
if (value instanceof String s) {
try {
point.resetFromString(s, true);
return point;
} catch (Exception e) {
return null;
}
}
if (value instanceof Map<?, ?> m) {
if (m.get("type") != null) {
var coordinates = (List<Double>) m.get("coordinates");
point.reset(coordinates.get(0), coordinates.get(1));
} else {
point.reset((Double) m.get("x"), (Double) m.get("y"));
}
return point;
}
if (value instanceof List<?> l) {
point.reset((Double) l.get(0), (Double) l.get(1));
return point;
}
// Malformed values are excluded
return null;
}
private Long encode(CartesianPoint point) {
if (point == null) {
return null;
}
return new XYDocValuesField("f", (float) point.getX(), (float) point.getY()).numericValue().longValue();
}
private BytesRef toWKB(CartesianPoint cartesianPoint) {
if (cartesianPoint == null) {
return null;
}
return new BytesRef(WellKnownBinary.toWKB(new Point(cartesianPoint.getX(), cartesianPoint.getY()), ByteOrder.LITTLE_ENDIAN));
}
@Override
protected Collection<? extends Plugin> getPlugins() {
var plugin = new LocalStateSpatialPlugin();
plugin.loadExtensions(new ExtensiblePlugin.ExtensionLoader() {
@Override
public <T> List<T> loadExtensions(Class<T> extensionPointType) {
return List.of();
}
});
return Collections.singletonList(plugin);
}
}

View file

@ -36,7 +36,7 @@ public class ShapeFieldBlockLoaderTests extends BlockLoaderTestCase {
@Override
@SuppressWarnings("unchecked")
protected Object expected(Map<String, Object> fieldMapping, Object value) {
protected Object expected(Map<String, Object> fieldMapping, Object value, TestContext testContext) {
if (value instanceof List<?> == false) {
return convert(value);
}