You are viewing a plain text version of this content. The canonical link for it is here.
Posted to issues@carbondata.apache.org by lionelcao <gi...@git.apache.org> on 2017/05/04 18:03:05 UTC

[GitHub] incubator-carbondata pull request #882: [CARBONDATA-936] Parse partition tab...

GitHub user lionelcao opened a pull request:

    https://github.com/apache/incubator-carbondata/pull/882

    [CARBONDATA-936] Parse partition table ddl

    Parse partition table ddl, get partitionInfo and store in carbon table.
    SUPPORT range, list, hash partition
    SUPPORT basic syntax validate, partition info in table properties validate
    NOT SUPPORT partition value validate for now. Maybe support in the future or just let user confirm.
    NOT SUPPORT range interval partition for now.
    Partition column must be part of the schema.

You can merge this pull request into a Git repository by running:

    $ git pull https://github.com/lionelcao/incubator-carbondata 12-dev_910_2

Alternatively you can review and apply these changes as the patch at:

    https://github.com/apache/incubator-carbondata/pull/882.patch

To close this pull request, make a commit to your master/trunk branch
with (at least) the following in the commit message:

    This closes #882
    
----
commit c95b311ca2077ef68d4eceddacac633637a9bc1b
Author: lionelcao <wh...@gmail.com>
Date:   2017-04-25T03:50:10Z

    parser hash partition table

commit 28e7981d3dea7570532d50c614e15fafbf98e650
Author: lionelcao <wh...@gmail.com>
Date:   2017-04-25T09:49:18Z

    add partition validate & example

commit 981a3254014af39cbda62b4e7d88c1bf2c92003f
Author: lionelcao <wh...@gmail.com>
Date:   2017-04-25T10:04:53Z

    fix style error

commit 58883d6c3937dacc7976f972cbbd5351fb910c6e
Author: lionelcao <wh...@gmail.com>
Date:   2017-04-27T03:24:32Z

    fix style error

commit 6677497e7ab9ceffe30e6239f5c7b0162616a317
Author: lionelcao <wh...@gmail.com>
Date:   2017-05-04T14:50:12Z

    add range partition

commit 3e7c23052443ffa01ea6f7877547561ed0d6045e
Author: lionelcao <wh...@gmail.com>
Date:   2017-05-04T17:47:29Z

    add range/list parser

----


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

[GitHub] incubator-carbondata issue #882: [CARBONDATA-936] Parse partition table ddl

Posted by CarbonDataQA <gi...@git.apache.org>.
Github user CarbonDataQA commented on the issue:

    https://github.com/apache/incubator-carbondata/pull/882
  
    Build Failed  with Spark 1.6.2, Please check CI http://136.243.101.176:8080/job/ApacheCarbonPRBuilder/1896/



---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

[GitHub] incubator-carbondata issue #882: [CARBONDATA-936] Parse partition table ddl

Posted by CarbonDataQA <gi...@git.apache.org>.
Github user CarbonDataQA commented on the issue:

    https://github.com/apache/incubator-carbondata/pull/882
  
    Build Success with Spark 1.6.2, Please check CI http://136.243.101.176:8080/job/ApacheCarbonPRBuilder/1900/



---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

[GitHub] incubator-carbondata pull request #882: [CARBONDATA-936] Parse partition tab...

Posted by jackylk <gi...@git.apache.org>.
Github user jackylk commented on a diff in the pull request:

    https://github.com/apache/incubator-carbondata/pull/882#discussion_r115163166
  
    --- Diff: integration/spark-common/src/main/scala/org/apache/spark/sql/catalyst/CarbonDDLSqlParser.scala ---
    @@ -258,7 +261,12 @@ abstract class CarbonDDLSqlParser extends AbstractCarbonSparkSQLParser {
     
         // get no inverted index columns from table properties.
         val noInvertedIdxCols = extractNoInvertedIndexColumns(fields, tableProperties)
    +    // get partitionInfo
    +    var partitionInfo: Option[PartitionInfo] = None
     
    +    if (!(partitionCols.length < 1)) {
    --- End diff --
    
    use `partitionCols.length >= 1`


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

[GitHub] incubator-carbondata pull request #882: [CARBONDATA-936] Parse partition tab...

Posted by QiangCai <gi...@git.apache.org>.
Github user QiangCai commented on a diff in the pull request:

    https://github.com/apache/incubator-carbondata/pull/882#discussion_r114950536
  
    --- Diff: integration/spark-common/src/main/scala/org/apache/spark/sql/catalyst/CarbonDDLSqlParser.scala ---
    @@ -347,43 +356,65 @@ abstract class CarbonDDLSqlParser extends AbstractCarbonSparkSQLParser {
       }
     
       /**
    -   * For getting the partitioner Object
    -   *
    +   * get partition info
        * @param partitionCols
        * @param tableProperties
    -   * @return
        */
    -  protected def getPartitionerObject(partitionCols: Seq[PartitionerField],
    -      tableProperties: Map[String, String]):
    -  Option[Partitioner] = {
    -
    -    // by default setting partition class empty.
    -    // later in table schema it is setting to default value.
    -    var partitionClass: String = ""
    -    var partitionCount: Int = 1
    -    var partitionColNames: Array[String] = Array[String]()
    -    if (tableProperties.get(CarbonCommonConstants.PARTITIONCLASS).isDefined) {
    -      partitionClass = tableProperties.get(CarbonCommonConstants.PARTITIONCLASS).get
    +  protected def getPartitionInfo(partitionCols: Seq[PartitionerField],
    +      tableProperties: Map[String, String]): Option[PartitionInfo] = {
    +    var partitionType: String = ""
    +    var hashNumber = 0
    +    var rangeInfo = List[String]()
    +    var listInfo = ListBuffer[List[String]]()
    +    var templist = ListBuffer[String]()
    +    if (tableProperties.get(CarbonCommonConstants.PARTITION_TYPE).isDefined) {
    +      partitionType = tableProperties.get(CarbonCommonConstants.PARTITION_TYPE).get
         }
    -
    -    if (tableProperties.get(CarbonCommonConstants.PARTITIONCOUNT).isDefined) {
    -      try {
    -        partitionCount = tableProperties.get(CarbonCommonConstants.PARTITIONCOUNT).get.toInt
    -      } catch {
    -        case e: Exception => // no need to do anything.
    +    if (tableProperties.get(CarbonCommonConstants.HASH_NUMBER).isDefined) {
    +      hashNumber = tableProperties.get(CarbonCommonConstants.HASH_NUMBER).get.toInt
    +    }
    +    if (tableProperties.get(CarbonCommonConstants.RANGE_INFO).isDefined) {
    +      rangeInfo = tableProperties.get(CarbonCommonConstants.RANGE_INFO).get.replace(" ", "")
    --- End diff --
    
    Maybe can't remove the middle blank.


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

[GitHub] incubator-carbondata pull request #882: [CARBONDATA-936] Parse partition tab...

Posted by jackylk <gi...@git.apache.org>.
Github user jackylk commented on a diff in the pull request:

    https://github.com/apache/incubator-carbondata/pull/882#discussion_r115162313
  
    --- Diff: integration/spark-common/src/main/scala/org/apache/carbondata/spark/util/CommonUtil.scala ---
    @@ -144,6 +145,41 @@ object CommonUtil {
         isValid
       }
     
    +  /**
    +   * 1. If partitioned by clause exists, then partition_type should be defined
    +   * 2. If partition_type is Hash, then hash_number should be defined
    +   * 3. If partition_type is List, then list_info should be defined
    +   * 4. If partition_type is Range, then range_info should be defined
    +   * 5. Only support single level partition for now
    +   * @param tableProperties
    +   * @param partitionCols
    +   * @return
    --- End diff --
    
    fill return description


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

[GitHub] incubator-carbondata issue #882: [CARBONDATA-936] Parse partition table ddl

Posted by CarbonDataQA <gi...@git.apache.org>.
Github user CarbonDataQA commented on the issue:

    https://github.com/apache/incubator-carbondata/pull/882
  
    Build Success with Spark 1.6.2, Please check CI http://136.243.101.176:8080/job/ApacheCarbonPRBuilder/1946/



---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

[GitHub] incubator-carbondata pull request #882: [CARBONDATA-936] Parse partition tab...

Posted by jackylk <gi...@git.apache.org>.
Github user jackylk commented on a diff in the pull request:

    https://github.com/apache/incubator-carbondata/pull/882#discussion_r115163651
  
    --- Diff: integration/spark2/src/main/scala/org/apache/spark/sql/parser/CarbonSparkSqlParser.scala ---
    @@ -97,6 +98,8 @@ class CarbonSqlAstBuilder(conf: SQLConf) extends SparkSqlAstBuilder(conf) {
             operationNotAllowed("CREATE TABLE ... CLUSTERED BY", ctx)
           }
           val partitionCols = Option(ctx.partitionColumns).toSeq.flatMap(visitColTypeList)
    +      val partitionFields = partitionCols.map(x =>
    --- End diff --
    
    use 
    ```
     val partitionFields = partitionCols.map { (name, dataType) =>
       PartitionerField(name, Some(dataType.toString), null)
    }


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

[GitHub] incubator-carbondata issue #882: [CARBONDATA-936] Parse partition table ddl

Posted by CarbonDataQA <gi...@git.apache.org>.
Github user CarbonDataQA commented on the issue:

    https://github.com/apache/incubator-carbondata/pull/882
  
    Build Failed  with Spark 1.6.2, Please check CI http://136.243.101.176:8080/job/ApacheCarbonPRBuilder/1940/



---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

[GitHub] incubator-carbondata pull request #882: [CARBONDATA-936] Parse partition tab...

Posted by jackylk <gi...@git.apache.org>.
Github user jackylk commented on a diff in the pull request:

    https://github.com/apache/incubator-carbondata/pull/882#discussion_r115190921
  
    --- Diff: core/src/main/java/org/apache/carbondata/core/constants/CarbonCommonConstants.java ---
    @@ -806,8 +806,10 @@
       public static final String DICTIONARY_EXCLUDE = "dictionary_exclude";
       public static final String DICTIONARY_INCLUDE = "dictionary_include";
       public static final String SORT_COLUMNS = "sort_columns";
    -  public static final String PARTITIONCLASS = "partitionclass";
    -  public static final String PARTITIONCOUNT = "partitioncount";
    +  public static final String PARTITION_TYPE = "partition_type";
    +  public static final String HASH_NUMBER = "hash_number";
    --- End diff --
    
    rename to NUMBER_OF_PARTITIONS


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

[GitHub] incubator-carbondata issue #882: [CARBONDATA-936] Parse partition table ddl

Posted by CarbonDataQA <gi...@git.apache.org>.
Github user CarbonDataQA commented on the issue:

    https://github.com/apache/incubator-carbondata/pull/882
  
    Build Success with Spark 1.6.2, Please check CI http://136.243.101.176:8080/job/ApacheCarbonPRBuilder/1937/



---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

[GitHub] incubator-carbondata pull request #882: [CARBONDATA-936] Parse partition tab...

Posted by jackylk <gi...@git.apache.org>.
Github user jackylk commented on a diff in the pull request:

    https://github.com/apache/incubator-carbondata/pull/882#discussion_r115163281
  
    --- Diff: integration/spark-common/src/main/scala/org/apache/spark/sql/catalyst/CarbonDDLSqlParser.scala ---
    @@ -258,7 +261,12 @@ abstract class CarbonDDLSqlParser extends AbstractCarbonSparkSQLParser {
     
         // get no inverted index columns from table properties.
         val noInvertedIdxCols = extractNoInvertedIndexColumns(fields, tableProperties)
    +    // get partitionInfo
    +    var partitionInfo: Option[PartitionInfo] = None
    --- End diff --
    
    use 
    ```
    var partitionInfo = 
       if (partitionCols.length < 1) None
        else getPartition(partitionCols, tableProperties)
    ```


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

[GitHub] incubator-carbondata pull request #882: [CARBONDATA-936] Parse partition tab...

Posted by jackylk <gi...@git.apache.org>.
Github user jackylk commented on a diff in the pull request:

    https://github.com/apache/incubator-carbondata/pull/882#discussion_r115162145
  
    --- Diff: examples/spark2/src/main/scala/org/apache/carbondata/examples/CarbonPartitionExample.scala ---
    @@ -0,0 +1,131 @@
    +/*
    + * 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.examples
    +
    +import java.io.File
    +
    +import org.apache.spark.sql.SparkSession
    +
    +import org.apache.carbondata.core.constants.CarbonCommonConstants
    +import org.apache.carbondata.core.util.CarbonProperties
    +
    +object CarbonPartitionExample {
    +
    +  def main(args: Array[String]) {
    +    val rootPath = new File(this.getClass.getResource("/").getPath
    +                            + "../../../..").getCanonicalPath
    +    val storeLocation = s"$rootPath/examples/spark2/target/store"
    +    val warehouse = s"$rootPath/examples/spark2/target/warehouse"
    +    val metastoredb = s"$rootPath/examples/spark2/target"
    +    val testData = s"$rootPath/examples/spark2/src/main/resources/partition_data.csv"
    +
    +    CarbonProperties.getInstance()
    +      .addProperty(CarbonCommonConstants.CARBON_TIMESTAMP_FORMAT, "yyyy/MM/dd")
    +
    +    import org.apache.spark.sql.CarbonSession._
    +
    +    val spark = SparkSession
    +      .builder()
    +      .master("local")
    +      .appName("CarbonPartitionExample")
    +      .config("spark.sql.warehouse.dir", warehouse)
    +      .getOrCreateCarbonSession(storeLocation, metastoredb)
    +
    +    spark.sparkContext.setLogLevel("WARN")
    +
    +    // none partition table
    +    spark.sql("DROP TABLE IF EXISTS t0")
    +
    +    spark.sql("""
    +                | CREATE TABLE IF NOT EXISTS t0
    +                | (
    +                | vin String,
    +                | logdate Timestamp,
    +                | phonenumber Long,
    +                | country String,
    +                | area String
    +                | )
    +                | STORED BY 'carbondata'
    +              """.stripMargin)
    +
    +    // range partition
    +    spark.sql("DROP TABLE IF EXISTS t1")
    +
    +    spark.sql("""
    +                | CREATE TABLE IF NOT EXISTS t1
    +                | (
    +                | vin String,
    +                | logdate Timestamp,
    +                | phonenumber Long,
    +                | country String,
    +                | area String
    +                | )
    +                | PARTITIONED BY (logdate Timestamp)
    +                | STORED BY 'carbondata'
    +                | TBLPROPERTIES('PARTITION_TYPE'='RANGE',
    +                | 'RANGE_INFO'='20140101, 2015/01/01 ,2016-01-01, ')
    +              """.stripMargin)
    +
    +    // hash partition
    +    spark.sql("DROP TABLE IF EXISTS t3")
    +
    +    spark.sql("""
    +                | CREATE TABLE IF NOT EXISTS t3
    +                | (
    +                | vin String,
    +                | logdate Timestamp,
    +                | phonenumber Long,
    +                | country String,
    +                | area String
    +                | )
    +                | PARTITIONED BY (vin String)
    +                | STORED BY 'carbondata'
    +                | TBLPROPERTIES('PARTITION_TYPE'='HASH','HASH_NUMBER'='5')
    +                """.stripMargin)
    +
    +    // list partition
    +    spark.sql("DROP TABLE IF EXISTS t5")
    +
    +    spark.sql("""
    +       | CREATE TABLE IF NOT EXISTS t5
    +       | (
    +       | vin String,
    +       | logdate Timestamp,
    +       | phonenumber Long,
    +       | country String,
    +       | area String
    +       |)
    +       | PARTITIONED BY (country string)
    +       | STORED BY 'carbondata'
    +       | TBLPROPERTIES('PARTITION_TYPE'='LIST',
    +       | 'LIST_INFO'='(China,United States),UK ,japan,(Canada,Russia), South Korea ')
    +       """.stripMargin)
    +
    +    // spark.sql(s"""
    +    //   LOAD DATA LOCAL INPATH '$testData' into table t3
    --- End diff --
    
    Loading is not working in this PR?


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

[GitHub] incubator-carbondata pull request #882: [CARBONDATA-936] Parse partition tab...

Posted by jackylk <gi...@git.apache.org>.
Github user jackylk commented on a diff in the pull request:

    https://github.com/apache/incubator-carbondata/pull/882#discussion_r115162314
  
    --- Diff: integration/spark-common/src/main/scala/org/apache/carbondata/spark/util/CommonUtil.scala ---
    @@ -144,6 +145,41 @@ object CommonUtil {
         isValid
       }
     
    +  /**
    +   * 1. If partitioned by clause exists, then partition_type should be defined
    +   * 2. If partition_type is Hash, then hash_number should be defined
    +   * 3. If partition_type is List, then list_info should be defined
    +   * 4. If partition_type is Range, then range_info should be defined
    +   * 5. Only support single level partition for now
    +   * @param tableProperties
    +   * @param partitionCols
    +   * @return
    +   */
    +  def validatePartitionColumns(tableProperties: Map[String, String],
    +      partitionCols: Seq[StructField]): Boolean = {
    +    var isValid: Boolean = true
    +    val partitionType = tableProperties.get(CarbonCommonConstants.PARTITION_TYPE)
    +    val hashNumber = tableProperties.get(CarbonCommonConstants.HASH_NUMBER)
    +    val rangeInfo = tableProperties.get(CarbonCommonConstants.RANGE_INFO)
    +    val listInfo = tableProperties.get(CarbonCommonConstants.LIST_INFO)
    +
    +    // partition column and partition_type should be both exist or not exist
    +    if (partitionCols.isEmpty ^ partitionType.isEmpty) {
    --- End diff --
    
    use | or & for better readability


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

[GitHub] incubator-carbondata pull request #882: [CARBONDATA-936] Parse partition tab...

Posted by jackylk <gi...@git.apache.org>.
Github user jackylk commented on a diff in the pull request:

    https://github.com/apache/incubator-carbondata/pull/882#discussion_r115163717
  
    --- Diff: integration/spark2/src/main/scala/org/apache/spark/sql/parser/CarbonSparkSqlParser.scala ---
    @@ -111,18 +114,22 @@ class CarbonSqlAstBuilder(conf: SQLConf) extends SparkSqlAstBuilder(conf) {
                                 duplicateColumns.mkString("[", ",", "]"), ctx)
           }
     
    -      // For Hive tables, partition columns must not be part of the schema
    -      val badPartCols = partitionCols.map(_.name).toSet.intersect(colNames.toSet)
    -      if (badPartCols.nonEmpty) {
    -        operationNotAllowed(s"Partition columns may not be specified in the schema: " +
    -                            badPartCols.map("\"" + _ + "\"").mkString("[", ",", "]"), ctx)
    -      }
    +      val tableProperties = mutable.Map[String, String]()
    +      properties.foreach(f => tableProperties.put(f._1, f._2.toLowerCase))
    --- End diff --
    
    change f to a more meaningful name, and use `{` instead of `(`


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

[GitHub] incubator-carbondata issue #882: [CARBONDATA-936] Parse partition table ddl

Posted by CarbonDataQA <gi...@git.apache.org>.
Github user CarbonDataQA commented on the issue:

    https://github.com/apache/incubator-carbondata/pull/882
  
    Build Failed  with Spark 1.6.2, Please check CI http://136.243.101.176:8080/job/ApacheCarbonPRBuilder/1897/



---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

[GitHub] incubator-carbondata pull request #882: [CARBONDATA-936] Parse partition tab...

Posted by lionelcao <gi...@git.apache.org>.
Github user lionelcao commented on a diff in the pull request:

    https://github.com/apache/incubator-carbondata/pull/882#discussion_r114960569
  
    --- Diff: integration/spark-common/src/main/scala/org/apache/spark/sql/catalyst/CarbonDDLSqlParser.scala ---
    @@ -347,43 +356,65 @@ abstract class CarbonDDLSqlParser extends AbstractCarbonSparkSQLParser {
       }
     
       /**
    -   * For getting the partitioner Object
    -   *
    +   * get partition info
        * @param partitionCols
        * @param tableProperties
    -   * @return
        */
    -  protected def getPartitionerObject(partitionCols: Seq[PartitionerField],
    -      tableProperties: Map[String, String]):
    -  Option[Partitioner] = {
    -
    -    // by default setting partition class empty.
    -    // later in table schema it is setting to default value.
    -    var partitionClass: String = ""
    -    var partitionCount: Int = 1
    -    var partitionColNames: Array[String] = Array[String]()
    -    if (tableProperties.get(CarbonCommonConstants.PARTITIONCLASS).isDefined) {
    -      partitionClass = tableProperties.get(CarbonCommonConstants.PARTITIONCLASS).get
    +  protected def getPartitionInfo(partitionCols: Seq[PartitionerField],
    +      tableProperties: Map[String, String]): Option[PartitionInfo] = {
    +    var partitionType: String = ""
    +    var hashNumber = 0
    +    var rangeInfo = List[String]()
    +    var listInfo = ListBuffer[List[String]]()
    +    var templist = ListBuffer[String]()
    +    if (tableProperties.get(CarbonCommonConstants.PARTITION_TYPE).isDefined) {
    +      partitionType = tableProperties.get(CarbonCommonConstants.PARTITION_TYPE).get
         }
    -
    -    if (tableProperties.get(CarbonCommonConstants.PARTITIONCOUNT).isDefined) {
    -      try {
    -        partitionCount = tableProperties.get(CarbonCommonConstants.PARTITIONCOUNT).get.toInt
    -      } catch {
    -        case e: Exception => // no need to do anything.
    +    if (tableProperties.get(CarbonCommonConstants.HASH_NUMBER).isDefined) {
    +      hashNumber = tableProperties.get(CarbonCommonConstants.HASH_NUMBER).get.toInt
    +    }
    +    if (tableProperties.get(CarbonCommonConstants.RANGE_INFO).isDefined) {
    +      rangeInfo = tableProperties.get(CarbonCommonConstants.RANGE_INFO).get.replace(" ", "")
    --- End diff --
    
    Could you explain more details?


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

[GitHub] incubator-carbondata pull request #882: [CARBONDATA-936] Parse partition tab...

Posted by jackylk <gi...@git.apache.org>.
Github user jackylk commented on a diff in the pull request:

    https://github.com/apache/incubator-carbondata/pull/882#discussion_r115160665
  
    --- Diff: core/src/main/java/org/apache/carbondata/core/metadata/schema/PartitionInfo.java ---
    @@ -54,7 +51,6 @@
       private int hashNumber;
     
       /**
    -   * For range partition table
    --- End diff --
    
    remove all comment for this constructor


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

[GitHub] incubator-carbondata pull request #882: [CARBONDATA-936] Parse partition tab...

Posted by jackylk <gi...@git.apache.org>.
Github user jackylk commented on a diff in the pull request:

    https://github.com/apache/incubator-carbondata/pull/882#discussion_r115190776
  
    --- Diff: core/src/main/java/org/apache/carbondata/core/metadata/schema/PartitionInfo.java ---
    @@ -23,41 +23,18 @@
     import org.apache.carbondata.core.metadata.schema.partition.PartitionType;
     import org.apache.carbondata.core.metadata.schema.table.column.ColumnSchema;
     
    -/**
    - * Partition Information of carbon partition table
    --- End diff --
    
    Do not remove these comments


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

[GitHub] incubator-carbondata issue #882: [CARBONDATA-936] Parse partition table ddl

Posted by CarbonDataQA <gi...@git.apache.org>.
Github user CarbonDataQA commented on the issue:

    https://github.com/apache/incubator-carbondata/pull/882
  
    Build Success with Spark 1.6.2, Please check CI http://136.243.101.176:8080/job/ApacheCarbonPRBuilder/1925/



---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

[GitHub] incubator-carbondata pull request #882: [CARBONDATA-936] Parse partition tab...

Posted by lionelcao <gi...@git.apache.org>.
Github user lionelcao closed the pull request at:

    https://github.com/apache/incubator-carbondata/pull/882


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

[GitHub] incubator-carbondata pull request #882: [CARBONDATA-936] Parse partition tab...

Posted by jackylk <gi...@git.apache.org>.
Github user jackylk commented on a diff in the pull request:

    https://github.com/apache/incubator-carbondata/pull/882#discussion_r115162020
  
    --- Diff: examples/spark2/src/main/scala/org/apache/carbondata/examples/CarbonPartitionExample.scala ---
    @@ -0,0 +1,131 @@
    +/*
    + * 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.examples
    +
    +import java.io.File
    +
    +import org.apache.spark.sql.SparkSession
    +
    +import org.apache.carbondata.core.constants.CarbonCommonConstants
    +import org.apache.carbondata.core.util.CarbonProperties
    +
    +object CarbonPartitionExample {
    +
    +  def main(args: Array[String]) {
    +    val rootPath = new File(this.getClass.getResource("/").getPath
    +                            + "../../../..").getCanonicalPath
    +    val storeLocation = s"$rootPath/examples/spark2/target/store"
    +    val warehouse = s"$rootPath/examples/spark2/target/warehouse"
    +    val metastoredb = s"$rootPath/examples/spark2/target"
    +    val testData = s"$rootPath/examples/spark2/src/main/resources/partition_data.csv"
    +
    +    CarbonProperties.getInstance()
    +      .addProperty(CarbonCommonConstants.CARBON_TIMESTAMP_FORMAT, "yyyy/MM/dd")
    +
    +    import org.apache.spark.sql.CarbonSession._
    +
    +    val spark = SparkSession
    +      .builder()
    +      .master("local")
    +      .appName("CarbonPartitionExample")
    +      .config("spark.sql.warehouse.dir", warehouse)
    +      .getOrCreateCarbonSession(storeLocation, metastoredb)
    +
    +    spark.sparkContext.setLogLevel("WARN")
    +
    +    // none partition table
    +    spark.sql("DROP TABLE IF EXISTS t0")
    +
    +    spark.sql("""
    +                | CREATE TABLE IF NOT EXISTS t0
    +                | (
    +                | vin String,
    +                | logdate Timestamp,
    +                | phonenumber Long,
    +                | country String,
    +                | area String
    +                | )
    +                | STORED BY 'carbondata'
    +              """.stripMargin)
    +
    +    // range partition
    +    spark.sql("DROP TABLE IF EXISTS t1")
    +
    +    spark.sql("""
    +                | CREATE TABLE IF NOT EXISTS t1
    +                | (
    +                | vin String,
    +                | logdate Timestamp,
    +                | phonenumber Long,
    +                | country String,
    +                | area String
    +                | )
    +                | PARTITIONED BY (logdate Timestamp)
    --- End diff --
    
    I think this syntax is ok as of initial implementation, but I believe it needs further improvement in the future. One thing is that user should not provide data type twice. 


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

[GitHub] incubator-carbondata pull request #882: [CARBONDATA-936] Parse partition tab...

Posted by jackylk <gi...@git.apache.org>.
Github user jackylk commented on a diff in the pull request:

    https://github.com/apache/incubator-carbondata/pull/882#discussion_r115196558
  
    --- Diff: core/src/main/java/org/apache/carbondata/core/metadata/schema/PartitionInfo.java ---
    @@ -24,40 +24,29 @@
     import org.apache.carbondata.core.metadata.schema.table.column.ColumnSchema;
     
     /**
    - * Partition Information of carbon partition table
    + * Partition information of carbon partition table
      */
     public class PartitionInfo implements Serializable {
     
    -  /**
    -   * Partition columns
    -   */
       private List<ColumnSchema> columnSchemaList;
     
    -  /**
    -   * partition type
    -   */
       private PartitionType partitionType;
     
       /**
    -   * Range Partition definition
    +   * range infomation defined for range partition table
        */
       private List<String> rangeInfo;
     
       /**
    -   * List Partition definition
    +   * value list defined for list partition table
        */
       private List<List<String>> listInfo;
     
       /**
    -   * Hash Partition numbers
    +   * hash partition numbers
    --- End diff --
    
    This should be number of partition. 
    The variable name suggest to change to `numPartitions`, which is aligned with spark


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

[GitHub] incubator-carbondata pull request #882: [CARBONDATA-936] Parse partition tab...

Posted by jackylk <gi...@git.apache.org>.
Github user jackylk commented on a diff in the pull request:

    https://github.com/apache/incubator-carbondata/pull/882#discussion_r115163408
  
    --- Diff: integration/spark-common/src/main/scala/org/apache/spark/sql/catalyst/CarbonDDLSqlParser.scala ---
    @@ -347,43 +356,65 @@ abstract class CarbonDDLSqlParser extends AbstractCarbonSparkSQLParser {
       }
     
       /**
    -   * For getting the partitioner Object
    -   *
    +   * get partition info
        * @param partitionCols
        * @param tableProperties
    -   * @return
        */
    -  protected def getPartitionerObject(partitionCols: Seq[PartitionerField],
    -      tableProperties: Map[String, String]):
    -  Option[Partitioner] = {
    -
    -    // by default setting partition class empty.
    -    // later in table schema it is setting to default value.
    -    var partitionClass: String = ""
    -    var partitionCount: Int = 1
    -    var partitionColNames: Array[String] = Array[String]()
    -    if (tableProperties.get(CarbonCommonConstants.PARTITIONCLASS).isDefined) {
    -      partitionClass = tableProperties.get(CarbonCommonConstants.PARTITIONCLASS).get
    +  protected def getPartitionInfo(partitionCols: Seq[PartitionerField],
    +      tableProperties: Map[String, String]): Option[PartitionInfo] = {
    +    var partitionType: String = ""
    +    var hashNumber = 0
    +    var rangeInfo = List[String]()
    +    var listInfo = ListBuffer[List[String]]()
    +    var templist = ListBuffer[String]()
    +    if (tableProperties.get(CarbonCommonConstants.PARTITION_TYPE).isDefined) {
    +      partitionType = tableProperties.get(CarbonCommonConstants.PARTITION_TYPE).get
         }
    -
    -    if (tableProperties.get(CarbonCommonConstants.PARTITIONCOUNT).isDefined) {
    -      try {
    -        partitionCount = tableProperties.get(CarbonCommonConstants.PARTITIONCOUNT).get.toInt
    -      } catch {
    -        case e: Exception => // no need to do anything.
    +    if (tableProperties.get(CarbonCommonConstants.HASH_NUMBER).isDefined) {
    +      hashNumber = tableProperties.get(CarbonCommonConstants.HASH_NUMBER).get.toInt
    +    }
    +    if (tableProperties.get(CarbonCommonConstants.RANGE_INFO).isDefined) {
    +      rangeInfo = tableProperties.get(CarbonCommonConstants.RANGE_INFO).get.split(",")
    +        .map(_.trim()).toList
    +    }
    +    if (tableProperties.get(CarbonCommonConstants.LIST_INFO).isDefined) {
    +      val arr = tableProperties.get(CarbonCommonConstants.LIST_INFO).get.split(",")
    +        .map(_.trim())
    +      val iter = arr.iterator
    +      while(iter.hasNext) {
    --- End diff --
    
    add space after `while`


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

[GitHub] incubator-carbondata issue #882: [CARBONDATA-936] Parse partition table ddl

Posted by CarbonDataQA <gi...@git.apache.org>.
Github user CarbonDataQA commented on the issue:

    https://github.com/apache/incubator-carbondata/pull/882
  
    Build Success with Spark 1.6.2, Please check CI http://136.243.101.176:8080/job/ApacheCarbonPRBuilder/1930/



---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

[GitHub] incubator-carbondata issue #882: [CARBONDATA-936] Parse partition table ddl

Posted by CarbonDataQA <gi...@git.apache.org>.
Github user CarbonDataQA commented on the issue:

    https://github.com/apache/incubator-carbondata/pull/882
  
    Build Success with Spark 1.6.2, Please check CI http://136.243.101.176:8080/job/ApacheCarbonPRBuilder/1941/



---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---