You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@pinot.apache.org by xi...@apache.org on 2021/01/27 07:02:55 UTC

[incubator-pinot] branch master updated: Adding scalar function JsonPathArray to extract arrays from json (#6490)

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

xiangfu pushed a commit to branch master
in repository https://gitbox.apache.org/repos/asf/incubator-pinot.git


The following commit(s) were added to refs/heads/master by this push:
     new fde307b  Adding scalar function JsonPathArray to extract arrays from json (#6490)
fde307b is described below

commit fde307bdff25d59738dbd39ed0e8ef4ef84372cf
Author: Xiang Fu <fx...@gmail.com>
AuthorDate: Tue Jan 26 23:02:40 2021 -0800

    Adding scalar function JsonPathArray to extract arrays from json (#6490)
---
 .../common/function/scalar/JsonFunctions.java      | 55 ++++++++++++++
 .../pinot/common/function/JsonFunctionsTest.java   | 86 ++++++++++++++++++++++
 .../core/data/function/InbuiltFunctionsTest.java   | 37 ++++++++++
 .../tests/JsonPathClusterIntegrationTest.java      | 80 +++++++++++++++++---
 4 files changed, 248 insertions(+), 10 deletions(-)

diff --git a/pinot-common/src/main/java/org/apache/pinot/common/function/scalar/JsonFunctions.java b/pinot-common/src/main/java/org/apache/pinot/common/function/scalar/JsonFunctions.java
index 3a9f0f9..37027ff 100644
--- a/pinot-common/src/main/java/org/apache/pinot/common/function/scalar/JsonFunctions.java
+++ b/pinot-common/src/main/java/org/apache/pinot/common/function/scalar/JsonFunctions.java
@@ -19,8 +19,17 @@
 package org.apache.pinot.common.function.scalar;
 
 import com.fasterxml.jackson.core.JsonProcessingException;
+import com.jayway.jsonpath.Configuration;
 import com.jayway.jsonpath.JsonPath;
+import com.jayway.jsonpath.Option;
+import com.jayway.jsonpath.spi.json.JacksonJsonProvider;
+import com.jayway.jsonpath.spi.json.JsonProvider;
+import com.jayway.jsonpath.spi.mapper.JacksonMappingProvider;
+import com.jayway.jsonpath.spi.mapper.MappingProvider;
+import java.util.EnumSet;
+import java.util.List;
 import java.util.Map;
+import java.util.Set;
 import org.apache.pinot.spi.annotations.ScalarFunction;
 import org.apache.pinot.spi.utils.JsonUtils;
 
@@ -37,6 +46,28 @@ import org.apache.pinot.spi.utils.JsonUtils;
  *   </code>
  */
 public class JsonFunctions {
+  static {
+    Configuration.setDefaults(new Configuration.Defaults() {
+      private final JsonProvider jsonProvider = new JacksonJsonProvider();
+      private final MappingProvider mappingProvider = new JacksonMappingProvider();
+
+      @Override
+      public JsonProvider jsonProvider() {
+        return jsonProvider;
+      }
+
+      @Override
+      public MappingProvider mappingProvider() {
+        return mappingProvider;
+      }
+
+      @Override
+      public Set<Option> options() {
+        return EnumSet.noneOf(Option.class);
+      }
+    });
+  }
+
   private JsonFunctions() {
   }
 
@@ -70,6 +101,30 @@ public class JsonFunctions {
   }
 
   /**
+   * Extract object array based on Json path
+   */
+  @ScalarFunction
+  public static Object[] jsonPathArray(Object object, String jsonPath)
+      throws JsonProcessingException {
+    if (object instanceof String) {
+      return convertObjectToArray(JsonPath.read((String) object, jsonPath));
+    }
+    if (object instanceof Object[]) {
+      return convertObjectToArray(JsonPath.read(JsonUtils.objectToString(object), jsonPath));
+    }
+    return convertObjectToArray(JsonPath.read(object, jsonPath));
+  }
+
+  private static Object[] convertObjectToArray(Object arrayObject) {
+    if (arrayObject instanceof List) {
+      return ((List) arrayObject).toArray();
+    } else if (arrayObject instanceof Object[]) {
+      return (Object[]) arrayObject;
+    }
+    return new Object[]{arrayObject};
+  }
+
+  /**
    * Extract from Json with path to String
    */
   @ScalarFunction
diff --git a/pinot-common/src/test/java/org/apache/pinot/common/function/JsonFunctionsTest.java b/pinot-common/src/test/java/org/apache/pinot/common/function/JsonFunctionsTest.java
index f11f256..ff09a87 100644
--- a/pinot-common/src/test/java/org/apache/pinot/common/function/JsonFunctionsTest.java
+++ b/pinot-common/src/test/java/org/apache/pinot/common/function/JsonFunctionsTest.java
@@ -19,7 +19,13 @@
 package org.apache.pinot.common.function;
 
 import com.fasterxml.jackson.core.JsonProcessingException;
+import com.google.common.collect.ImmutableMap;
 import com.jayway.jsonpath.JsonPath;
+import java.util.ArrayList;
+import java.util.Arrays;
+import java.util.Collections;
+import java.util.List;
+import java.util.Map;
 import org.apache.pinot.common.function.scalar.JsonFunctions;
 import org.testng.annotations.Test;
 
@@ -65,4 +71,84 @@ public class JsonFunctionsTest {
     assertEquals(JsonFunctions.jsonPathDouble(jsonString, "$.actor.aaa", 53.2), 53.2);
   }
 
+  @Test
+  public void testJsonFunctionExtractingArray()
+      throws JsonProcessingException {
+    String jsonString = "{\n" +
+        "    \"name\": \"Pete\",\n" +
+        "    \"age\": 24,\n" +
+        "    \"subjects\": [\n" +
+        "        {\n" +
+        "            \"name\": \"maths\",\n" +
+        "            \"homework_grades\": [80, 85, 90, 95, 100],\n" +
+        "            \"grade\": \"A\"\n" +
+        "        },\n" +
+        "        {\n" +
+        "            \"name\": \"english\",\n" +
+        "            \"homework_grades\": [60, 65, 70, 85, 90],\n" +
+        "            \"grade\": \"B\"\n" +
+        "        }\n" +
+        "    ]\n" +
+        "}";
+    assertEquals(JsonFunctions.jsonPathArray(jsonString, "$.subjects[*].name"), new String[]{"maths", "english"});
+    assertEquals(JsonFunctions.jsonPathArray(jsonString, "$.subjects[*].grade"), new String[]{"A", "B"});
+    assertEquals(JsonFunctions.jsonPathArray(jsonString, "$.subjects[*].homework_grades"),
+        new Object[]{Arrays.asList(80, 85, 90, 95, 100), Arrays.asList(60, 65, 70, 85, 90)});
+  }
+
+  @Test
+  public void testJsonFunctionOnJsonArray()
+      throws JsonProcessingException {
+    String jsonArrayString =
+        "[\n" +
+        "        {\n" +
+        "            \"name\": \"maths\",\n" +
+        "            \"grade\": \"A\",\n" +
+        "            \"homework_grades\": [80, 85, 90, 95, 100],\n" +
+        "            \"score\": 90\n" +
+        "        },\n" +
+        "        {\n" +
+        "            \"name\": \"english\",\n" +
+        "            \"grade\": \"B\",\n" +
+        "            \"homework_grades\": [60, 65, 70, 85, 90],\n" +
+        "            \"score\": 50\n" +
+        "        }\n" +
+        "]";
+    assertEquals(JsonFunctions.jsonPathArray(jsonArrayString, "$.[*].name"), new String[]{"maths", "english"});
+    assertEquals(JsonFunctions.jsonPathArray(jsonArrayString, "$.[*].grade"), new String[]{"A", "B"});
+    assertEquals(JsonFunctions.jsonPathArray(jsonArrayString, "$.[*].homework_grades"),
+        new Object[]{Arrays.asList(80, 85, 90, 95, 100), Arrays.asList(60, 65, 70, 85, 90)});
+    assertEquals(JsonFunctions.jsonPathArray(jsonArrayString, "$.[*].score"), new Integer[]{90, 50});
+  }
+
+  @Test
+  public void testJsonFunctionOnList()
+      throws JsonProcessingException {
+    List<Map<String, Object>> rawData = new ArrayList<Map<String, Object>>();
+    rawData.add(ImmutableMap.of("name", "maths", "grade", "A", "score", 90,
+        "homework_grades", Arrays.asList(80, 85, 90, 95, 100)));
+    rawData.add(ImmutableMap.of("name", "english", "grade", "B", "score", 50,
+        "homework_grades", Arrays.asList(60, 65, 70, 85, 90)));
+    assertEquals(JsonFunctions.jsonPathArray(rawData, "$.[*].name"), new String[]{"maths", "english"});
+    assertEquals(JsonFunctions.jsonPathArray(rawData, "$.[*].grade"), new String[]{"A", "B"});
+    assertEquals(JsonFunctions.jsonPathArray(rawData, "$.[*].homework_grades"),
+        new Object[]{Arrays.asList(80, 85, 90, 95, 100), Arrays.asList(60, 65, 70, 85, 90)});
+    assertEquals(JsonFunctions.jsonPathArray(rawData, "$.[*].score"), new Integer[]{90, 50});
+  }
+
+  @Test
+  public void testJsonFunctionOnObjectArray()
+      throws JsonProcessingException {
+    Object[] rawData = new Object[] {
+        ImmutableMap.of("name", "maths", "grade", "A", "score", 90,
+            "homework_grades", Arrays.asList(80, 85, 90, 95, 100)),
+        ImmutableMap.of("name", "english", "grade", "B", "score", 50,
+            "homework_grades", Arrays.asList(60, 65, 70, 85, 90))
+    };
+    assertEquals(JsonFunctions.jsonPathArray(rawData, "$.[*].name"), new String[]{"maths", "english"});
+    assertEquals(JsonFunctions.jsonPathArray(rawData, "$.[*].grade"), new String[]{"A", "B"});
+    assertEquals(JsonFunctions.jsonPathArray(rawData, "$.[*].homework_grades"),
+        new Object[]{Arrays.asList(80, 85, 90, 95, 100), Arrays.asList(60, 65, 70, 85, 90)});
+    assertEquals(JsonFunctions.jsonPathArray(rawData, "$.[*].score"), new Integer[]{90, 50});
+  }
 }
diff --git a/pinot-core/src/test/java/org/apache/pinot/core/data/function/InbuiltFunctionsTest.java b/pinot-core/src/test/java/org/apache/pinot/core/data/function/InbuiltFunctionsTest.java
index 19d6603..9a63583 100644
--- a/pinot-core/src/test/java/org/apache/pinot/core/data/function/InbuiltFunctionsTest.java
+++ b/pinot-core/src/test/java/org/apache/pinot/core/data/function/InbuiltFunctionsTest.java
@@ -339,6 +339,43 @@ public class InbuiltFunctionsTest {
     row5.putValue("jsonMap", JsonUtils.stringToObject(jsonStr, List.class));
     inputs.add(new Object[]{"json_format(jsonMap)", Lists.newArrayList("jsonMap"), row5, jsonStr});
 
+    GenericRow row6 = new GenericRow();
+    jsonStr =
+        "[{\"one\":1,\"two\":{\"sub1\":1.1,\"sub2\":1.2},\"three\":[\"a\",\"b\"]},{\"one\":11,\"two\":{\"sub1\":11.1,\"sub2\":11.2},\"three\":[\"aa\",\"bb\"]}]";
+    row6.putValue("jsonPathArray", JsonUtils.stringToObject(jsonStr, List.class));
+    inputs.add(new Object[]{"json_path_array(jsonPathArray, '$.[*].one')", Lists.newArrayList(
+        "jsonPathArray"), row6, new Object[]{1, 11}});
+
+    GenericRow row7 = new GenericRow();
+    jsonStr =
+        "[{\"one\":1,\"two\":{\"sub1\":1.1,\"sub2\":1.2},\"three\":[\"a\",\"b\"]},{\"one\":11,\"two\":{\"sub1\":11.1,\"sub2\":11.2},\"three\":[\"aa\",\"bb\"]}]";
+    row7.putValue("jsonPathArray", JsonUtils.stringToObject(jsonStr, List.class));
+    inputs.add(new Object[]{"json_path_array(jsonPathArray, '$.[*].three')", Lists.newArrayList(
+        "jsonPathArray"), row7, new Object[]{Arrays.asList("a", "b"), Arrays.asList("aa", "bb")}});
+
+    GenericRow row8 = new GenericRow();
+    jsonStr = "{\"k3\":{\"sub1\":10,\"sub2\":1.0},\"k4\":\"baz\",\"k5\":[1,2,3]}";
+    row8.putValue("jsonPathString", JsonUtils.stringToObject(jsonStr, Map.class));
+    inputs.add(new Object[]{"json_path_string(jsonPathString, '$.k3')", Lists.newArrayList(
+        "jsonPathString"), row8, "{\"sub1\":10,\"sub2\":1.0}"});
+
+    GenericRow row9 = new GenericRow();
+    jsonStr = "{\"k3\":{\"sub1\":10,\"sub2\":1.0},\"k4\":\"baz\",\"k5\":[1,2,3]}";
+    row9.putValue("jsonPathString", JsonUtils.stringToObject(jsonStr, Map.class));
+    inputs.add(new Object[]{"json_path_string(jsonPathString, '$.k4')", Lists.newArrayList(
+        "jsonPathString"), row9, "baz"});
+
+    GenericRow row10 = new GenericRow();
+    jsonStr = "{\"k3\":{\"sub1\":10,\"sub2\":1.0},\"k4\":\"baz\",\"k5\":[1,2,3]}";
+    row10.putValue("jsonPathString", JsonUtils.stringToObject(jsonStr, Map.class));
+    inputs.add(new Object[]{"json_path_long(jsonPathString, '$.k3.sub1')", Lists.newArrayList(
+        "jsonPathString"), row10, 10L});
+
+    GenericRow row11 = new GenericRow();
+    jsonStr = "{\"k3\":{\"sub1\":10,\"sub2\":1.0},\"k4\":\"baz\",\"k5\":[1,2,3]}";
+    row11.putValue("jsonPathString", JsonUtils.stringToObject(jsonStr, Map.class));
+    inputs.add(new Object[]{"json_path_double(jsonPathString, '$.k3.sub2')", Lists.newArrayList(
+        "jsonPathString"), row11, 1.0});
     return inputs.toArray(new Object[0][]);
   }
 
diff --git a/pinot-integration-tests/src/test/java/org/apache/pinot/integration/tests/JsonPathClusterIntegrationTest.java b/pinot-integration-tests/src/test/java/org/apache/pinot/integration/tests/JsonPathClusterIntegrationTest.java
index 1bb6018..4769e9c 100644
--- a/pinot-integration-tests/src/test/java/org/apache/pinot/integration/tests/JsonPathClusterIntegrationTest.java
+++ b/pinot-integration-tests/src/test/java/org/apache/pinot/integration/tests/JsonPathClusterIntegrationTest.java
@@ -21,6 +21,7 @@ package org.apache.pinot.integration.tests;
 import com.fasterxml.jackson.databind.JsonNode;
 import com.fasterxml.jackson.databind.node.ArrayNode;
 import com.google.common.collect.ImmutableMap;
+import com.google.common.collect.Lists;
 import java.io.File;
 import java.util.ArrayList;
 import java.util.Arrays;
@@ -36,6 +37,8 @@ import org.apache.avro.generic.GenericDatumWriter;
 import org.apache.commons.io.FileUtils;
 import org.apache.pinot.spi.config.table.TableConfig;
 import org.apache.pinot.spi.config.table.TableType;
+import org.apache.pinot.spi.config.table.ingestion.IngestionConfig;
+import org.apache.pinot.spi.config.table.ingestion.TransformConfig;
 import org.apache.pinot.spi.data.FieldSpec.DataType;
 import org.apache.pinot.spi.data.Schema;
 import org.apache.pinot.spi.utils.JsonUtils;
@@ -50,6 +53,11 @@ import org.testng.annotations.Test;
 public class JsonPathClusterIntegrationTest extends BaseClusterIntegrationTest {
   private static final int NUM_TOTAL_DOCS = 1000;
   private final List<String> sortedSequenceIds = new ArrayList<>(NUM_TOTAL_DOCS);
+  private final String MY_MAP_STR_FIELD_NAME = "myMapStr";
+  private final String MY_MAP_STR_K1_FIELD_NAME = "myMapStr_k1";
+  private final String MY_MAP_STR_K2_FIELD_NAME = "myMapStr_k2";
+  private final String COMPLEX_MAP_STR_FIELD_NAME = "complexMapStr";
+  private final String COMPLEX_MAP_STR_K3_FIELD_NAME = "complexMapStr_k3";
 
   @Override
   protected long getCountStarResult() {
@@ -71,10 +79,19 @@ public class JsonPathClusterIntegrationTest extends BaseClusterIntegrationTest {
     String rawTableName = getTableName();
     Schema schema =
         new Schema.SchemaBuilder().setSchemaName(rawTableName).addSingleValueDimension("myMap", DataType.STRING)
-            .addSingleValueDimension("myMapStr", DataType.STRING)
-            .addSingleValueDimension("complexMapStr", DataType.STRING).build();
+            .addSingleValueDimension(MY_MAP_STR_FIELD_NAME, DataType.STRING)
+            .addSingleValueDimension(MY_MAP_STR_K1_FIELD_NAME, DataType.STRING)
+            .addSingleValueDimension(MY_MAP_STR_K2_FIELD_NAME, DataType.STRING)
+            .addSingleValueDimension(COMPLEX_MAP_STR_FIELD_NAME, DataType.STRING)
+            .addMultiValueDimension(COMPLEX_MAP_STR_K3_FIELD_NAME, DataType.STRING)
+            .build();
     addSchema(schema);
-    TableConfig tableConfig = new TableConfigBuilder(TableType.OFFLINE).setTableName(rawTableName).build();
+    List<TransformConfig> transformConfigs = Lists.newArrayList(
+        new TransformConfig(MY_MAP_STR_K1_FIELD_NAME, "jsonPathString(" + MY_MAP_STR_FIELD_NAME + ", '$.k1')"),
+        new TransformConfig(MY_MAP_STR_K2_FIELD_NAME, "jsonPathString(" + MY_MAP_STR_FIELD_NAME + ", '$.k2')"),
+        new TransformConfig(COMPLEX_MAP_STR_K3_FIELD_NAME, "jsonPathArray(" + COMPLEX_MAP_STR_FIELD_NAME + ", '$.k3')"));
+    TableConfig tableConfig = new TableConfigBuilder(TableType.OFFLINE).setTableName(rawTableName)
+        .setIngestionConfig(new IngestionConfig(null, null, null, transformConfigs)).build();
     addTableConfig(tableConfig);
 
     // Create and upload segments
@@ -89,8 +106,8 @@ public class JsonPathClusterIntegrationTest extends BaseClusterIntegrationTest {
   private File createAvroFile()
       throws Exception {
     org.apache.avro.Schema avroSchema = org.apache.avro.Schema.createRecord("myRecord", null, null, false);
-    List<Field> fields = Arrays.asList(new Field("myMapStr", org.apache.avro.Schema.create(Type.STRING), null, null),
-        new Field("complexMapStr", org.apache.avro.Schema.create(Type.STRING), null, null));
+    List<Field> fields = Arrays.asList(new Field(MY_MAP_STR_FIELD_NAME, org.apache.avro.Schema.create(Type.STRING), null, null),
+        new Field(COMPLEX_MAP_STR_FIELD_NAME, org.apache.avro.Schema.create(Type.STRING), null, null));
     avroSchema.setFields(fields);
 
     File avroFile = new File(_tempDir, "data.avro");
@@ -101,7 +118,7 @@ public class JsonPathClusterIntegrationTest extends BaseClusterIntegrationTest {
         map.put("k1", "value-k1-" + i);
         map.put("k2", "value-k2-" + i);
         GenericData.Record record = new GenericData.Record(avroSchema);
-        record.put("myMapStr", JsonUtils.objectToString(map));
+        record.put(MY_MAP_STR_FIELD_NAME, JsonUtils.objectToString(map));
 
         Map<String, Object> complexMap = new HashMap<>();
         complexMap.put("k1", "value-k1-" + i);
@@ -109,7 +126,7 @@ public class JsonPathClusterIntegrationTest extends BaseClusterIntegrationTest {
         complexMap.put("k3", Arrays.asList("value-k3-0-" + i, "value-k3-1-" + i, "value-k3-2-" + i));
         complexMap.put("k4", ImmutableMap
             .of("k4-k1", "value-k4-k1-" + i, "k4-k2", "value-k4-k2-" + i, "k4-k3", "value-k4-k3-" + i, "met", i));
-        record.put("complexMapStr", JsonUtils.objectToString(complexMap));
+        record.put(COMPLEX_MAP_STR_FIELD_NAME, JsonUtils.objectToString(complexMap));
         fileWriter.append(record);
         sortedSequenceIds.add(String.valueOf(i));
       }
@@ -124,7 +141,7 @@ public class JsonPathClusterIntegrationTest extends BaseClusterIntegrationTest {
       throws Exception {
 
     //Selection Query
-    String pqlQuery = "Select myMapStr from " + DEFAULT_TABLE_NAME;
+    String pqlQuery = "Select " + MY_MAP_STR_FIELD_NAME + " from " + DEFAULT_TABLE_NAME;
     JsonNode pinotResponse = postQuery(pqlQuery);
     ArrayNode selectionResults = (ArrayNode) pinotResponse.get("selectionResults").get("results");
     Assert.assertNotNull(selectionResults);
@@ -145,6 +162,16 @@ public class JsonPathClusterIntegrationTest extends BaseClusterIntegrationTest {
       String value = selectionResults.get(i).get(0).textValue();
       Assert.assertEquals(value, "value-k1-0");
     }
+    pqlQuery = "Select " + MY_MAP_STR_K1_FIELD_NAME + " from " + DEFAULT_TABLE_NAME
+        + "  where " + MY_MAP_STR_K1_FIELD_NAME + " = 'value-k1-0'";
+    pinotResponse = postQuery(pqlQuery);
+    selectionResults = (ArrayNode) pinotResponse.get("selectionResults").get("results");
+    Assert.assertNotNull(selectionResults);
+    Assert.assertTrue(selectionResults.size() > 0);
+    for (int i = 0; i < selectionResults.size(); i++) {
+      String value = selectionResults.get(i).get(0).textValue();
+      Assert.assertEquals(value, "value-k1-0");
+    }
 
     //selection order by
     pqlQuery = "Select jsonExtractScalar(myMapStr,'$.k1','STRING') from " + DEFAULT_TABLE_NAME
@@ -157,6 +184,16 @@ public class JsonPathClusterIntegrationTest extends BaseClusterIntegrationTest {
       String value = selectionResults.get(i).get(0).textValue();
       Assert.assertTrue(value.indexOf("-k1-") > 0);
     }
+    pqlQuery = "Select " + MY_MAP_STR_K1_FIELD_NAME + " from " + DEFAULT_TABLE_NAME
+        + " order by " + MY_MAP_STR_K1_FIELD_NAME;
+    pinotResponse = postQuery(pqlQuery);
+    selectionResults = (ArrayNode) pinotResponse.get("selectionResults").get("results");
+    Assert.assertNotNull(selectionResults);
+    Assert.assertTrue(selectionResults.size() > 0);
+    for (int i = 0; i < selectionResults.size(); i++) {
+      String value = selectionResults.get(i).get(0).textValue();
+      Assert.assertTrue(value.indexOf("-k1-") > 0);
+    }
 
     //Group By Query
     pqlQuery = "Select count(*) from " + DEFAULT_TABLE_NAME + " group by jsonExtractScalar(myMapStr,'$.k1','STRING')";
@@ -166,6 +203,14 @@ public class JsonPathClusterIntegrationTest extends BaseClusterIntegrationTest {
     Assert.assertNotNull(groupByResult);
     Assert.assertTrue(groupByResult.isArray());
     Assert.assertTrue(groupByResult.size() > 0);
+
+    pqlQuery = "Select count(*) from " + DEFAULT_TABLE_NAME + " group by " + MY_MAP_STR_K1_FIELD_NAME;
+    pinotResponse = postQuery(pqlQuery);
+    Assert.assertNotNull(pinotResponse.get("aggregationResults"));
+    groupByResult = pinotResponse.get("aggregationResults").get(0).get("groupByResult");
+    Assert.assertNotNull(groupByResult);
+    Assert.assertTrue(groupByResult.isArray());
+    Assert.assertTrue(groupByResult.size() > 0);
   }
 
   @Test
@@ -173,7 +218,7 @@ public class JsonPathClusterIntegrationTest extends BaseClusterIntegrationTest {
       throws Exception {
 
     //Selection Query
-    String pqlQuery = "Select complexMapStr from " + DEFAULT_TABLE_NAME;
+    String pqlQuery = "Select " + COMPLEX_MAP_STR_FIELD_NAME + " from " + DEFAULT_TABLE_NAME;
     JsonNode pinotResponse = postQuery(pqlQuery);
     ArrayNode selectionResults = (ArrayNode) pinotResponse.get("selectionResults").get("results");
 
@@ -198,6 +243,21 @@ public class JsonPathClusterIntegrationTest extends BaseClusterIntegrationTest {
       Assert.assertEquals(Double.parseDouble(k4.get("met").toString()), (double) i);
     }
 
+    pqlQuery = "Select " + COMPLEX_MAP_STR_K3_FIELD_NAME + " from " + DEFAULT_TABLE_NAME;
+    pinotResponse = postQuery(pqlQuery);
+    selectionResults = (ArrayNode) pinotResponse.get("selectionResults").get("results");
+    System.out.println("selectionResults = " + selectionResults);
+
+    Assert.assertNotNull(selectionResults);
+    Assert.assertTrue(selectionResults.size() > 0);
+    for (int i = 0; i < selectionResults.size(); i++) {
+      JsonNode k3 = selectionResults.get(i).get(0);
+      Assert.assertEquals(k3.size(), 3);
+      Assert.assertEquals(k3.get(0).asText(), "value-k3-0-" + i);
+      Assert.assertEquals(k3.get(1).asText(), "value-k3-1-" + i);
+      Assert.assertEquals(k3.get(2).asText(), "value-k3-2-" + i);
+    }
+
     //Filter Query
     pqlQuery = "Select jsonExtractScalar(complexMapStr,'$.k4','STRING') from " + DEFAULT_TABLE_NAME
         + "  where jsonExtractScalar(complexMapStr,'$.k4.k4-k1','STRING') = 'value-k4-k1-0'";
@@ -212,7 +272,7 @@ public class JsonPathClusterIntegrationTest extends BaseClusterIntegrationTest {
     }
 
     //selection order by
-    pqlQuery = "Select complexMapStr from " + DEFAULT_TABLE_NAME
+    pqlQuery = "Select " + COMPLEX_MAP_STR_FIELD_NAME + " from " + DEFAULT_TABLE_NAME
         + " order by jsonExtractScalar(complexMapStr,'$.k4.k4-k1','STRING') DESC LIMIT " + NUM_TOTAL_DOCS;
     pinotResponse = postQuery(pqlQuery);
     selectionResults = (ArrayNode) pinotResponse.get("selectionResults").get("results");


---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscribe@pinot.apache.org
For additional commands, e-mail: commits-help@pinot.apache.org