akkio-97 commented on a change in pull request #3773:
URL: https://github.com/apache/carbondata/pull/3773#discussion_r462599726



##########
File path: 
core/src/main/java/org/apache/carbondata/core/scan/executor/impl/AbstractQueryExecutor.java
##########
@@ -508,7 +508,8 @@ private BlockExecutionInfo 
getBlockExecutionInfoForBlock(QueryModel queryModel,
     int[] dimensionChunkIndexes = 
QueryUtil.getDimensionChunkIndexes(projectDimensions,
         segmentProperties.getDimensionOrdinalToChunkMapping(),
         currentBlockFilterDimensions, allProjectionListDimensionIndexes);
-    ReusableDataBuffer[] dimensionBuffer = new 
ReusableDataBuffer[projectDimensions.size()];
+    // 4 because each column(including itself) can have a maximum of 3 nested 
complex levels
+    ReusableDataBuffer[] dimensionBuffer = new 
ReusableDataBuffer[projectDimensions.size() * 4];

Review comment:
       done

##########
File path: 
core/src/main/java/org/apache/carbondata/core/scan/result/BlockletScannedResult.java
##########
@@ -153,6 +154,8 @@
 
   private ReusableDataBuffer[] measureReusableBuffer;
 
+  int startIndex = 0;

Review comment:
       done
   

##########
File path: 
core/src/main/java/org/apache/carbondata/core/scan/result/vector/CarbonColumnVector.java
##########
@@ -114,4 +114,14 @@
 
   void setLazyPage(LazyPageLoader lazyPage);
 
+  CarbonColumnVector getColumnVector();
+
+  CarbonColumnVector getChildrenVector();

Review comment:
       done

##########
File path: 
core/src/main/java/org/apache/carbondata/core/scan/result/vector/CarbonColumnVector.java
##########
@@ -114,4 +114,14 @@
 
   void setLazyPage(LazyPageLoader lazyPage);
 
+  CarbonColumnVector getColumnVector();
+
+  CarbonColumnVector getChildrenVector();
+
+  void putArrayObject();
+
+  int getIndex();

Review comment:
       done

##########
File path: 
core/src/main/java/org/apache/carbondata/core/scan/result/vector/CarbonColumnVector.java
##########
@@ -114,4 +114,14 @@
 
   void setLazyPage(LazyPageLoader lazyPage);
 
+  CarbonColumnVector getColumnVector();

Review comment:
       done

##########
File path: 
integration/presto/src/main/java/org/apache/carbondata/presto/PrestoCarbonVectorizedRecordReader.java
##########
@@ -176,8 +191,16 @@ private void initBatch() {
     for (int i = 0; i < queryDimension.size(); i++) {
       ProjectionDimension dim = queryDimension.get(i);
       if (dim.getDimension().isComplex()) {
+        List<CarbonDimension> childDimensions =
+                dim.getDimension().getListOfChildDimensions();
+        ArrayList<StructField> childFields = new ArrayList<StructField>();
+        for (int ind = 0; ind < childDimensions.size(); ind++) {

Review comment:
       done

##########
File path: 
integration/presto/src/main/prestosql/org/apache/carbondata/presto/readers/ArrayStreamReader.java
##########
@@ -0,0 +1,145 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements.  See the NOTICE file distributed with
+ * this work for additional information regarding copyright ownership.
+ * The ASF licenses this file to You under the Apache License, Version 2.0
+ * (the "License"); you may not use this file except in compliance with
+ * the License.  You may obtain a copy of the License at
+ *
+ *    http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+
+package org.apache.carbondata.presto.readers;
+
+import io.prestosql.spi.type.*;
+
+import org.apache.carbondata.core.metadata.datatype.DataType;
+import org.apache.carbondata.core.metadata.datatype.DataTypes;
+import org.apache.carbondata.core.metadata.datatype.StructField;
+import 
org.apache.carbondata.core.scan.result.vector.impl.CarbonColumnVectorImpl;
+
+import io.prestosql.spi.block.Block;
+import io.prestosql.spi.block.BlockBuilder;
+
+import org.apache.carbondata.presto.CarbonVectorBatch;
+
+/**
+ * Class to read the Array Stream
+ */
+
+public class ArrayStreamReader extends CarbonColumnVectorImpl implements 
PrestoVectorBlockBuilder {
+
+  protected int batchSize;
+
+  protected Type type;
+  protected BlockBuilder builder;
+  Block childBlock = null;
+  private int index = 0;
+
+  public ArrayStreamReader(int batchSize, DataType dataType, StructField 
field) {
+    super(batchSize, dataType);
+    this.batchSize = batchSize;
+    this.type = getArrayOfType(field, dataType);
+    setChildrenVector(
+        CarbonVectorBatch.createDirectStreamReader(this.batchSize, 
field.getDataType(), field));
+    this.builder = type.createBlockBuilder(null, batchSize);
+  }
+
+  public int getIndex() {
+    return index;
+  }
+
+  public void setIndex(int index) {
+    this.index = index;
+  }
+
+  public String getDataTypeName() {
+    return "ARRAY";
+  }
+
+  Type getArrayOfType(StructField field, DataType dataType) {
+    if (dataType == DataTypes.STRING) {
+      return new ArrayType(VarcharType.VARCHAR);
+    } else if (dataType == DataTypes.INT) {
+      return new ArrayType(IntegerType.INTEGER);
+    } else if (dataType == DataTypes.LONG) {
+      return new ArrayType(BigintType.BIGINT);
+    } else if (dataType == DataTypes.DOUBLE) {
+      return new ArrayType(DoubleType.DOUBLE);
+    } else if (dataType == DataTypes.FLOAT) {
+      return new ArrayType(RealType.REAL);
+    } else if (dataType == DataTypes.BOOLEAN) {
+      return new ArrayType(BooleanType.BOOLEAN);
+    } else {
+      StructField childField = field.getChildren().get(0);
+      return new ArrayType(getArrayOfType(childField, 
childField.getDataType()));
+    }
+  }
+
+  @Override
+  public Block buildBlock() {
+    return builder.build();
+  }
+
+  public boolean isComplex() {
+    return true;
+  }
+
+  @Override
+  public void setBatchSize(int batchSize) {
+    this.batchSize = batchSize;
+  }
+
+  @Override
+  public void putObject(int rowId, Object value) {
+    if (value == null) {
+      putNull(rowId);
+    } else {
+      getChildrenVector().putObject(rowId, value);
+    }
+  }
+
+  public void putArrayObject() {
+    if (this.getType().getName() == "ARRAY") {
+      childBlock = ((ArrayStreamReader) getChildrenVector()).buildBlock();
+    } else if (this.getType().getName() == "STRING") {

Review comment:
       done

##########
File path: 
integration/presto/src/test/scala/org/apache/carbondata/presto/integrationtest/PrestoReadTableFilesTest.scala
##########
@@ -0,0 +1,443 @@
+package org.apache.carbondata.presto.integrationtest
+
+import java.io.File
+import java.sql.{SQLException, Timestamp}
+import java.util
+import java.util.Arrays.asList
+
+import io.prestosql.jdbc.PrestoArray
+import org.apache.carbondata.common.logging.LogServiceFactory
+import org.apache.carbondata.core.constants.CarbonCommonConstants
+import org.apache.carbondata.core.datastore.filesystem.CarbonFile
+import org.apache.carbondata.core.datastore.impl.FileFactory
+import org.apache.carbondata.core.metadata.datatype.{DataTypes, Field}
+import org.apache.carbondata.core.util.{CarbonProperties, CarbonUtil}
+import org.apache.carbondata.presto.server.PrestoServer
+import org.apache.carbondata.sdk.file.{CarbonWriter, Schema}
+import org.apache.commons.io.FileUtils
+import org.apache.commons.lang.RandomStringUtils
+import org.apache.spark.sql.Row
+import org.scalatest.{BeforeAndAfterAll, FunSuiteLike, BeforeAndAfterEach}
+
+import scala.collection.mutable
+import scala.collection.JavaConverters._
+class PrestoReadTableFilesTest extends FunSuiteLike with BeforeAndAfterAll 
with BeforeAndAfterEach{
+  private val logger = LogServiceFactory
+    
.getLogService(classOf[PrestoTestNonTransactionalTableFiles].getCanonicalName)
+
+  private val rootPath = new File(this.getClass.getResource("/").getPath
+    + "../../../..").getCanonicalPath
+  private val storePath = s"$rootPath/integration/presto/target/store"
+  private val systemPath = s"$rootPath/integration/presto/target/system"
+  private var writerPath = storePath + "/sdk_output/files"
+  private val prestoServer = new PrestoServer
+  private var varcharString = new String
+
+  override def beforeAll: Unit = {
+    
CarbonProperties.getInstance().addProperty(CarbonCommonConstants.CARBON_WRITTEN_BY_APPNAME,
+      "Presto")
+    
CarbonProperties.getInstance().addProperty(CarbonCommonConstants.CARBON_WRITTEN_BY_APPNAME,
+      "Presto")
+    val map = new util.HashMap[String, String]()
+    map.put("hive.metastore", "file")
+    map.put("hive.metastore.catalog.dir", s"file://$storePath")
+
+    prestoServer.startServer("sdk_output", map)
+  }
+
+  override def afterAll(): Unit = {
+    prestoServer.stopServer()
+    CarbonUtil.deleteFoldersAndFiles(FileFactory.getCarbonFile(storePath))
+  }
+
+  private def createComplexTableForSingleLevelArray = {
+    prestoServer.execute("drop table if exists sdk_output.files")
+    prestoServer.execute("drop schema if exists sdk_output")
+    prestoServer.execute("create schema sdk_output")
+    prestoServer
+      .execute(
+        "create table sdk_output.files(stringCol varchar, intCol int, 
doubleCol double, realCol real, boolCol boolean, arrayStringCol1 
array(varchar), arrayStringcol2 array(varchar), arrayIntCol array(int), 
arrayBigIntCol array(bigint), arrayRealCol array(real), arrayDoubleCol 
array(double), arrayBooleanCol array(boolean)) with(format='CARBON') ")
+  }
+
+  private def createComplexTableFor2LevelArray = {
+    prestoServer.execute("drop table if exists sdk_output.files2")
+    prestoServer.execute("drop schema if exists sdk_output")
+    prestoServer.execute("create schema sdk_output")
+        prestoServer
+      .execute(
+        "create table sdk_output.files2(arrayArrayInt array(array(int)), 
arrayArrayBigInt array(array(bigint)), arrayArrayReal array(array(real)), 
arrayArrayDouble array(array(double)), arrayArrayString array(array(varchar)), 
arrayArrayBoolean array(array(boolean))) with(format='CARBON') ")
+  }
+
+  private def createComplexTableFor3LevelArray = {
+    prestoServer.execute("drop table if exists sdk_output.files3")
+    prestoServer.execute("drop schema if exists sdk_output")
+    prestoServer.execute("create schema sdk_output")
+    prestoServer
+        .execute(
+          "create table sdk_output.files3(array3_Int array(array(array(int))), 
array3_BigInt array(array(array(bigint))), array3_Real 
array(array(array(real))), array3_Double array(array(array(double))), 
array3_String array(array(array(varchar))), array3_Boolean 
array(array(array(boolean))) ) with(format='CARBON') ")
+    }
+
+  def buildComplexTestForSingleLevelArray(): Any = {
+    FileUtils.deleteDirectory(new File(writerPath))
+    createComplexTableForSingleLevelArray
+    import java.io.IOException
+    val source = new File(this.getClass.getResource("/").getPath + "../../" + 
"/temp/table1").getCanonicalPath
+    val srcDir = new File(source)
+    val destination = new File(this.getClass.getResource("/").getPath + 
"../../" + "/target/store/sdk_output/files/").getCanonicalPath
+    val destDir = new File(destination)
+    try FileUtils.copyDirectory(srcDir, destDir)
+    catch {
+      case e: IOException =>
+        e.printStackTrace()
+    }
+  }
+
+  def buildComplexTestFor2LevelArray(): Any = {
+    writerPath = storePath + "/sdk_output/files2"
+    FileUtils.deleteDirectory(new File(writerPath))
+    createComplexTableFor2LevelArray
+    import java.io.IOException
+    val source = new File(this.getClass.getResource("/").getPath + "../../" + 
"/temp/table2").getCanonicalPath
+    val srcDir = new File(source)
+    val destination = new File(this.getClass.getResource("/").getPath + 
"../../" + "/target/store/sdk_output/files2/").getCanonicalPath
+    val destDir = new File(destination)
+    try FileUtils.copyDirectory(srcDir, destDir)
+    catch {
+      case e: IOException =>
+        e.printStackTrace()

Review comment:
       done

##########
File path: 
core/src/main/java/org/apache/carbondata/core/scan/result/BlockletScannedResult.java
##########
@@ -153,6 +154,8 @@
 
   private ReusableDataBuffer[] measureReusableBuffer;
 
+  int startIndex = 0;

Review comment:
       done

##########
File path: 
core/src/main/java/org/apache/carbondata/core/scan/result/vector/CarbonColumnVector.java
##########
@@ -114,4 +114,14 @@
 
   void setLazyPage(LazyPageLoader lazyPage);
 
+  CarbonColumnVector getColumnVector();
+
+  CarbonColumnVector getChildrenVector();

Review comment:
       done

##########
File path: 
core/src/main/java/org/apache/carbondata/core/scan/result/BlockletScannedResult.java
##########
@@ -153,6 +154,8 @@
 
   private ReusableDataBuffer[] measureReusableBuffer;
 
+  int startIndex = 0;

Review comment:
       done

##########
File path: 
core/src/main/java/org/apache/carbondata/core/scan/result/vector/CarbonColumnVector.java
##########
@@ -114,4 +114,14 @@
 
   void setLazyPage(LazyPageLoader lazyPage);
 
+  CarbonColumnVector getColumnVector();
+
+  CarbonColumnVector getChildrenVector();
+
+  void putArrayObject();
+
+  int getIndex();

Review comment:
       done

##########
File path: 
core/src/main/java/org/apache/carbondata/core/scan/result/vector/CarbonColumnVector.java
##########
@@ -114,4 +114,14 @@
 
   void setLazyPage(LazyPageLoader lazyPage);
 
+  CarbonColumnVector getColumnVector();

Review comment:
       yes

##########
File path: 
core/src/main/java/org/apache/carbondata/core/scan/executor/impl/AbstractQueryExecutor.java
##########
@@ -508,7 +508,8 @@ private BlockExecutionInfo 
getBlockExecutionInfoForBlock(QueryModel queryModel,
     int[] dimensionChunkIndexes = 
QueryUtil.getDimensionChunkIndexes(projectDimensions,
         segmentProperties.getDimensionOrdinalToChunkMapping(),
         currentBlockFilterDimensions, allProjectionListDimensionIndexes);
-    ReusableDataBuffer[] dimensionBuffer = new 
ReusableDataBuffer[projectDimensions.size()];
+    // 4 because each column(including itself) can have a maximum of 3 nested 
complex levels
+    ReusableDataBuffer[] dimensionBuffer = new 
ReusableDataBuffer[projectDimensions.size() * 4];

Review comment:
       done

##########
File path: 
core/src/main/java/org/apache/carbondata/core/metadata/schema/table/column/CarbonDimension.java
##########
@@ -39,6 +39,16 @@
    */
   private int keyOrdinal;
 
+  private byte[] childElements;

Review comment:
       done

##########
File path: 
core/src/main/java/org/apache/carbondata/core/datastore/page/encoding/compress/DirectCompressCodec.java
##########
@@ -246,7 +241,28 @@ public void decodeAndFillVector(byte[] pageData, 
ColumnVectorInfo vectorInfo, Bi
       vector = ColumnarVectorWrapperDirectFactory
           .getDirectVectorWrapperFactory(vector, vectorInfo.invertedIndex, 
nullBits, deletedRows,
               true, false);
-      fillVector(pageData, vector, vectorDataType, pageDataType, pageSize, 
vectorInfo, nullBits);
+      Deque<CarbonColumnVectorImpl> vectorStack = vectorInfo.getVectorStack();
+      // initialize vectorStack if null
+      if (vectorStack == null && vectorInfo.vector.getColumnVector() != null) {
+        vectorStack = new ArrayDeque<CarbonColumnVectorImpl>();
+        vectorStack.push((CarbonColumnVectorImpl) 
vectorInfo.vector.getColumnVector());
+        vectorInfo.setVectorStack(vectorStack);

Review comment:
       done

##########
File path: 
core/src/main/java/org/apache/carbondata/core/datastore/page/encoding/compress/DirectCompressCodec.java
##########
@@ -32,29 +29,27 @@
 import org.apache.carbondata.core.datastore.page.ColumnPageValueConverter;
 import org.apache.carbondata.core.datastore.page.LazyColumnPage;
 import org.apache.carbondata.core.datastore.page.VarLengthColumnPageBase;
-import org.apache.carbondata.core.datastore.page.encoding.ColumnPageCodec;
-import org.apache.carbondata.core.datastore.page.encoding.ColumnPageDecoder;
-import org.apache.carbondata.core.datastore.page.encoding.ColumnPageEncoder;
-import 
org.apache.carbondata.core.datastore.page.encoding.ColumnPageEncoderMeta;
+import org.apache.carbondata.core.datastore.page.encoding.*;
 import org.apache.carbondata.core.metadata.datatype.DataType;
 import org.apache.carbondata.core.metadata.datatype.DataTypes;
 import org.apache.carbondata.core.metadata.datatype.DecimalConverterFactory;
 import org.apache.carbondata.core.scan.result.vector.CarbonColumnVector;
 import org.apache.carbondata.core.scan.result.vector.ColumnVectorInfo;
+import 
org.apache.carbondata.core.scan.result.vector.impl.CarbonColumnVectorImpl;
 import 
org.apache.carbondata.core.scan.result.vector.impl.directread.ColumnarVectorWrapperDirectFactory;
 import 
org.apache.carbondata.core.scan.result.vector.impl.directread.ConvertibleVector;
 import 
org.apache.carbondata.core.scan.result.vector.impl.directread.SequentialFill;
-import org.apache.carbondata.core.util.ByteUtil;
 import org.apache.carbondata.format.Encoding;
 
 /**
  * This codec directly apply compression on the input data
  */
+//public class DirectCompressCodec extends SerializeComplexTypes implements 
ColumnPageCodec {

Review comment:
       done

##########
File path: 
core/src/main/java/org/apache/carbondata/core/scan/result/vector/CarbonColumnVector.java
##########
@@ -114,4 +114,14 @@
 
   void setLazyPage(LazyPageLoader lazyPage);
 
+  CarbonColumnVector getColumnVector();

Review comment:
       yes

##########
File path: 
core/src/main/java/org/apache/carbondata/core/scan/result/vector/CarbonColumnVector.java
##########
@@ -114,4 +114,14 @@
 
   void setLazyPage(LazyPageLoader lazyPage);
 
+  CarbonColumnVector getColumnVector();
+
+  CarbonColumnVector getChildrenVector();
+
+  void putArrayObject();
+
+  int getIndex();

Review comment:
       done

##########
File path: 
core/src/main/java/org/apache/carbondata/core/scan/result/vector/CarbonColumnVector.java
##########
@@ -114,4 +114,14 @@
 
   void setLazyPage(LazyPageLoader lazyPage);
 
+  CarbonColumnVector getColumnVector();
+
+  CarbonColumnVector getChildrenVector();

Review comment:
       done

##########
File path: 
core/src/main/java/org/apache/carbondata/core/scan/result/BlockletScannedResult.java
##########
@@ -153,6 +154,8 @@
 
   private ReusableDataBuffer[] measureReusableBuffer;
 
+  int startIndex = 0;

Review comment:
       done

##########
File path: 
core/src/main/java/org/apache/carbondata/core/scan/result/BlockletScannedResult.java
##########
@@ -153,6 +154,8 @@
 
   private ReusableDataBuffer[] measureReusableBuffer;
 
+  int startIndex = 0;

Review comment:
       done
   

##########
File path: 
core/src/main/java/org/apache/carbondata/core/scan/executor/impl/AbstractQueryExecutor.java
##########
@@ -508,7 +508,8 @@ private BlockExecutionInfo 
getBlockExecutionInfoForBlock(QueryModel queryModel,
     int[] dimensionChunkIndexes = 
QueryUtil.getDimensionChunkIndexes(projectDimensions,
         segmentProperties.getDimensionOrdinalToChunkMapping(),
         currentBlockFilterDimensions, allProjectionListDimensionIndexes);
-    ReusableDataBuffer[] dimensionBuffer = new 
ReusableDataBuffer[projectDimensions.size()];
+    // 4 because each column(including itself) can have a maximum of 3 nested 
complex levels
+    ReusableDataBuffer[] dimensionBuffer = new 
ReusableDataBuffer[projectDimensions.size() * 4];

Review comment:
       done




----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
us...@infra.apache.org


Reply via email to