You are viewing a plain text version of this content. The canonical link for it is here.
Posted to issues@carbondata.apache.org by xuchuanyin <gi...@git.apache.org> on 2018/08/10 14:09:48 UTC

[GitHub] carbondata pull request #2628: WIP: Support zstd as column compressor in fin...

GitHub user xuchuanyin opened a pull request:

    https://github.com/apache/carbondata/pull/2628

    WIP: Support zstd as column compressor in final store

    1. add zstd compressor for compressing column data
    2. add zstd support in thrift
    3. legacy store is not considered in this commit
    
    Be sure to do all of the following checklist to help us incorporate 
    your contribution quickly and easily:
    
     - [ ] Any interfaces changed?
     
     - [ ] Any backward compatibility impacted?
     
     - [ ] Document update required?
    
     - [ ] Testing done
            Please provide details on 
            - Whether new unit test cases have been added or why no new tests are required?
            - How it is tested? Please attach test report.
            - Is it a performance related change? Please attach the performance test report.
            - Any additional information to help reviewers in testing this change.
           
     - [ ] For large changes, please consider breaking it into sub-tasks under an umbrella JIRA. 
    


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

    $ git pull https://github.com/xuchuanyin/carbondata 0810_support_zstd_compressor_final_store

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

    https://github.com/apache/carbondata/pull/2628.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 #2628
    
----
commit c19c90852feaf33eaaca93a2bfdb3783c40159af
Author: xuchuanyin <xu...@...>
Date:   2018-08-10T14:02:57Z

    Support zstd as column compressor in final store
    
    1. add zstd compressor for compressing column data
    2. add zstd support in thrift
    3. legacy store is not considered in this commit

----


---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Failed with Spark 2.2.1, Please check CI http://95.216.28.178:8080/job/ApacheCarbonPRBuilder1/38/



---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Failed  with Spark 2.1.0, Please check CI http://136.243.101.176:8080/job/ApacheCarbonPRBuilder1/8277/



---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Success with Spark 2.1.0, Please check CI http://136.243.101.176:8080/job/ApacheCarbonPRBuilder1/7911/



---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Failed  with Spark 2.1.0, Please check CI http://136.243.101.176:8080/job/ApacheCarbonPRBuilder1/8251/



---

[GitHub] carbondata pull request #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zs...

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

    https://github.com/apache/carbondata/pull/2628#discussion_r212533250
  
    --- Diff: integration/spark-common-test/src/test/scala/org/apache/carbondata/integration/spark/testsuite/dataload/TestLoadDataWithCompressor.scala ---
    @@ -0,0 +1,180 @@
    +/*
    + * 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.integration.spark.testsuite.dataload
    +
    +import org.apache.spark.sql.Row
    +import org.apache.spark.sql.test.util.QueryTest
    +import org.scalatest.{BeforeAndAfterAll, BeforeAndAfterEach}
    +
    +import org.apache.carbondata.core.constants.CarbonCommonConstants
    +import org.apache.carbondata.core.util.CarbonProperties
    +
    +class TestLoadDataWithCompressor extends QueryTest with BeforeAndAfterEach with BeforeAndAfterAll {
    +  private val tableName = "load_test_with_compressor"
    +
    +  override protected def afterEach(): Unit = {
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.ENABLE_OFFHEAP_SORT,
    +      CarbonCommonConstants.ENABLE_OFFHEAP_SORT_DEFAULT)
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.COMPRESSOR,
    +      CarbonCommonConstants.DEFAULT_COMPRESSOR)
    +    sql(s"DROP TABLE IF EXISTS $tableName")
    +  }
    +
    +  private def createTable(): Unit = {
    +    sql(s"DROP TABLE IF EXISTS $tableName")
    +    sql(
    +      s"""
    +         | CREATE TABLE $tableName(
    +         |    booleanField boolean,
    +         |    shortField smallint,
    +         |    intField int,
    +         |    bigintField bigint,
    +         |    doubleField double,
    +         |    stringField string,
    +         |    timestampField timestamp,
    +         |    decimalField decimal(18,2),
    +         |    dateField date,
    +         |    charField string,
    +         |    floatField float,
    +         |    stringDictField string,
    +         |    stringSortField string,
    +         |    stringLocalDictField string,
    +         |    longStringField string
    +         | )
    +         | STORED BY 'carbondata'
    +         | TBLPROPERTIES(
    +         |  'LONG_STRING_COLUMNS'='longStringField',
    +         |  'SORT_COLUMNS'='stringSortField',
    +         |  'DICTIONARY_INCLUDE'='stringDictField',
    +         |  'local_dictionary_enable'='true',
    +         |  'local_dictionary_threshold'='10000',
    +         |  'local_dictionary_include'='stringLocalDictField')
    +       """.stripMargin)
    +  }
    +
    +  private def loadData(): Unit = {
    +    sql(
    +      s"""
    +         | INSERT INTO TABLE $tableName VALUES
    +         |  (true,1,11,101,41.4,'string1','2015/4/23 12:01:01',12.34,'2015/4/23','aaa',1.5,'dict1','sort1','local_dict1','longstring1'),
    +         | (false,2,12,102,42.4,'string2','2015/5/23 12:01:03',23.45,'2015/5/23','bbb',2.5,'dict2','sort2','local_dict2','longstring2'),
    +         |  (true,3,13,163,43.4,'string3','2015/7/26 12:01:06',34.56,'2015/7/26','ccc',3.5,'dict3','sort3','local_dict3','longstring3'),
    +         | (NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL)
    +       """.stripMargin)
    +    sql(
    +      s"""
    +         | INSERT INTO TABLE $tableName VALUES
    +         |  (true,${Short.MaxValue - 2},${Int.MinValue + 2},${Long.MaxValue - 2},${Double.MinValue + 2},'string1','2015/4/23 12:01:01',${Double.MinValue + 2},'2015/4/23','aaa',${Float.MaxValue - 2},'dict1','sort1','local_dict1','longstring1'),
    +         | (false,2,12,102,42.4,'string2','2015/5/23 12:01:03',23.45,'2015/5/23','bbb',2.5,'dict2','sort2','local_dict2','longstring2'),
    +         |  (true,${Short.MinValue + 2},${Int.MaxValue - 2},${Long.MinValue + 2},${Double.MaxValue - 2},'string3','2015/7/26 12:01:06',${Double.MinValue + 2},'2015/7/26','ccc',${Float.MinValue + 2},'dict3','sort3','local_dict3','longstring3'),
    +         | (NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL)
    +       """.stripMargin)
    +  }
    +
    +  private def testQuery(): Unit = {
    +    sql(s"SELECT * FROM $tableName").show(false)
    +    checkAnswer(sql(s"SELECT count(*) FROM $tableName"), Seq(Row(8)))
    +  }
    +
    +  test("test data loading with snappy compressor and offheap") {
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.ENABLE_OFFHEAP_SORT, "true")
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.COMPRESSOR, "snappy")
    +    createTable()
    +    loadData()
    +    testQuery()
    +  }
    +
    +  test("test data loading with zstd compressor and offheap") {
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.ENABLE_OFFHEAP_SORT, "true")
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.COMPRESSOR, "zstd")
    +    createTable()
    +    loadData()
    +    testQuery()
    +  }
    +
    +  test("test data loading with zstd compressor and onheap") {
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.ENABLE_OFFHEAP_SORT, "false")
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.COMPRESSOR, "zstd")
    +    createTable()
    +    loadData()
    +    testQuery()
    +  }
    +
    +  test("test current zstd compressor on legacy store with snappy") {
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.ENABLE_OFFHEAP_SORT, "true")
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.COMPRESSOR, "snappy")
    +    createTable()
    +    loadData()
    +
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.ENABLE_OFFHEAP_SORT, "true")
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.COMPRESSOR, "zstd")
    +    loadData()
    +    sql(s"SELECT * FROM $tableName").show(false)
    +    checkAnswer(sql(s"SELECT count(*) FROM $tableName"), Seq(Row(16)))
    +  }
    +
    +  test("test current snappy compressor on legacy store with zstd") {
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.ENABLE_OFFHEAP_SORT, "true")
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.COMPRESSOR, "zstd")
    +    createTable()
    +    loadData()
    +
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.ENABLE_OFFHEAP_SORT, "true")
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.COMPRESSOR, "snappy")
    +    loadData()
    +    sql(s"SELECT * FROM $tableName").show(false)
    +    checkAnswer(sql(s"SELECT count(*) FROM $tableName"), Seq(Row(16)))
    +  }
    +
    +  test("test compaction with different compressor for each load") {
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.ENABLE_OFFHEAP_SORT, "true")
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.COMPRESSOR, "zstd")
    +    createTable()
    +    loadData()
    +
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.ENABLE_OFFHEAP_SORT, "false")
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.COMPRESSOR, "snappy")
    +    loadData()
    +
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.ENABLE_OFFHEAP_SORT, "true")
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.COMPRESSOR, "zstd")
    +    loadData()
    +
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.ENABLE_OFFHEAP_SORT, "true")
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.COMPRESSOR, "snappy")
    +    loadData()
    +
    +    // there are 8 loads
    +    checkAnswer(sql(s"SELECT count(*) FROM $tableName"), Seq(Row(4 * 8)))
    +    assert(sql(s"SHOW SEGMENTS FOR TABLE $tableName").count() == 8)
    +    sql(s"ALTER TABLE $tableName COMPACT 'major'")
    +    sql(s"CLEAN FILES FOR TABLE $tableName")
    +    // after compaction and clean, there should be on segment
    +    checkAnswer(sql(s"SELECT count(*) FROM $tableName"), Seq(Row(4 * 8)))
    +    assert(sql(s"SHOW SEGMENTS FOR TABLE $tableName").count() == 1)
    +  }
    +
    +  test("test data loading with unsupported compressor and onheap") {
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.ENABLE_OFFHEAP_SORT, "false")
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.COMPRESSOR, "fake")
    +    createTable()
    +    val exception = intercept[Throwable] {
    --- End diff --
    
    optimized


---

[GitHub] carbondata pull request #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zs...

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

    https://github.com/apache/carbondata/pull/2628#discussion_r213940095
  
    --- Diff: core/src/main/java/org/apache/carbondata/core/datastore/page/ColumnPage.java ---
    @@ -66,15 +66,19 @@
       protected static final boolean unsafe = Boolean.parseBoolean(CarbonProperties.getInstance()
           .getProperty(CarbonCommonConstants.ENABLE_UNSAFE_COLUMN_PAGE,
               CarbonCommonConstants.ENABLE_UNSAFE_COLUMN_PAGE_DEFAULT));
    +  // compressor to be used for this column
    +  protected String columnCompressorName;
    --- End diff --
    
    ColumnPageEncoderMeta also has a compressor name variable.
    Is it necessary to use re-factory it?


---

[GitHub] carbondata pull request #2628: [CARBONDATA-2851] Support zstd as column comp...

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

    https://github.com/apache/carbondata/pull/2628


---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    LGTM


---

[GitHub] carbondata issue #2628: WIP: Support zstd as column compressor in final stor...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Failed with Spark 2.2.1, Please check CI http://88.99.58.216:8080/job/ApacheCarbonPRBuilder/6610/



---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Success with Spark 2.2.1, Please check CI http://88.99.58.216:8080/job/ApacheCarbonPRBuilder/6634/



---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    retest this please


---

[GitHub] carbondata issue #2628: WIP: Support zstd as column compressor in final stor...

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

    https://github.com/apache/carbondata/pull/2628
  
    SDV Build Success , Please check CI http://144.76.159.231:8080/job/ApacheSDVTests/6247/



---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Success with Spark 2.1.0, Please check CI http://136.243.101.176:8080/job/ApacheCarbonPRBuilder2.1/231/



---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Failed  with Spark 2.1.0, Please check CI http://136.243.101.176:8080/job/ApacheCarbonPRBuilder1/8159/



---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    SDV Build Fail , Please check CI http://144.76.159.231:8080/job/ApacheSDVTests/6265/



---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Failed  with Spark 2.1.0, Please check CI http://136.243.101.176:8080/job/ApacheCarbonPRBuilder1/8265/



---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Success with Spark 2.2.1, Please check CI http://95.216.28.178:8080/job/ApacheCarbonPRBuilder1/420/



---

[GitHub] carbondata pull request #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zs...

Posted by xuchuanyin <gi...@git.apache.org>.
GitHub user xuchuanyin reopened a pull request:

    https://github.com/apache/carbondata/pull/2628

    [CARBONDATA-2851][CARBONDATA-2852] Support zstd as column compressor in final store

    1. add zstd compressor for compressing column data
    2. add zstd support in thrift
    3. since zstd does not support zero-copy while compressing, offheap will
    not take effect for zstd
    4. Column compressor is configured through system property and can be changed in each load. During querying, carbondata will get the compressor information from metadata in the file data.
    5. This PR also considered and verified on the legacy store and compaction
    
    A simple test with 1.2GB raw CSV data shows that the size (in MB) of final store with different compressor: 
    
    | local dictionary | snappy | zstd | Size Reduced |
    | --- | --- | --- | -- |
    | local dict enabled | 335 | 207 | 38.2% |
    | local dict disabled | 375 | 225 | 40% |
    
    Be sure to do all of the following checklist to help us incorporate 
    your contribution quickly and easily:
    
     - [x] Any interfaces changed?
     `Yes, only internal used interfaces are changed`
     - [x] Any backward compatibility impacted?
     `Yes, backward compatibility is handled`
     - [x] Document update required?
    `Yes`
     - [x] Testing done
            Please provide details on 
            - Whether new unit test cases have been added or why no new tests are required?
    `Added tests`
            - How it is tested? Please attach test report.
    `Tested in local machine`
            - Is it a performance related change? Please attach the performance test report.
    `The size of final store has been decreased by 40% compared with default snappy`
            - Any additional information to help reviewers in testing this change.
    `NA`
           
     - [x] For large changes, please consider breaking it into sub-tasks under an umbrella JIRA. 
    `NA`


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

    $ git pull https://github.com/xuchuanyin/carbondata 0810_support_zstd_compressor_final_store

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

    https://github.com/apache/carbondata/pull/2628.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 #2628
    
----
commit e840c5bcfe7c27a5d2eb6459d7e391dac0a2091f
Author: xuchuanyin <xu...@...>
Date:   2018-08-10T14:02:57Z

    Support zstd as column compressor in final store
    
    1. add zstd compressor for compressing column data
    2. add zstd support in thrift
    3. legacy store is not considered in this commit
    4. since zstd does not support zero-copy while compressing, offheap will
    not take effect for zstd
    5. support lazy load for compressor

commit 926d64a245e76448d71d0557941d7e29559571c7
Author: xuchuanyin <xu...@...>
Date:   2018-08-13T13:45:42Z

    Support new compressor on legacy store
    
    In query procedure, we need to decompress the column page. Previously we
    get the compressor from system property. Now since we support new
    compressors, we should read the compressor information from the metadata
    in datafiles.
    This PR also solve the compatibility related problems on V1/V2 store where we
    only support snappy.

commit df0ca034b74cde02ec3568b8b5b94930a44c5763
Author: xuchuanyin <xu...@...>
Date:   2018-08-14T08:38:00Z

    fix comments

----


---

[GitHub] carbondata pull request #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zs...

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

    https://github.com/apache/carbondata/pull/2628


---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Failed with Spark 2.2.1, Please check CI http://95.216.28.178:8080/job/ApacheCarbonPRBuilder1/88/



---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Failed  with Spark 2.1.0, Please check CI http://136.243.101.176:8080/job/ApacheCarbonPRBuilder1/8024/



---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Success with Spark 2.1.0, Please check CI http://136.243.101.176:8080/job/ApacheCarbonPRBuilder2.1/242/



---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Failed with Spark 2.2.1, Please check CI http://95.216.28.178:8080/job/ApacheCarbonPRBuilder1/206/



---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    retest this please


---

[GitHub] carbondata issue #2628: [CARBONDATA-2851] Support zstd as column compressor ...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Success with Spark 2.2.1, Please check CI http://88.99.58.216:8080/job/ApacheCarbonPRBuilder/6621/



---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    SDV Build Fail , Please check CI http://144.76.159.231:8080/job/ApacheSDVTests/6476/



---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Success with Spark 2.1.0, Please check CI http://136.243.101.176:8080/job/ApacheCarbonPRBuilder2.1/26/



---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    retest this please


---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    SDV Build Fail , Please check CI http://144.76.159.231:8080/job/ApacheSDVTests/6425/



---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    @jackylk 
    We do not have the environment to test this now.
    The decompression speed of Zstd will be worse than Snappy based on Zstd's official document. But still we do not know how much does it affect the query.


---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    retest this please


---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Failed with Spark 2.2.1, Please check CI http://95.216.28.178:8080/job/ApacheCarbonPRBuilder1/100/



---

[GitHub] carbondata pull request #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zs...

Posted by xuchuanyin <gi...@git.apache.org>.
GitHub user xuchuanyin reopened a pull request:

    https://github.com/apache/carbondata/pull/2628

    [CARBONDATA-2851][CARBONDATA-2852] Support zstd as column compressor in final store

    1. add zstd compressor for compressing column data
    2. add zstd support in thrift
    3. since zstd does not support zero-copy while compressing, offheap will not take effect for zstd
    4. Column compressor is configured through system property and can be changed in each load. Before loading, Carbondata will get the compressor and use that compressor during that loading. During querying, carbondata will get the compressor information from metadata in the file data.
    5. Also support compressing streaming table using zstd. The compressor info is stored in FileHeader of the streaming file.
    6. This PR also considered and verified on the legacy store and compaction
    
    A simple test with 1.2GB raw CSV data shows that the size (in MB) of final store with different compressor: 
    
    | local dictionary | snappy | zstd | Size Reduced |
    | --- | --- | --- | -- |
    | enabled | 335 | 207 | 38.2% |
    | disabled | 375 | 225 | 40% |
    
    Be sure to do all of the following checklist to help us incorporate 
    your contribution quickly and easily:
    
     - [x] Any interfaces changed?
     `Yes, only internal used interfaces are changed`
     - [x] Any backward compatibility impacted?
     `Yes, backward compatibility is handled`
     - [x] Document update required?
    `Yes`
     - [x] Testing done
            Please provide details on 
            - Whether new unit test cases have been added or why no new tests are required?
    `Added tests`
            - How it is tested? Please attach test report.
    `Tested in local machine`
            - Is it a performance related change? Please attach the performance test report.
    `The size of final store has been decreased by 40% compared with default snappy`
            - Any additional information to help reviewers in testing this change.
    `NA`
           
     - [x] For large changes, please consider breaking it into sub-tasks under an umbrella JIRA. 
    `NA`


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

    $ git pull https://github.com/xuchuanyin/carbondata 0810_support_zstd_compressor_final_store

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

    https://github.com/apache/carbondata/pull/2628.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 #2628
    
----
commit c171ee13136785110f6fff8104afebc4b2f222c7
Author: xuchuanyin <xu...@...>
Date:   2018-08-10T14:02:57Z

    Support zstd as column compressor in final store
    
    1. add zstd compressor for compressing column data
    2. add zstd support in thrift
    3. legacy store is not considered in this commit
    4. since zstd does not support zero-copy while compressing, offheap will
    not take effect for zstd
    5. support lazy load for compressor

commit 6448e6f21da66172775b625730b922fdfa57822d
Author: xuchuanyin <xu...@...>
Date:   2018-08-13T13:45:42Z

    Support new compressor on legacy store
    
    In query procedure, we need to decompress the column page. Previously we
    get the compressor from system property. Now since we support new
    compressors, we should read the compressor information from the metadata
    in datafiles.
    This PR also solve the compatibility related problems on V1/V2 store where we
    only support snappy.

commit 2815c84f1d5fd99ff37ba6890d98fb2b73a95b00
Author: xuchuanyin <xu...@...>
Date:   2018-08-14T08:38:00Z

    fix comments

commit ac95c25fca1c37f10f9cce0db76062207d0d3cee
Author: xuchuanyin <xu...@...>
Date:   2018-08-23T09:35:23Z

    Determine the column compressor before data loading
    
    we will get the column compressor before data loading/compaction start,
    so that it can make all the pages use the same compressor in case of
    concurrent modifying compressor during loading.

commit a672d3baad1c476308c0aec5133e418afeaeacb2
Author: xuchuanyin <xu...@...>
Date:   2018-08-27T11:18:30Z

    set compressor in carbon load model
    
    column compressor is necessary for carbon load model, otherwise load
    will fail.

commit d05c1cc38e1fa42ef94f70577ee2a715f649ebe3
Author: xuchuanyin <xu...@...>
Date:   2018-08-30T04:02:33Z

    fix error in test

commit fb8cdfb1258b477f7d8c867ee74bd59386725d9c
Author: xuchuanyin <xu...@...>
Date:   2018-09-03T03:58:02Z

    fix review comments
    
    optimize parameters for column page, use columnPageEncodeMeta instead of
    its members

----


---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Failed  with Spark 2.3.1, Please check CI http://136.243.101.176:8080/job/ApacheCarbonPRBuilder2.3/8342/



---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Success with Spark 2.1.0, Please check CI http://136.243.101.176:8080/job/ApacheCarbonPRBuilder2.1/248/



---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Success with Spark 2.1.0, Please check CI http://136.243.101.176:8080/job/ApacheCarbonPRBuilder2.1/233/



---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Success with Spark 2.1.0, Please check CI http://136.243.101.176:8080/job/ApacheCarbonPRBuilder2.1/251/



---

[GitHub] carbondata issue #2628: WIP: Support zstd as column compressor in final stor...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Failed  with Spark 2.1.0, Please check CI http://136.243.101.176:8080/job/ApacheCarbonPRBuilder1/7886/



---

[GitHub] carbondata pull request #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zs...

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

    https://github.com/apache/carbondata/pull/2628#discussion_r212533227
  
    --- Diff: integration/spark-common-test/src/test/scala/org/apache/carbondata/integration/spark/testsuite/dataload/TestLoadDataWithCompressor.scala ---
    @@ -0,0 +1,180 @@
    +/*
    + * 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.integration.spark.testsuite.dataload
    +
    +import org.apache.spark.sql.Row
    +import org.apache.spark.sql.test.util.QueryTest
    +import org.scalatest.{BeforeAndAfterAll, BeforeAndAfterEach}
    +
    +import org.apache.carbondata.core.constants.CarbonCommonConstants
    +import org.apache.carbondata.core.util.CarbonProperties
    +
    +class TestLoadDataWithCompressor extends QueryTest with BeforeAndAfterEach with BeforeAndAfterAll {
    +  private val tableName = "load_test_with_compressor"
    +
    +  override protected def afterEach(): Unit = {
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.ENABLE_OFFHEAP_SORT,
    +      CarbonCommonConstants.ENABLE_OFFHEAP_SORT_DEFAULT)
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.COMPRESSOR,
    +      CarbonCommonConstants.DEFAULT_COMPRESSOR)
    +    sql(s"DROP TABLE IF EXISTS $tableName")
    +  }
    +
    +  private def createTable(): Unit = {
    +    sql(s"DROP TABLE IF EXISTS $tableName")
    +    sql(
    +      s"""
    +         | CREATE TABLE $tableName(
    +         |    booleanField boolean,
    +         |    shortField smallint,
    +         |    intField int,
    +         |    bigintField bigint,
    +         |    doubleField double,
    +         |    stringField string,
    +         |    timestampField timestamp,
    +         |    decimalField decimal(18,2),
    +         |    dateField date,
    +         |    charField string,
    +         |    floatField float,
    +         |    stringDictField string,
    +         |    stringSortField string,
    +         |    stringLocalDictField string,
    +         |    longStringField string
    +         | )
    +         | STORED BY 'carbondata'
    +         | TBLPROPERTIES(
    +         |  'LONG_STRING_COLUMNS'='longStringField',
    +         |  'SORT_COLUMNS'='stringSortField',
    +         |  'DICTIONARY_INCLUDE'='stringDictField',
    +         |  'local_dictionary_enable'='true',
    +         |  'local_dictionary_threshold'='10000',
    +         |  'local_dictionary_include'='stringLocalDictField')
    +       """.stripMargin)
    +  }
    +
    +  private def loadData(): Unit = {
    +    sql(
    +      s"""
    +         | INSERT INTO TABLE $tableName VALUES
    +         |  (true,1,11,101,41.4,'string1','2015/4/23 12:01:01',12.34,'2015/4/23','aaa',1.5,'dict1','sort1','local_dict1','longstring1'),
    +         | (false,2,12,102,42.4,'string2','2015/5/23 12:01:03',23.45,'2015/5/23','bbb',2.5,'dict2','sort2','local_dict2','longstring2'),
    +         |  (true,3,13,163,43.4,'string3','2015/7/26 12:01:06',34.56,'2015/7/26','ccc',3.5,'dict3','sort3','local_dict3','longstring3'),
    +         | (NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL)
    +       """.stripMargin)
    +    sql(
    +      s"""
    +         | INSERT INTO TABLE $tableName VALUES
    +         |  (true,${Short.MaxValue - 2},${Int.MinValue + 2},${Long.MaxValue - 2},${Double.MinValue + 2},'string1','2015/4/23 12:01:01',${Double.MinValue + 2},'2015/4/23','aaa',${Float.MaxValue - 2},'dict1','sort1','local_dict1','longstring1'),
    +         | (false,2,12,102,42.4,'string2','2015/5/23 12:01:03',23.45,'2015/5/23','bbb',2.5,'dict2','sort2','local_dict2','longstring2'),
    +         |  (true,${Short.MinValue + 2},${Int.MaxValue - 2},${Long.MinValue + 2},${Double.MaxValue - 2},'string3','2015/7/26 12:01:06',${Double.MinValue + 2},'2015/7/26','ccc',${Float.MinValue + 2},'dict3','sort3','local_dict3','longstring3'),
    +         | (NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL)
    +       """.stripMargin)
    +  }
    +
    +  private def testQuery(): Unit = {
    +    sql(s"SELECT * FROM $tableName").show(false)
    +    checkAnswer(sql(s"SELECT count(*) FROM $tableName"), Seq(Row(8)))
    +  }
    +
    +  test("test data loading with snappy compressor and offheap") {
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.ENABLE_OFFHEAP_SORT, "true")
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.COMPRESSOR, "snappy")
    +    createTable()
    +    loadData()
    --- End diff --
    
    tests added


---

[GitHub] carbondata pull request #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zs...

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

    https://github.com/apache/carbondata/pull/2628#discussion_r213937956
  
    --- Diff: core/src/main/java/org/apache/carbondata/core/datastore/page/ColumnPage.java ---
    @@ -94,222 +98,240 @@ public void setStatsCollector(ColumnPageStatsCollector statsCollector) {
       }
     
       private static ColumnPage createDecimalPage(TableSpec.ColumnSpec columnSpec, DataType dataType,
    -      int pageSize) {
    +      int pageSize, String compressorName) {
         if (unsafe) {
           try {
    -        return new UnsafeDecimalColumnPage(columnSpec, dataType, pageSize);
    +        return new UnsafeDecimalColumnPage(columnSpec, dataType, pageSize, compressorName);
           } catch (MemoryException e) {
             throw new RuntimeException(e);
           }
         } else {
    -      return new SafeDecimalColumnPage(columnSpec, dataType, pageSize);
    +      return new SafeDecimalColumnPage(columnSpec, dataType, pageSize, compressorName);
         }
       }
     
       private static ColumnPage createVarLengthPage(TableSpec.ColumnSpec columnSpec, DataType dataType,
    -      int pageSize) {
    +      int pageSize, String compressorName) {
         if (unsafe) {
           try {
    -        return new UnsafeVarLengthColumnPage(columnSpec, dataType, pageSize);
    +        return new UnsafeVarLengthColumnPage(columnSpec, dataType, pageSize, compressorName);
           } catch (MemoryException e) {
             throw new RuntimeException(e);
           }
         } else {
    -      return new SafeVarLengthColumnPage(columnSpec, dataType, pageSize);
    +      return new SafeVarLengthColumnPage(columnSpec, dataType, pageSize, compressorName);
         }
       }
     
       private static ColumnPage createFixLengthPage(TableSpec.ColumnSpec columnSpec, DataType dataType,
    -      int pageSize) {
    +      int pageSize, String compressorName) {
         if (unsafe) {
           try {
    -        return new UnsafeFixLengthColumnPage(columnSpec, dataType, pageSize);
    +        return new UnsafeFixLengthColumnPage(columnSpec, dataType, pageSize, compressorName);
           } catch (MemoryException e) {
             throw new RuntimeException(e);
           }
         } else {
    -      return new SafeFixLengthColumnPage(columnSpec, dataType, pageSize);
    +      return new SafeFixLengthColumnPage(columnSpec, dataType, pageSize, compressorName);
         }
       }
     
       private static ColumnPage createFixLengthByteArrayPage(TableSpec.ColumnSpec columnSpec,
    -      DataType dataType, int pageSize, int eachValueSize) {
    +      DataType dataType, int pageSize, int eachValueSize, String compressorName) {
         if (unsafe) {
           try {
    -        return new UnsafeFixLengthColumnPage(columnSpec, dataType, pageSize, eachValueSize);
    +        return new UnsafeFixLengthColumnPage(columnSpec, dataType, pageSize, eachValueSize,
    +            compressorName);
           } catch (MemoryException e) {
             throw new RuntimeException(e);
           }
         } else {
    -      return new SafeFixLengthColumnPage(columnSpec, dataType, pageSize);
    +      return new SafeFixLengthColumnPage(columnSpec, dataType, pageSize, compressorName);
         }
       }
     
       private static ColumnPage createPage(TableSpec.ColumnSpec columnSpec, DataType dataType,
    -      int pageSize) {
    +      int pageSize, String compressorName) {
         if (DataTypes.isDecimal(dataType)) {
    -      return createDecimalPage(columnSpec, dataType, pageSize);
    +      return createDecimalPage(columnSpec, dataType, pageSize, compressorName);
         } else if (dataType.equals(BYTE_ARRAY)) {
    -      return createVarLengthPage(columnSpec, dataType, pageSize);
    +      return createVarLengthPage(columnSpec, dataType, pageSize, compressorName);
         } else {
    -      return createFixLengthPage(columnSpec, dataType, pageSize);
    +      return createFixLengthPage(columnSpec, dataType, pageSize, compressorName);
         }
       }
     
       public static ColumnPage newDecimalPage(TableSpec.ColumnSpec columnSpec, DataType dataType,
    -      int pageSize)
    +      int pageSize, String compressor)
         throws MemoryException {
    -    return newPage(columnSpec, dataType, pageSize);
    +    return newPage(columnSpec, dataType, pageSize, compressor);
       }
     
       public static ColumnPage newLocalDictPage(TableSpec.ColumnSpec columnSpec, DataType dataType,
           int pageSize, LocalDictionaryGenerator localDictionaryGenerator,
    -      boolean isComplexTypePrimitive) throws MemoryException {
    +      boolean isComplexTypePrimitive, String compressorName) throws MemoryException {
         if (unsafe) {
    -      return new LocalDictColumnPage(new UnsafeVarLengthColumnPage(columnSpec, dataType, pageSize),
    -          new UnsafeFixLengthColumnPage(columnSpec, DataTypes.BYTE_ARRAY, pageSize,
    -              CarbonCommonConstants.LOCAL_DICT_ENCODED_BYTEARRAY_SIZE),
    -          localDictionaryGenerator, isComplexTypePrimitive);
    +      ColumnPage actualPage =
    +          new UnsafeVarLengthColumnPage(columnSpec, dataType, pageSize, compressorName);
    +      ColumnPage encodedPage = new UnsafeFixLengthColumnPage(columnSpec, DataTypes.BYTE_ARRAY,
    +          pageSize, CarbonCommonConstants.LOCAL_DICT_ENCODED_BYTEARRAY_SIZE, compressorName);
    +      return new LocalDictColumnPage(actualPage, encodedPage, localDictionaryGenerator,
    +          isComplexTypePrimitive, compressorName);
         } else {
    -      return new LocalDictColumnPage(new SafeVarLengthColumnPage(columnSpec, dataType, pageSize),
    -          new SafeFixLengthColumnPage(columnSpec, DataTypes.BYTE_ARRAY, pageSize),
    -          localDictionaryGenerator, isComplexTypePrimitive);
    +      ColumnPage actualPage =
    +          new SafeVarLengthColumnPage(columnSpec, dataType, pageSize, compressorName);
    +      ColumnPage encodedPage =
    +          new SafeFixLengthColumnPage(columnSpec, DataTypes.BYTE_ARRAY, pageSize, compressorName);
    +      return new LocalDictColumnPage(actualPage, encodedPage, localDictionaryGenerator,
    +          isComplexTypePrimitive, compressorName);
         }
       }
     
       /**
        * Create a new page of dataType and number of row = pageSize
        */
       public static ColumnPage newPage(TableSpec.ColumnSpec columnSpec, DataType dataType,
    --- End diff --
    
    how about to change the first parameter to ColumnPageEncoderMeta instead of adding an extra parameter?


---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Failed with Spark 2.2.1, Please check CI http://88.99.58.216:8080/job/ApacheCarbonPRBuilder/6748/



---

[GitHub] carbondata issue #2628: WIP: Support zstd as column compressor in final stor...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Failed  with Spark 2.1.0, Please check CI http://136.243.101.176:8080/job/ApacheCarbonPRBuilder1/7885/



---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Failed  with Spark 2.3.1, Please check CI http://136.243.101.176:8080/job/ApacheCarbonPRBuilder2.3/8488/



---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Failed with Spark 2.2.1, Please check CI http://95.216.28.178:8080/job/ApacheCarbonPRBuilder1/94/



---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    @xuchuanyin I feel it is very necessary to save compressor name in thrift instead of enum. It will not be a good idea to change thrift for every compression support and also it limits the user to give their custom compressor interface while creating the table.


---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Success with Spark 2.2.1, Please check CI http://95.216.28.178:8080/job/ApacheCarbonPRBuilder1/409/



---

[GitHub] carbondata pull request #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zs...

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

    https://github.com/apache/carbondata/pull/2628#discussion_r214560605
  
    --- Diff: core/src/main/java/org/apache/carbondata/core/util/BlockletDataMapUtil.java ---
    @@ -357,8 +357,8 @@ private static boolean isSameColumnSchemaList(List<ColumnSchema> indexFileColumn
           columnSchema.write(dataOutput);
         }
         byte[] byteArray = stream.toByteArray();
    -    // Compress with snappy to reduce the size of schema
    -    return CompressorFactory.getInstance().getCompressor().compressByte(byteArray);
    +    // Compress to reduce the size of schema
    +    return CompressorFactory.getInstance().getCompressor("snappy").compressByte(byteArray);
    --- End diff --
    
    OK~


---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Failed  with Spark 2.1.0, Please check CI http://136.243.101.176:8080/job/ApacheCarbonPRBuilder1/8097/



---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    @QiangCai Fixed.
    We will get and validate the compressor before loading and will use that compressor during that loading procedure.
    Tests are added to verify this: during data loading, we will dynamically change the compressor in property and the test is OK.


---

[GitHub] carbondata pull request #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zs...

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

    https://github.com/apache/carbondata/pull/2628#discussion_r210782737
  
    --- Diff: integration/spark-common-test/src/test/scala/org/apache/carbondata/integration/spark/testsuite/dataload/TestLoadDataWithCompressor.scala ---
    @@ -0,0 +1,180 @@
    +/*
    + * 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.integration.spark.testsuite.dataload
    +
    +import org.apache.spark.sql.Row
    +import org.apache.spark.sql.test.util.QueryTest
    +import org.scalatest.{BeforeAndAfterAll, BeforeAndAfterEach}
    +
    +import org.apache.carbondata.core.constants.CarbonCommonConstants
    +import org.apache.carbondata.core.util.CarbonProperties
    +
    +class TestLoadDataWithCompressor extends QueryTest with BeforeAndAfterEach with BeforeAndAfterAll {
    +  private val tableName = "load_test_with_compressor"
    +
    +  override protected def afterEach(): Unit = {
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.ENABLE_OFFHEAP_SORT,
    +      CarbonCommonConstants.ENABLE_OFFHEAP_SORT_DEFAULT)
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.COMPRESSOR,
    +      CarbonCommonConstants.DEFAULT_COMPRESSOR)
    +    sql(s"DROP TABLE IF EXISTS $tableName")
    +  }
    +
    +  private def createTable(): Unit = {
    +    sql(s"DROP TABLE IF EXISTS $tableName")
    +    sql(
    +      s"""
    +         | CREATE TABLE $tableName(
    +         |    booleanField boolean,
    +         |    shortField smallint,
    +         |    intField int,
    +         |    bigintField bigint,
    +         |    doubleField double,
    +         |    stringField string,
    +         |    timestampField timestamp,
    +         |    decimalField decimal(18,2),
    +         |    dateField date,
    +         |    charField string,
    +         |    floatField float,
    +         |    stringDictField string,
    +         |    stringSortField string,
    +         |    stringLocalDictField string,
    +         |    longStringField string
    +         | )
    +         | STORED BY 'carbondata'
    +         | TBLPROPERTIES(
    +         |  'LONG_STRING_COLUMNS'='longStringField',
    +         |  'SORT_COLUMNS'='stringSortField',
    +         |  'DICTIONARY_INCLUDE'='stringDictField',
    +         |  'local_dictionary_enable'='true',
    +         |  'local_dictionary_threshold'='10000',
    +         |  'local_dictionary_include'='stringLocalDictField')
    +       """.stripMargin)
    +  }
    +
    +  private def loadData(): Unit = {
    +    sql(
    +      s"""
    +         | INSERT INTO TABLE $tableName VALUES
    +         |  (true,1,11,101,41.4,'string1','2015/4/23 12:01:01',12.34,'2015/4/23','aaa',1.5,'dict1','sort1','local_dict1','longstring1'),
    +         | (false,2,12,102,42.4,'string2','2015/5/23 12:01:03',23.45,'2015/5/23','bbb',2.5,'dict2','sort2','local_dict2','longstring2'),
    +         |  (true,3,13,163,43.4,'string3','2015/7/26 12:01:06',34.56,'2015/7/26','ccc',3.5,'dict3','sort3','local_dict3','longstring3'),
    +         | (NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL)
    +       """.stripMargin)
    +    sql(
    +      s"""
    +         | INSERT INTO TABLE $tableName VALUES
    +         |  (true,${Short.MaxValue - 2},${Int.MinValue + 2},${Long.MaxValue - 2},${Double.MinValue + 2},'string1','2015/4/23 12:01:01',${Double.MinValue + 2},'2015/4/23','aaa',${Float.MaxValue - 2},'dict1','sort1','local_dict1','longstring1'),
    +         | (false,2,12,102,42.4,'string2','2015/5/23 12:01:03',23.45,'2015/5/23','bbb',2.5,'dict2','sort2','local_dict2','longstring2'),
    +         |  (true,${Short.MinValue + 2},${Int.MaxValue - 2},${Long.MinValue + 2},${Double.MaxValue - 2},'string3','2015/7/26 12:01:06',${Double.MinValue + 2},'2015/7/26','ccc',${Float.MinValue + 2},'dict3','sort3','local_dict3','longstring3'),
    +         | (NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL)
    +       """.stripMargin)
    +  }
    +
    +  private def testQuery(): Unit = {
    +    sql(s"SELECT * FROM $tableName").show(false)
    +    checkAnswer(sql(s"SELECT count(*) FROM $tableName"), Seq(Row(8)))
    +  }
    +
    +  test("test data loading with snappy compressor and offheap") {
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.ENABLE_OFFHEAP_SORT, "true")
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.COMPRESSOR, "snappy")
    +    createTable()
    +    loadData()
    +    testQuery()
    +  }
    +
    +  test("test data loading with zstd compressor and offheap") {
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.ENABLE_OFFHEAP_SORT, "true")
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.COMPRESSOR, "zstd")
    +    createTable()
    +    loadData()
    +    testQuery()
    +  }
    +
    +  test("test data loading with zstd compressor and onheap") {
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.ENABLE_OFFHEAP_SORT, "false")
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.COMPRESSOR, "zstd")
    +    createTable()
    +    loadData()
    +    testQuery()
    +  }
    +
    +  test("test current zstd compressor on legacy store with snappy") {
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.ENABLE_OFFHEAP_SORT, "true")
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.COMPRESSOR, "snappy")
    +    createTable()
    +    loadData()
    +
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.ENABLE_OFFHEAP_SORT, "true")
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.COMPRESSOR, "zstd")
    +    loadData()
    --- End diff --
    
    It is better to add loading option also if you want to specify the compressor for every load


---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Failed with Spark 2.2.1, Please check CI http://95.216.28.178:8080/job/ApacheCarbonPRBuilder1/204/



---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Success with Spark 2.1.0, Please check CI http://136.243.101.176:8080/job/ApacheCarbonPRBuilder2.1/240/



---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    SDV Build Fail , Please check CI http://144.76.159.231:8080/job/ApacheSDVTests/6460/



---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    retest this please


---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Failed with Spark 2.2.1, Please check CI http://95.216.28.178:8080/job/ApacheCarbonPRBuilder1/254/



---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Failed  with Spark 2.3.1, Please check CI http://136.243.101.176:8080/job/ApacheCarbonPRBuilder2.3/8465/



---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Failed with Spark 2.2.1, Please check CI http://95.216.28.178:8080/job/ApacheCarbonPRBuilder1/418/



---

[GitHub] carbondata pull request #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zs...

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

    https://github.com/apache/carbondata/pull/2628#discussion_r213923265
  
    --- Diff: core/src/main/java/org/apache/carbondata/core/datastore/compression/CompressorFactory.java ---
    @@ -17,42 +17,99 @@
     
     package org.apache.carbondata.core.datastore.compression;
     
    +import java.util.Arrays;
    +import java.util.HashMap;
    +import java.util.Map;
    +
     import org.apache.carbondata.core.constants.CarbonCommonConstants;
     import org.apache.carbondata.core.util.CarbonProperties;
    +import org.apache.carbondata.format.CompressionCodec;
     
     public class CompressorFactory {
    -
       private static final CompressorFactory COMPRESSOR_FACTORY = new CompressorFactory();
     
    -  private final Compressor snappyCompressor;
    +  private final Map<String, SupportedCompressor> compressors = new HashMap<>();
    +
    +  public enum SupportedCompressor {
    +    SNAPPY(CompressionCodec.SNAPPY, "snappy", SnappyCompressor.class),
    +    ZSTD(CompressionCodec.ZSTD, "zstd", ZstdCompressor.class);
    +
    +    private CompressionCodec codec;
    +    private String name;
    +    private Class<Compressor> compressorClass;
    +    private transient Compressor compressor;
    +
    +    SupportedCompressor(CompressionCodec codec, String name, Class compressorCls) {
    +      this.codec = codec;
    +      this.name = name;
    +      this.compressorClass = compressorCls;
    +    }
    +
    +    public CompressionCodec getCodec() {
    +      return codec;
    +    }
    +
    +    public String getName() {
    +      return name;
    +    }
    +
    +    /**
    +     * we will load the compressor only if it is needed
    +     */
    +    public Compressor getCompressor() {
    +      if (this.compressor == null) {
    +        try {
    +          this.compressor = compressorClass.newInstance();
    +        } catch (InstantiationException | IllegalAccessException e) {
    +          throw new RuntimeException("Exception occurs while getting compressor for " + name);
    +        }
    +      }
    +      return this.compressor;
    +    }
    +  }
     
       private CompressorFactory() {
    -    String compressorType = CarbonProperties.getInstance()
    -        .getProperty(CarbonCommonConstants.COMPRESSOR, CarbonCommonConstants.DEFAULT_COMPRESSOR);
    -    switch (compressorType) {
    -      case "snappy":
    -        snappyCompressor = new SnappyCompressor();
    -        break;
    -      default:
    -        throw new RuntimeException(
    -            "Invalid compressor type provided! Please provide valid compressor type");
    +    for (SupportedCompressor supportedCompressor : SupportedCompressor.values()) {
    +      compressors.put(supportedCompressor.getName(), supportedCompressor);
         }
       }
     
       public static CompressorFactory getInstance() {
         return COMPRESSOR_FACTORY;
       }
     
    +  /**
    +   * get the default compressor.
    +   * This method can only be called in data load procedure to compress column page.
    +   * In query procedure, we should read the compressor information from the metadata
    +   * in datafiles when we want to decompress the content.
    +   */
       public Compressor getCompressor() {
    -    return getCompressor(CarbonCommonConstants.DEFAULT_COMPRESSOR);
    +    String compressorType = CarbonProperties.getInstance()
    --- End diff --
    
    Is compressorType in case sensitive?


---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    SDV Build Fail , Please check CI http://144.76.159.231:8080/job/ApacheSDVTests/6416/



---

[GitHub] carbondata pull request #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zs...

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

    https://github.com/apache/carbondata/pull/2628


---

[GitHub] carbondata issue #2628: [CARBONDATA-2851] Support zstd as column compressor ...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Success with Spark 2.1.0, Please check CI http://136.243.101.176:8080/job/ApacheCarbonPRBuilder1/7897/



---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    retest this please


---

[GitHub] carbondata pull request #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zs...

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

    https://github.com/apache/carbondata/pull/2628#discussion_r211845798
  
    --- Diff: integration/spark-common-test/src/test/scala/org/apache/carbondata/integration/spark/testsuite/dataload/TestLoadDataWithCompressor.scala ---
    @@ -0,0 +1,180 @@
    +/*
    + * 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.integration.spark.testsuite.dataload
    +
    +import org.apache.spark.sql.Row
    +import org.apache.spark.sql.test.util.QueryTest
    +import org.scalatest.{BeforeAndAfterAll, BeforeAndAfterEach}
    +
    +import org.apache.carbondata.core.constants.CarbonCommonConstants
    +import org.apache.carbondata.core.util.CarbonProperties
    +
    +class TestLoadDataWithCompressor extends QueryTest with BeforeAndAfterEach with BeforeAndAfterAll {
    +  private val tableName = "load_test_with_compressor"
    +
    +  override protected def afterEach(): Unit = {
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.ENABLE_OFFHEAP_SORT,
    +      CarbonCommonConstants.ENABLE_OFFHEAP_SORT_DEFAULT)
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.COMPRESSOR,
    +      CarbonCommonConstants.DEFAULT_COMPRESSOR)
    +    sql(s"DROP TABLE IF EXISTS $tableName")
    +  }
    +
    +  private def createTable(): Unit = {
    +    sql(s"DROP TABLE IF EXISTS $tableName")
    +    sql(
    +      s"""
    +         | CREATE TABLE $tableName(
    +         |    booleanField boolean,
    +         |    shortField smallint,
    +         |    intField int,
    +         |    bigintField bigint,
    +         |    doubleField double,
    +         |    stringField string,
    +         |    timestampField timestamp,
    +         |    decimalField decimal(18,2),
    +         |    dateField date,
    +         |    charField string,
    +         |    floatField float,
    +         |    stringDictField string,
    +         |    stringSortField string,
    +         |    stringLocalDictField string,
    +         |    longStringField string
    +         | )
    +         | STORED BY 'carbondata'
    +         | TBLPROPERTIES(
    +         |  'LONG_STRING_COLUMNS'='longStringField',
    +         |  'SORT_COLUMNS'='stringSortField',
    +         |  'DICTIONARY_INCLUDE'='stringDictField',
    +         |  'local_dictionary_enable'='true',
    +         |  'local_dictionary_threshold'='10000',
    +         |  'local_dictionary_include'='stringLocalDictField')
    +       """.stripMargin)
    +  }
    +
    +  private def loadData(): Unit = {
    +    sql(
    +      s"""
    +         | INSERT INTO TABLE $tableName VALUES
    +         |  (true,1,11,101,41.4,'string1','2015/4/23 12:01:01',12.34,'2015/4/23','aaa',1.5,'dict1','sort1','local_dict1','longstring1'),
    +         | (false,2,12,102,42.4,'string2','2015/5/23 12:01:03',23.45,'2015/5/23','bbb',2.5,'dict2','sort2','local_dict2','longstring2'),
    +         |  (true,3,13,163,43.4,'string3','2015/7/26 12:01:06',34.56,'2015/7/26','ccc',3.5,'dict3','sort3','local_dict3','longstring3'),
    +         | (NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL)
    +       """.stripMargin)
    +    sql(
    +      s"""
    +         | INSERT INTO TABLE $tableName VALUES
    +         |  (true,${Short.MaxValue - 2},${Int.MinValue + 2},${Long.MaxValue - 2},${Double.MinValue + 2},'string1','2015/4/23 12:01:01',${Double.MinValue + 2},'2015/4/23','aaa',${Float.MaxValue - 2},'dict1','sort1','local_dict1','longstring1'),
    +         | (false,2,12,102,42.4,'string2','2015/5/23 12:01:03',23.45,'2015/5/23','bbb',2.5,'dict2','sort2','local_dict2','longstring2'),
    +         |  (true,${Short.MinValue + 2},${Int.MaxValue - 2},${Long.MinValue + 2},${Double.MaxValue - 2},'string3','2015/7/26 12:01:06',${Double.MinValue + 2},'2015/7/26','ccc',${Float.MinValue + 2},'dict3','sort3','local_dict3','longstring3'),
    +         | (NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL)
    +       """.stripMargin)
    +  }
    +
    +  private def testQuery(): Unit = {
    +    sql(s"SELECT * FROM $tableName").show(false)
    +    checkAnswer(sql(s"SELECT count(*) FROM $tableName"), Seq(Row(8)))
    +  }
    +
    +  test("test data loading with snappy compressor and offheap") {
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.ENABLE_OFFHEAP_SORT, "true")
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.COMPRESSOR, "snappy")
    +    createTable()
    +    loadData()
    +    testQuery()
    +  }
    +
    +  test("test data loading with zstd compressor and offheap") {
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.ENABLE_OFFHEAP_SORT, "true")
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.COMPRESSOR, "zstd")
    +    createTable()
    +    loadData()
    +    testQuery()
    +  }
    +
    +  test("test data loading with zstd compressor and onheap") {
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.ENABLE_OFFHEAP_SORT, "false")
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.COMPRESSOR, "zstd")
    +    createTable()
    +    loadData()
    +    testQuery()
    +  }
    +
    +  test("test current zstd compressor on legacy store with snappy") {
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.ENABLE_OFFHEAP_SORT, "true")
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.COMPRESSOR, "snappy")
    +    createTable()
    +    loadData()
    +
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.ENABLE_OFFHEAP_SORT, "true")
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.COMPRESSOR, "zstd")
    +    loadData()
    +    sql(s"SELECT * FROM $tableName").show(false)
    +    checkAnswer(sql(s"SELECT count(*) FROM $tableName"), Seq(Row(16)))
    +  }
    +
    +  test("test current snappy compressor on legacy store with zstd") {
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.ENABLE_OFFHEAP_SORT, "true")
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.COMPRESSOR, "zstd")
    +    createTable()
    +    loadData()
    +
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.ENABLE_OFFHEAP_SORT, "true")
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.COMPRESSOR, "snappy")
    +    loadData()
    +    sql(s"SELECT * FROM $tableName").show(false)
    +    checkAnswer(sql(s"SELECT count(*) FROM $tableName"), Seq(Row(16)))
    +  }
    +
    +  test("test compaction with different compressor for each load") {
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.ENABLE_OFFHEAP_SORT, "true")
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.COMPRESSOR, "zstd")
    +    createTable()
    +    loadData()
    +
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.ENABLE_OFFHEAP_SORT, "false")
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.COMPRESSOR, "snappy")
    +    loadData()
    +
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.ENABLE_OFFHEAP_SORT, "true")
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.COMPRESSOR, "zstd")
    +    loadData()
    +
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.ENABLE_OFFHEAP_SORT, "true")
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.COMPRESSOR, "snappy")
    +    loadData()
    +
    +    // there are 8 loads
    +    checkAnswer(sql(s"SELECT count(*) FROM $tableName"), Seq(Row(4 * 8)))
    +    assert(sql(s"SHOW SEGMENTS FOR TABLE $tableName").count() == 8)
    +    sql(s"ALTER TABLE $tableName COMPACT 'major'")
    +    sql(s"CLEAN FILES FOR TABLE $tableName")
    +    // after compaction and clean, there should be on segment
    +    checkAnswer(sql(s"SELECT count(*) FROM $tableName"), Seq(Row(4 * 8)))
    +    assert(sql(s"SHOW SEGMENTS FOR TABLE $tableName").count() == 1)
    +  }
    +
    +  test("test data loading with unsupported compressor and onheap") {
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.ENABLE_OFFHEAP_SORT, "false")
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.COMPRESSOR, "fake")
    +    createTable()
    +    val exception = intercept[Throwable] {
    --- End diff --
    
    please intercept the exact exception install of Throwable


---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Failed  with Spark 2.1.0, Please check CI http://136.243.101.176:8080/job/ApacheCarbonPRBuilder1/8077/



---

[GitHub] carbondata pull request #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zs...

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

    https://github.com/apache/carbondata/pull/2628#discussion_r213922415
  
    --- Diff: core/src/main/java/org/apache/carbondata/core/datastore/compression/CompressorFactory.java ---
    @@ -17,42 +17,99 @@
     
     package org.apache.carbondata.core.datastore.compression;
     
    +import java.util.Arrays;
    +import java.util.HashMap;
    +import java.util.Map;
    +
     import org.apache.carbondata.core.constants.CarbonCommonConstants;
     import org.apache.carbondata.core.util.CarbonProperties;
    +import org.apache.carbondata.format.CompressionCodec;
     
     public class CompressorFactory {
    -
       private static final CompressorFactory COMPRESSOR_FACTORY = new CompressorFactory();
     
    -  private final Compressor snappyCompressor;
    +  private final Map<String, SupportedCompressor> compressors = new HashMap<>();
    +
    +  public enum SupportedCompressor {
    +    SNAPPY(CompressionCodec.SNAPPY, "snappy", SnappyCompressor.class),
    +    ZSTD(CompressionCodec.ZSTD, "zstd", ZstdCompressor.class);
    +
    +    private CompressionCodec codec;
    +    private String name;
    +    private Class<Compressor> compressorClass;
    +    private transient Compressor compressor;
    +
    +    SupportedCompressor(CompressionCodec codec, String name, Class compressorCls) {
    +      this.codec = codec;
    +      this.name = name;
    +      this.compressorClass = compressorCls;
    +    }
    +
    +    public CompressionCodec getCodec() {
    +      return codec;
    +    }
    +
    +    public String getName() {
    +      return name;
    +    }
    +
    +    /**
    +     * we will load the compressor only if it is needed
    +     */
    +    public Compressor getCompressor() {
    +      if (this.compressor == null) {
    +        try {
    +          this.compressor = compressorClass.newInstance();
    +        } catch (InstantiationException | IllegalAccessException e) {
    +          throw new RuntimeException("Exception occurs while getting compressor for " + name);
    +        }
    +      }
    +      return this.compressor;
    +    }
    +  }
     
       private CompressorFactory() {
    -    String compressorType = CarbonProperties.getInstance()
    -        .getProperty(CarbonCommonConstants.COMPRESSOR, CarbonCommonConstants.DEFAULT_COMPRESSOR);
    -    switch (compressorType) {
    -      case "snappy":
    -        snappyCompressor = new SnappyCompressor();
    -        break;
    -      default:
    -        throw new RuntimeException(
    -            "Invalid compressor type provided! Please provide valid compressor type");
    +    for (SupportedCompressor supportedCompressor : SupportedCompressor.values()) {
    +      compressors.put(supportedCompressor.getName(), supportedCompressor);
         }
       }
     
       public static CompressorFactory getInstance() {
         return COMPRESSOR_FACTORY;
       }
     
    +  /**
    +   * get the default compressor.
    +   * This method can only be called in data load procedure to compress column page.
    +   * In query procedure, we should read the compressor information from the metadata
    +   * in datafiles when we want to decompress the content.
    +   */
       public Compressor getCompressor() {
    -    return getCompressor(CarbonCommonConstants.DEFAULT_COMPRESSOR);
    +    String compressorType = CarbonProperties.getInstance()
    +        .getProperty(CarbonCommonConstants.COMPRESSOR, CarbonCommonConstants.DEFAULT_COMPRESSOR);
    +    if (!compressors.keySet().contains(compressorType)) {
    --- End diff --
    
    use compressors.containsKey directly


---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Success with Spark 2.2.1, Please check CI http://95.216.28.178:8080/job/ApacheCarbonPRBuilder1/395/



---

[GitHub] carbondata pull request #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zs...

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

    https://github.com/apache/carbondata/pull/2628


---

[GitHub] carbondata pull request #2628: [CARBONDATA-2851] Support zstd as column comp...

Posted by xuchuanyin <gi...@git.apache.org>.
GitHub user xuchuanyin reopened a pull request:

    https://github.com/apache/carbondata/pull/2628

    [CARBONDATA-2851] Support zstd as column compressor in final store

    1. add zstd compressor for compressing column data
    2. add zstd support in thrift
    3. legacy store is not considered in this commit
    4. since zstd does not support zero-copy while compressing, offheap will
    not take effect for zstd
    
    A simple test with 1.2GB raw CSV data shows that the size (in MB) of final store with different compressor: 
    
    | local dictionary | snappy | zstd | Size Reduced |
    | --- | --- | --- | -- |
    | local dict enabled | 335 | 207 | 38.2% |
    | local dict disabled | 375 | 225 | 40% |
    
    Be sure to do all of the following checklist to help us incorporate 
    your contribution quickly and easily:
    
     - [ ] Any interfaces changed?
     
     - [ ] Any backward compatibility impacted?
     
     - [ ] Document update required?
    
     - [ ] Testing done
            Please provide details on 
            - Whether new unit test cases have been added or why no new tests are required?
            - How it is tested? Please attach test report.
            - Is it a performance related change? Please attach the performance test report.
            - Any additional information to help reviewers in testing this change.
           
     - [ ] For large changes, please consider breaking it into sub-tasks under an umbrella JIRA. 
    


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

    $ git pull https://github.com/xuchuanyin/carbondata 0810_support_zstd_compressor_final_store

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

    https://github.com/apache/carbondata/pull/2628.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 #2628
    
----
commit bcd3d8f9c64f197668d46d29af1aa2ee2d956ceb
Author: xuchuanyin <xu...@...>
Date:   2018-08-10T14:02:57Z

    Support zstd as column compressor in final store
    
    1. add zstd compressor for compressing column data
    2. add zstd support in thrift
    3. legacy store is not considered in this commit
    4. since zstd does not support zero-copy while compressing, offheap will
    not take effect for zstd

----


---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    LGTM. spark 2.3 CI has problem currently, we are fixing it


---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Failed with Spark 2.2.1, Please check CI http://95.216.28.178:8080/job/ApacheCarbonPRBuilder1/19/



---

[GitHub] carbondata issue #2628: WIP: Support zstd as column compressor in final stor...

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

    https://github.com/apache/carbondata/pull/2628
  
    SDV Build Success , Please check CI http://144.76.159.231:8080/job/ApacheSDVTests/6246/



---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Success with Spark 2.1.0, Please check CI http://136.243.101.176:8080/job/ApacheCarbonPRBuilder2.1/104/



---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Failed  with Spark 2.3.1, Please check CI http://136.243.101.176:8080/job/ApacheCarbonPRBuilder2.3/8472/



---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Failed  with Spark 2.3.1, Please check CI http://136.243.101.176:8080/job/ApacheCarbonPRBuilder2.3/8479/



---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    @ravipesala @jackylk 
    I add an optional 'compressor_name' aside the 'compression_codec'. During processing, I use the compressor_name and set compression_codec to a deprecated value.
    
    Also I add an interface to register customize compressor and add a test for it.
    
    For now, all the review comments are resolved.


---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    retest this please


---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    retest this please


---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    fix conflicts


---

[GitHub] carbondata pull request #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zs...

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

    https://github.com/apache/carbondata/pull/2628#discussion_r214560533
  
    --- Diff: core/src/main/java/org/apache/carbondata/core/datastore/compression/ZstdCompressor.java ---
    @@ -0,0 +1,170 @@
    +/*
    + * 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.core.datastore.compression;
    +
    +import java.io.IOException;
    +import java.nio.ByteBuffer;
    +import java.nio.DoubleBuffer;
    +import java.nio.FloatBuffer;
    +import java.nio.IntBuffer;
    +import java.nio.LongBuffer;
    +import java.nio.ShortBuffer;
    +
    +import org.apache.carbondata.core.util.ByteUtil;
    +
    +import com.github.luben.zstd.Zstd;
    +
    +public class ZstdCompressor implements Compressor {
    +  private static final int COMPRESS_LEVEL = 3;
    --- End diff --
    
    It's a common parameter for almost all the compressors. The higher level it is, the bigger compress ratio will achieve while it will takes more time.
    Here Zstd use level 3 by default for common use.


---

[GitHub] carbondata pull request #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zs...

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

    https://github.com/apache/carbondata/pull/2628#discussion_r214583607
  
    --- Diff: core/src/main/java/org/apache/carbondata/core/datastore/page/ColumnPage.java ---
    @@ -66,15 +66,19 @@
       protected static final boolean unsafe = Boolean.parseBoolean(CarbonProperties.getInstance()
           .getProperty(CarbonCommonConstants.ENABLE_UNSAFE_COLUMN_PAGE,
               CarbonCommonConstants.ENABLE_UNSAFE_COLUMN_PAGE_DEFAULT));
    +  // compressor to be used for this column
    +  protected String columnCompressorName;
    --- End diff --
    
    refactored~


---

[GitHub] carbondata pull request #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zs...

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

    https://github.com/apache/carbondata/pull/2628


---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Failed  with Spark 2.3.1, Please check CI http://136.243.101.176:8080/job/ApacheCarbonPRBuilder2.3/8470/



---

[GitHub] carbondata pull request #2628: [CARBONDATA-2851] Support zstd as column comp...

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

    https://github.com/apache/carbondata/pull/2628#discussion_r209849039
  
    --- Diff: core/src/main/java/org/apache/carbondata/core/datastore/compression/ZstdCompressor.java ---
    @@ -0,0 +1,200 @@
    +/*
    + * 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.core.datastore.compression;
    +
    +import java.io.IOException;
    +import java.io.Serializable;
    +import java.nio.ByteBuffer;
    +
    +import org.apache.carbondata.common.logging.LogService;
    +import org.apache.carbondata.common.logging.LogServiceFactory;
    +
    +import com.github.luben.zstd.Zstd;
    +
    +public class ZstdCompressor implements Compressor, Serializable {
    +  private static final long serialVersionUID = 8181578747306832771L;
    +  private static final LogService LOGGER =
    +      LogServiceFactory.getLogService(ZstdCompressor.class.getName());
    +  private static final int COMPRESS_LEVEL = 3;
    +
    +  public ZstdCompressor() {
    +  }
    +
    +  @Override
    +  public String getName() {
    +    return "zstd";
    +  }
    +
    +  @Override
    +  public byte[] compressByte(byte[] unCompInput) {
    +    return Zstd.compress(unCompInput, 3);
    +  }
    +
    +  @Override
    +  public byte[] compressByte(byte[] unCompInput, int byteSize) {
    +    return Zstd.compress(unCompInput, COMPRESS_LEVEL);
    +  }
    +
    +  @Override
    +  public byte[] unCompressByte(byte[] compInput) {
    +    long estimatedUncompressLength = Zstd.decompressedSize(compInput);
    +    return Zstd.decompress(compInput, (int) estimatedUncompressLength);
    +  }
    +
    +  @Override
    +  public byte[] unCompressByte(byte[] compInput, int offset, int length) {
    +    // todo: how to avoid memory copy
    +    byte[] dstBytes = new byte[length];
    +    System.arraycopy(compInput, offset, dstBytes, 0, length);
    +    return unCompressByte(dstBytes);
    +  }
    +
    +  @Override
    +  public byte[] compressShort(short[] unCompInput) {
    +    // short use 2 bytes
    +    byte[] unCompArray = new byte[unCompInput.length * 2];
    +    ByteBuffer unCompBuffer = ByteBuffer.wrap(unCompArray);
    +    for (short input : unCompInput) {
    +      unCompBuffer.putShort(input);
    +    }
    +    return Zstd.compress(unCompBuffer.array(), COMPRESS_LEVEL);
    +  }
    +
    +  @Override
    +  public short[] unCompressShort(byte[] compInput, int offset, int length) {
    +    byte[] unCompArray = unCompressByte(compInput, offset, length);
    +    ByteBuffer unCompBuffer = ByteBuffer.wrap(unCompArray);
    +    short[] shorts = new short[unCompArray.length / 2];
    +    for (int i = 0; i < shorts.length; i++) {
    +      shorts[i] = unCompBuffer.getShort();
    +    }
    +    return shorts;
    +  }
    +
    +  @Override
    +  public byte[] compressInt(int[] unCompInput) {
    +    // int use 4 bytes
    +    byte[] unCompArray = new byte[unCompInput.length * 4];
    +    ByteBuffer unCompBuffer = ByteBuffer.wrap(unCompArray);
    +    for (int input : unCompInput) {
    +      unCompBuffer.putInt(input);
    +    }
    +    return Zstd.compress(unCompBuffer.array(), COMPRESS_LEVEL);
    +  }
    +
    +  @Override
    +  public int[] unCompressInt(byte[] compInput, int offset, int length) {
    +    byte[] unCompArray = unCompressByte(compInput, offset, length);
    +    ByteBuffer unCompBuffer = ByteBuffer.wrap(unCompArray);
    +    int[] ints = new int[unCompArray.length / 4];
    +    for (int i = 0; i < ints.length; i++) {
    +      ints[i] = unCompBuffer.getInt();
    +    }
    +    return ints;
    +  }
    +
    +  @Override
    +  public byte[] compressLong(long[] unCompInput) {
    +    // long use 8 bytes
    +    byte[] unCompArray = new byte[unCompInput.length * 8];
    +    ByteBuffer unCompBuffer = ByteBuffer.wrap(unCompArray);
    +    for (long input : unCompInput) {
    +      unCompBuffer.putLong(input);
    +    }
    +    return Zstd.compress(unCompBuffer.array(), COMPRESS_LEVEL);
    +  }
    +
    +  @Override
    +  public long[] unCompressLong(byte[] compInput, int offset, int length) {
    +    byte[] unCompArray = unCompressByte(compInput, offset, length);
    +    ByteBuffer unCompBuffer = ByteBuffer.wrap(unCompArray);
    +    long[] longs = new long[unCompArray.length / 8];
    +    for (int i = 0; i < longs.length; i++) {
    +      longs[i] = unCompBuffer.getLong();
    +    }
    +    return longs;
    +  }
    +
    +  @Override
    +  public byte[] compressFloat(float[] unCompInput) {
    +    // float use 4 bytes
    +    byte[] unCompArray = new byte[unCompInput.length * 4];
    +    ByteBuffer unCompBuffer = ByteBuffer.wrap(unCompArray);
    +    for (float input : unCompInput) {
    +      unCompBuffer.putFloat(input);
    +    }
    +    return Zstd.compress(unCompBuffer.array(), COMPRESS_LEVEL);
    +  }
    +
    +  @Override
    +  public float[] unCompressFloat(byte[] compInput, int offset, int length) {
    +    byte[] unCompArray = unCompressByte(compInput, offset, length);
    +    ByteBuffer unCompBuffer = ByteBuffer.wrap(unCompArray);
    +    float[] floats = new float[unCompArray.length / 4];
    +    for (int i = 0; i < floats.length; i++) {
    +      floats[i] = unCompBuffer.getFloat();
    +    }
    +    return floats;
    +  }
    +
    +  @Override
    +  public byte[] compressDouble(double[] unCompInput) {
    +    // double use 8 bytes
    +    byte[] unCompArray = new byte[unCompInput.length * 8];
    +    ByteBuffer unCompBuffer = ByteBuffer.wrap(unCompArray);
    +    for (double input : unCompInput) {
    +      unCompBuffer.putDouble(input);
    +    }
    +    return Zstd.compress(unCompBuffer.array(), COMPRESS_LEVEL);
    +  }
    +
    +  @Override
    +  public double[] unCompressDouble(byte[] compInput, int offset, int length) {
    +    byte[] unCompArray = unCompressByte(compInput, offset, length);
    +    ByteBuffer unCompBuffer = ByteBuffer.wrap(unCompArray);
    +    double[] doubles = new double[unCompArray.length / 8];
    +    for (int i = 0; i < doubles.length; i++) {
    +      doubles[i] = unCompBuffer.getDouble();
    +    }
    +    return doubles;
    +  }
    +
    +  @Override
    +  public long rawCompress(long inputAddress, int inputSize, long outputAddress) throws IOException {
    +    throw new RuntimeException("Not implemented rawCompress for zstd yet");
    +  }
    +
    +  @Override
    +  public long rawUncompress(byte[] input, byte[] output) throws IOException {
    +    return Zstd.decompress(output, input);
    +  }
    +
    +  @Override
    +  public int maxCompressedLength(int inputSize) {
    +    throw new RuntimeException("Not implemented maxCompressedLength for zstd yet");
    +  }
    +
    --- End diff --
    
    OK~


---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Failed  with Spark 2.1.0, Please check CI http://136.243.101.176:8080/job/ApacheCarbonPRBuilder1/7910/



---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    SDV Build Fail , Please check CI http://144.76.159.231:8080/job/ApacheSDVTests/6264/



---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    @ravipesala It's OK to add support column compressor in table properties.
    
    But as for another proposal "It is better to save compressor interface name in thrift instead of enum"
    ---
    It's not easy to handle this now. Cause the enum in thrift is used directly in our code, changing it is impossible in my opinion. Do you mean we shall duplicate the info and add another description for it by the side of current enum?


---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Failed with Spark 2.2.1, Please check CI http://95.216.28.178:8080/job/ApacheCarbonPRBuilder1/29/



---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Failed with Spark 2.2.1, Please check CI http://95.216.28.178:8080/job/ApacheCarbonPRBuilder1/35/



---

[GitHub] carbondata pull request #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zs...

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

    https://github.com/apache/carbondata/pull/2628#discussion_r214561432
  
    --- Diff: core/src/main/java/org/apache/carbondata/core/datastore/page/ColumnPage.java ---
    @@ -94,222 +98,240 @@ public void setStatsCollector(ColumnPageStatsCollector statsCollector) {
       }
     
       private static ColumnPage createDecimalPage(TableSpec.ColumnSpec columnSpec, DataType dataType,
    -      int pageSize) {
    +      int pageSize, String compressorName) {
         if (unsafe) {
           try {
    -        return new UnsafeDecimalColumnPage(columnSpec, dataType, pageSize);
    +        return new UnsafeDecimalColumnPage(columnSpec, dataType, pageSize, compressorName);
           } catch (MemoryException e) {
             throw new RuntimeException(e);
           }
         } else {
    -      return new SafeDecimalColumnPage(columnSpec, dataType, pageSize);
    +      return new SafeDecimalColumnPage(columnSpec, dataType, pageSize, compressorName);
         }
       }
     
       private static ColumnPage createVarLengthPage(TableSpec.ColumnSpec columnSpec, DataType dataType,
    -      int pageSize) {
    +      int pageSize, String compressorName) {
         if (unsafe) {
           try {
    -        return new UnsafeVarLengthColumnPage(columnSpec, dataType, pageSize);
    +        return new UnsafeVarLengthColumnPage(columnSpec, dataType, pageSize, compressorName);
           } catch (MemoryException e) {
             throw new RuntimeException(e);
           }
         } else {
    -      return new SafeVarLengthColumnPage(columnSpec, dataType, pageSize);
    +      return new SafeVarLengthColumnPage(columnSpec, dataType, pageSize, compressorName);
         }
       }
     
       private static ColumnPage createFixLengthPage(TableSpec.ColumnSpec columnSpec, DataType dataType,
    -      int pageSize) {
    +      int pageSize, String compressorName) {
         if (unsafe) {
           try {
    -        return new UnsafeFixLengthColumnPage(columnSpec, dataType, pageSize);
    +        return new UnsafeFixLengthColumnPage(columnSpec, dataType, pageSize, compressorName);
           } catch (MemoryException e) {
             throw new RuntimeException(e);
           }
         } else {
    -      return new SafeFixLengthColumnPage(columnSpec, dataType, pageSize);
    +      return new SafeFixLengthColumnPage(columnSpec, dataType, pageSize, compressorName);
         }
       }
     
       private static ColumnPage createFixLengthByteArrayPage(TableSpec.ColumnSpec columnSpec,
    -      DataType dataType, int pageSize, int eachValueSize) {
    +      DataType dataType, int pageSize, int eachValueSize, String compressorName) {
         if (unsafe) {
           try {
    -        return new UnsafeFixLengthColumnPage(columnSpec, dataType, pageSize, eachValueSize);
    +        return new UnsafeFixLengthColumnPage(columnSpec, dataType, pageSize, eachValueSize,
    +            compressorName);
           } catch (MemoryException e) {
             throw new RuntimeException(e);
           }
         } else {
    -      return new SafeFixLengthColumnPage(columnSpec, dataType, pageSize);
    +      return new SafeFixLengthColumnPage(columnSpec, dataType, pageSize, compressorName);
         }
       }
     
       private static ColumnPage createPage(TableSpec.ColumnSpec columnSpec, DataType dataType,
    -      int pageSize) {
    +      int pageSize, String compressorName) {
         if (DataTypes.isDecimal(dataType)) {
    -      return createDecimalPage(columnSpec, dataType, pageSize);
    +      return createDecimalPage(columnSpec, dataType, pageSize, compressorName);
         } else if (dataType.equals(BYTE_ARRAY)) {
    -      return createVarLengthPage(columnSpec, dataType, pageSize);
    +      return createVarLengthPage(columnSpec, dataType, pageSize, compressorName);
         } else {
    -      return createFixLengthPage(columnSpec, dataType, pageSize);
    +      return createFixLengthPage(columnSpec, dataType, pageSize, compressorName);
         }
       }
     
       public static ColumnPage newDecimalPage(TableSpec.ColumnSpec columnSpec, DataType dataType,
    -      int pageSize)
    +      int pageSize, String compressor)
         throws MemoryException {
    -    return newPage(columnSpec, dataType, pageSize);
    +    return newPage(columnSpec, dataType, pageSize, compressor);
       }
     
       public static ColumnPage newLocalDictPage(TableSpec.ColumnSpec columnSpec, DataType dataType,
           int pageSize, LocalDictionaryGenerator localDictionaryGenerator,
    -      boolean isComplexTypePrimitive) throws MemoryException {
    +      boolean isComplexTypePrimitive, String compressorName) throws MemoryException {
         if (unsafe) {
    -      return new LocalDictColumnPage(new UnsafeVarLengthColumnPage(columnSpec, dataType, pageSize),
    -          new UnsafeFixLengthColumnPage(columnSpec, DataTypes.BYTE_ARRAY, pageSize,
    -              CarbonCommonConstants.LOCAL_DICT_ENCODED_BYTEARRAY_SIZE),
    -          localDictionaryGenerator, isComplexTypePrimitive);
    +      ColumnPage actualPage =
    +          new UnsafeVarLengthColumnPage(columnSpec, dataType, pageSize, compressorName);
    +      ColumnPage encodedPage = new UnsafeFixLengthColumnPage(columnSpec, DataTypes.BYTE_ARRAY,
    +          pageSize, CarbonCommonConstants.LOCAL_DICT_ENCODED_BYTEARRAY_SIZE, compressorName);
    +      return new LocalDictColumnPage(actualPage, encodedPage, localDictionaryGenerator,
    +          isComplexTypePrimitive, compressorName);
         } else {
    -      return new LocalDictColumnPage(new SafeVarLengthColumnPage(columnSpec, dataType, pageSize),
    -          new SafeFixLengthColumnPage(columnSpec, DataTypes.BYTE_ARRAY, pageSize),
    -          localDictionaryGenerator, isComplexTypePrimitive);
    +      ColumnPage actualPage =
    +          new SafeVarLengthColumnPage(columnSpec, dataType, pageSize, compressorName);
    +      ColumnPage encodedPage =
    +          new SafeFixLengthColumnPage(columnSpec, DataTypes.BYTE_ARRAY, pageSize, compressorName);
    +      return new LocalDictColumnPage(actualPage, encodedPage, localDictionaryGenerator,
    --- End diff --
    
    ok, extract the code


---

[GitHub] carbondata pull request #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zs...

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

    https://github.com/apache/carbondata/pull/2628#discussion_r214560354
  
    --- Diff: core/src/main/java/org/apache/carbondata/core/datastore/compression/CompressorFactory.java ---
    @@ -17,42 +17,99 @@
     
     package org.apache.carbondata.core.datastore.compression;
     
    +import java.util.Arrays;
    +import java.util.HashMap;
    +import java.util.Map;
    +
     import org.apache.carbondata.core.constants.CarbonCommonConstants;
     import org.apache.carbondata.core.util.CarbonProperties;
    +import org.apache.carbondata.format.CompressionCodec;
     
     public class CompressorFactory {
    -
       private static final CompressorFactory COMPRESSOR_FACTORY = new CompressorFactory();
     
    -  private final Compressor snappyCompressor;
    +  private final Map<String, SupportedCompressor> compressors = new HashMap<>();
    +
    +  public enum SupportedCompressor {
    +    SNAPPY(CompressionCodec.SNAPPY, "snappy", SnappyCompressor.class),
    +    ZSTD(CompressionCodec.ZSTD, "zstd", ZstdCompressor.class);
    +
    +    private CompressionCodec codec;
    +    private String name;
    +    private Class<Compressor> compressorClass;
    +    private transient Compressor compressor;
    +
    +    SupportedCompressor(CompressionCodec codec, String name, Class compressorCls) {
    +      this.codec = codec;
    +      this.name = name;
    +      this.compressorClass = compressorCls;
    +    }
    +
    +    public CompressionCodec getCodec() {
    +      return codec;
    +    }
    +
    +    public String getName() {
    +      return name;
    +    }
    +
    +    /**
    +     * we will load the compressor only if it is needed
    +     */
    +    public Compressor getCompressor() {
    +      if (this.compressor == null) {
    +        try {
    +          this.compressor = compressorClass.newInstance();
    +        } catch (InstantiationException | IllegalAccessException e) {
    +          throw new RuntimeException("Exception occurs while getting compressor for " + name);
    +        }
    +      }
    +      return this.compressor;
    +    }
    +  }
     
       private CompressorFactory() {
    -    String compressorType = CarbonProperties.getInstance()
    -        .getProperty(CarbonCommonConstants.COMPRESSOR, CarbonCommonConstants.DEFAULT_COMPRESSOR);
    -    switch (compressorType) {
    -      case "snappy":
    -        snappyCompressor = new SnappyCompressor();
    -        break;
    -      default:
    -        throw new RuntimeException(
    -            "Invalid compressor type provided! Please provide valid compressor type");
    +    for (SupportedCompressor supportedCompressor : SupportedCompressor.values()) {
    +      compressors.put(supportedCompressor.getName(), supportedCompressor);
         }
       }
     
       public static CompressorFactory getInstance() {
         return COMPRESSOR_FACTORY;
       }
     
    +  /**
    +   * get the default compressor.
    +   * This method can only be called in data load procedure to compress column page.
    +   * In query procedure, we should read the compressor information from the metadata
    +   * in datafiles when we want to decompress the content.
    +   */
       public Compressor getCompressor() {
    -    return getCompressor(CarbonCommonConstants.DEFAULT_COMPRESSOR);
    +    String compressorType = CarbonProperties.getInstance()
    +        .getProperty(CarbonCommonConstants.COMPRESSOR, CarbonCommonConstants.DEFAULT_COMPRESSOR);
    +    if (!compressors.keySet().contains(compressorType)) {
    --- End diff --
    
    OK~


---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    @ravipesala As for the implementation, is **duplicate the info and add another description for it by the side of current enum** OK? Or do you have another suggestion to implement this?


---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Failed with Spark 2.2.1, Please check CI http://95.216.28.178:8080/job/ApacheCarbonPRBuilder1/402/



---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    retest this please


---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    @jackylk As a result of our discussion, I'll remove the support for customize compressor in this PR


---

[GitHub] carbondata pull request #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zs...

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

    https://github.com/apache/carbondata/pull/2628#discussion_r211844549
  
    --- Diff: integration/spark-common-test/src/test/scala/org/apache/carbondata/integration/spark/testsuite/dataload/TestLoadDataWithCompressor.scala ---
    @@ -0,0 +1,180 @@
    +/*
    + * 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.integration.spark.testsuite.dataload
    +
    +import org.apache.spark.sql.Row
    +import org.apache.spark.sql.test.util.QueryTest
    +import org.scalatest.{BeforeAndAfterAll, BeforeAndAfterEach}
    +
    +import org.apache.carbondata.core.constants.CarbonCommonConstants
    +import org.apache.carbondata.core.util.CarbonProperties
    +
    +class TestLoadDataWithCompressor extends QueryTest with BeforeAndAfterEach with BeforeAndAfterAll {
    +  private val tableName = "load_test_with_compressor"
    +
    +  override protected def afterEach(): Unit = {
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.ENABLE_OFFHEAP_SORT,
    +      CarbonCommonConstants.ENABLE_OFFHEAP_SORT_DEFAULT)
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.COMPRESSOR,
    +      CarbonCommonConstants.DEFAULT_COMPRESSOR)
    +    sql(s"DROP TABLE IF EXISTS $tableName")
    +  }
    +
    +  private def createTable(): Unit = {
    +    sql(s"DROP TABLE IF EXISTS $tableName")
    +    sql(
    +      s"""
    +         | CREATE TABLE $tableName(
    +         |    booleanField boolean,
    +         |    shortField smallint,
    +         |    intField int,
    +         |    bigintField bigint,
    +         |    doubleField double,
    +         |    stringField string,
    +         |    timestampField timestamp,
    +         |    decimalField decimal(18,2),
    +         |    dateField date,
    +         |    charField string,
    +         |    floatField float,
    +         |    stringDictField string,
    +         |    stringSortField string,
    +         |    stringLocalDictField string,
    +         |    longStringField string
    +         | )
    +         | STORED BY 'carbondata'
    +         | TBLPROPERTIES(
    +         |  'LONG_STRING_COLUMNS'='longStringField',
    +         |  'SORT_COLUMNS'='stringSortField',
    +         |  'DICTIONARY_INCLUDE'='stringDictField',
    +         |  'local_dictionary_enable'='true',
    +         |  'local_dictionary_threshold'='10000',
    +         |  'local_dictionary_include'='stringLocalDictField')
    +       """.stripMargin)
    +  }
    +
    +  private def loadData(): Unit = {
    +    sql(
    +      s"""
    +         | INSERT INTO TABLE $tableName VALUES
    +         |  (true,1,11,101,41.4,'string1','2015/4/23 12:01:01',12.34,'2015/4/23','aaa',1.5,'dict1','sort1','local_dict1','longstring1'),
    +         | (false,2,12,102,42.4,'string2','2015/5/23 12:01:03',23.45,'2015/5/23','bbb',2.5,'dict2','sort2','local_dict2','longstring2'),
    +         |  (true,3,13,163,43.4,'string3','2015/7/26 12:01:06',34.56,'2015/7/26','ccc',3.5,'dict3','sort3','local_dict3','longstring3'),
    +         | (NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL)
    +       """.stripMargin)
    +    sql(
    +      s"""
    +         | INSERT INTO TABLE $tableName VALUES
    +         |  (true,${Short.MaxValue - 2},${Int.MinValue + 2},${Long.MaxValue - 2},${Double.MinValue + 2},'string1','2015/4/23 12:01:01',${Double.MinValue + 2},'2015/4/23','aaa',${Float.MaxValue - 2},'dict1','sort1','local_dict1','longstring1'),
    +         | (false,2,12,102,42.4,'string2','2015/5/23 12:01:03',23.45,'2015/5/23','bbb',2.5,'dict2','sort2','local_dict2','longstring2'),
    +         |  (true,${Short.MinValue + 2},${Int.MaxValue - 2},${Long.MinValue + 2},${Double.MaxValue - 2},'string3','2015/7/26 12:01:06',${Double.MinValue + 2},'2015/7/26','ccc',${Float.MinValue + 2},'dict3','sort3','local_dict3','longstring3'),
    +         | (NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL)
    +       """.stripMargin)
    +  }
    +
    +  private def testQuery(): Unit = {
    +    sql(s"SELECT * FROM $tableName").show(false)
    +    checkAnswer(sql(s"SELECT count(*) FROM $tableName"), Seq(Row(8)))
    +  }
    +
    +  test("test data loading with snappy compressor and offheap") {
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.ENABLE_OFFHEAP_SORT, "true")
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.COMPRESSOR, "snappy")
    +    createTable()
    +    loadData()
    --- End diff --
    
    please add a testcase for 3 column pages to test multipe page scenario


---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    @xuchuanyin yes, we cannot get rid of enum. But add another optional field in `ChunkCompressionMeta` to take interface name. Just ignore the enum and read only interface name.
    @jackylk Please give your opinion on this.


---

[GitHub] carbondata issue #2628: WIP: Support zstd as column compressor in final stor...

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

    https://github.com/apache/carbondata/pull/2628
  
    SDV Build Fail , Please check CI http://144.76.159.231:8080/job/ApacheSDVTests/6254/



---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Failed  with Spark 2.1.0, Please check CI http://136.243.101.176:8080/job/ApacheCarbonPRBuilder1/8090/



---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Failed with Spark 2.2.1, Please check CI http://95.216.28.178:8080/job/ApacheCarbonPRBuilder1/258/



---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    retest this please


---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Failed  with Spark 2.3.1, Please check CI http://136.243.101.176:8080/job/ApacheCarbonPRBuilder2.3/8490/



---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Failed  with Spark 2.1.0, Please check CI http://136.243.101.176:8080/job/ApacheCarbonPRBuilder1/8171/



---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Failed  with Spark 2.1.0, Please check CI http://136.243.101.176:8080/job/ApacheCarbonPRBuilder1/8275/



---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Failed with Spark 2.2.1, Please check CI http://95.216.28.178:8080/job/ApacheCarbonPRBuilder1/194/



---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    SDV Build Fail , Please check CI http://144.76.159.231:8080/job/ApacheSDVTests/6371/



---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    @ravipesala yeah, that's what I'm doing now. please check the commit: https://github.com/apache/carbondata/pull/2628/commits/d21fd869d442f535e4704dc06d9edc2f01984cb0



---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    I raised another PR #2689 to replace this PR


---

[GitHub] carbondata issue #2628: WIP: Support zstd as column compressor in final stor...

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

    https://github.com/apache/carbondata/pull/2628
  
    retest this please



---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Failed with Spark 2.2.1, Please check CI http://95.216.28.178:8080/job/ApacheCarbonPRBuilder1/272/



---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Failed  with Spark 2.1.0, Please check CI http://136.243.101.176:8080/job/ApacheCarbonPRBuilder1/8147/



---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Failed  with Spark 2.1.0, Please check CI http://136.243.101.176:8080/job/ApacheCarbonPRBuilder1/8101/



---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Failed with Spark 2.2.1, Please check CI http://95.216.28.178:8080/job/ApacheCarbonPRBuilder1/76/



---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Failed with Spark 2.2.1, Please check CI http://95.216.28.178:8080/job/ApacheCarbonPRBuilder1/277/



---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Success with Spark 2.2.1, Please check CI http://95.216.28.178:8080/job/ApacheCarbonPRBuilder1/400/



---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Failed with Spark 2.2.1, Please check CI http://95.216.28.178:8080/job/ApacheCarbonPRBuilder1/180/



---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    @xuchuanyin It is better to save compressor interface name in thrift instead of enum. We should not change thrift for each type of compressor. We can't let user/developer to give compressor interface from the table if we change thrift for each compression type. 


---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    retest this please


---

[GitHub] carbondata issue #2628: [CARBONDATA-2851] Support zstd as column compressor ...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Failed with Spark 2.2.1, Please check CI http://88.99.58.216:8080/job/ApacheCarbonPRBuilder/6633/



---

[GitHub] carbondata issue #2628: WIP: Support zstd as column compressor in final stor...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Failed with Spark 2.2.1, Please check CI http://88.99.58.216:8080/job/ApacheCarbonPRBuilder/6609/



---

[GitHub] carbondata pull request #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zs...

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

    https://github.com/apache/carbondata/pull/2628#discussion_r211130605
  
    --- Diff: integration/spark-common-test/src/test/scala/org/apache/carbondata/integration/spark/testsuite/dataload/TestLoadDataWithCompressor.scala ---
    @@ -0,0 +1,180 @@
    +/*
    + * 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.integration.spark.testsuite.dataload
    +
    +import org.apache.spark.sql.Row
    +import org.apache.spark.sql.test.util.QueryTest
    +import org.scalatest.{BeforeAndAfterAll, BeforeAndAfterEach}
    +
    +import org.apache.carbondata.core.constants.CarbonCommonConstants
    +import org.apache.carbondata.core.util.CarbonProperties
    +
    +class TestLoadDataWithCompressor extends QueryTest with BeforeAndAfterEach with BeforeAndAfterAll {
    +  private val tableName = "load_test_with_compressor"
    +
    +  override protected def afterEach(): Unit = {
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.ENABLE_OFFHEAP_SORT,
    +      CarbonCommonConstants.ENABLE_OFFHEAP_SORT_DEFAULT)
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.COMPRESSOR,
    +      CarbonCommonConstants.DEFAULT_COMPRESSOR)
    +    sql(s"DROP TABLE IF EXISTS $tableName")
    +  }
    +
    +  private def createTable(): Unit = {
    +    sql(s"DROP TABLE IF EXISTS $tableName")
    +    sql(
    +      s"""
    +         | CREATE TABLE $tableName(
    +         |    booleanField boolean,
    +         |    shortField smallint,
    +         |    intField int,
    +         |    bigintField bigint,
    +         |    doubleField double,
    +         |    stringField string,
    +         |    timestampField timestamp,
    +         |    decimalField decimal(18,2),
    +         |    dateField date,
    +         |    charField string,
    +         |    floatField float,
    +         |    stringDictField string,
    +         |    stringSortField string,
    +         |    stringLocalDictField string,
    +         |    longStringField string
    +         | )
    +         | STORED BY 'carbondata'
    +         | TBLPROPERTIES(
    +         |  'LONG_STRING_COLUMNS'='longStringField',
    +         |  'SORT_COLUMNS'='stringSortField',
    +         |  'DICTIONARY_INCLUDE'='stringDictField',
    +         |  'local_dictionary_enable'='true',
    +         |  'local_dictionary_threshold'='10000',
    +         |  'local_dictionary_include'='stringLocalDictField')
    +       """.stripMargin)
    +  }
    +
    +  private def loadData(): Unit = {
    +    sql(
    +      s"""
    +         | INSERT INTO TABLE $tableName VALUES
    +         |  (true,1,11,101,41.4,'string1','2015/4/23 12:01:01',12.34,'2015/4/23','aaa',1.5,'dict1','sort1','local_dict1','longstring1'),
    +         | (false,2,12,102,42.4,'string2','2015/5/23 12:01:03',23.45,'2015/5/23','bbb',2.5,'dict2','sort2','local_dict2','longstring2'),
    +         |  (true,3,13,163,43.4,'string3','2015/7/26 12:01:06',34.56,'2015/7/26','ccc',3.5,'dict3','sort3','local_dict3','longstring3'),
    +         | (NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL)
    +       """.stripMargin)
    +    sql(
    +      s"""
    +         | INSERT INTO TABLE $tableName VALUES
    +         |  (true,${Short.MaxValue - 2},${Int.MinValue + 2},${Long.MaxValue - 2},${Double.MinValue + 2},'string1','2015/4/23 12:01:01',${Double.MinValue + 2},'2015/4/23','aaa',${Float.MaxValue - 2},'dict1','sort1','local_dict1','longstring1'),
    +         | (false,2,12,102,42.4,'string2','2015/5/23 12:01:03',23.45,'2015/5/23','bbb',2.5,'dict2','sort2','local_dict2','longstring2'),
    +         |  (true,${Short.MinValue + 2},${Int.MaxValue - 2},${Long.MinValue + 2},${Double.MaxValue - 2},'string3','2015/7/26 12:01:06',${Double.MinValue + 2},'2015/7/26','ccc',${Float.MinValue + 2},'dict3','sort3','local_dict3','longstring3'),
    +         | (NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL,NULL)
    +       """.stripMargin)
    +  }
    +
    +  private def testQuery(): Unit = {
    +    sql(s"SELECT * FROM $tableName").show(false)
    +    checkAnswer(sql(s"SELECT count(*) FROM $tableName"), Seq(Row(8)))
    +  }
    +
    +  test("test data loading with snappy compressor and offheap") {
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.ENABLE_OFFHEAP_SORT, "true")
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.COMPRESSOR, "snappy")
    +    createTable()
    +    loadData()
    +    testQuery()
    +  }
    +
    +  test("test data loading with zstd compressor and offheap") {
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.ENABLE_OFFHEAP_SORT, "true")
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.COMPRESSOR, "zstd")
    +    createTable()
    +    loadData()
    +    testQuery()
    +  }
    +
    +  test("test data loading with zstd compressor and onheap") {
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.ENABLE_OFFHEAP_SORT, "false")
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.COMPRESSOR, "zstd")
    +    createTable()
    +    loadData()
    +    testQuery()
    +  }
    +
    +  test("test current zstd compressor on legacy store with snappy") {
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.ENABLE_OFFHEAP_SORT, "true")
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.COMPRESSOR, "snappy")
    +    createTable()
    +    loadData()
    +
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.ENABLE_OFFHEAP_SORT, "true")
    +    CarbonProperties.getInstance().addProperty(CarbonCommonConstants.COMPRESSOR, "zstd")
    +    loadData()
    --- End diff --
    
    Currently the compressor is only configured by the system property while the load is triggered.
    
    Furthermore we can support configuring it in TBLProperty or LoadOption. We will not do it in this PR~


---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    At the begin of loading, it should specify a compressor name, and use this value during the whole loading step.


---

[GitHub] carbondata pull request #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zs...

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

    https://github.com/apache/carbondata/pull/2628#discussion_r213929653
  
    --- Diff: core/src/main/java/org/apache/carbondata/core/datastore/compression/ZstdCompressor.java ---
    @@ -0,0 +1,170 @@
    +/*
    + * 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.core.datastore.compression;
    +
    +import java.io.IOException;
    +import java.nio.ByteBuffer;
    +import java.nio.DoubleBuffer;
    +import java.nio.FloatBuffer;
    +import java.nio.IntBuffer;
    +import java.nio.LongBuffer;
    +import java.nio.ShortBuffer;
    +
    +import org.apache.carbondata.core.util.ByteUtil;
    +
    +import com.github.luben.zstd.Zstd;
    +
    +public class ZstdCompressor implements Compressor {
    +  private static final int COMPRESS_LEVEL = 3;
    --- End diff --
    
    What is COMPRESS_LEVEL?


---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Success with Spark 2.2.1, Please check CI http://95.216.28.178:8080/job/ApacheCarbonPRBuilder1/411/



---

[GitHub] carbondata issue #2628: WIP: Support zstd as column compressor in final stor...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Failed with Spark 2.2.1, Please check CI http://88.99.58.216:8080/job/ApacheCarbonPRBuilder/6618/



---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    retest this please


---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    @xuchuanyin Please provide table property also for compressor name while creating table 


---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Failed  with Spark 2.3.1, Please check CI http://136.243.101.176:8080/job/ApacheCarbonPRBuilder2.3/8481/



---

[GitHub] carbondata pull request #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zs...

Posted by xuchuanyin <gi...@git.apache.org>.
GitHub user xuchuanyin reopened a pull request:

    https://github.com/apache/carbondata/pull/2628

    [CARBONDATA-2851][CARBONDATA-2852] Support zstd as column compressor in final store

    1. add zstd compressor for compressing column data
    2. add zstd support in thrift
    3. since zstd does not support zero-copy while compressing, offheap will not take effect for zstd
    4. Column compressor is configured through system property and can be changed in each load. Before loading, Carbondata will get the compressor and use that compressor during that loading. During querying, carbondata will get the compressor information from metadata in the file data.
    5. Also support compressing streaming table using zstd. The compressor info is stored in FileHeader of the streaming file.
    6. This PR also considered and verified on the legacy store and compaction
    
    A simple test with 1.2GB raw CSV data shows that the size (in MB) of final store with different compressor: 
    
    | local dictionary | snappy | zstd | Size Reduced |
    | --- | --- | --- | -- |
    | enabled | 335 | 207 | 38.2% |
    | disabled | 375 | 225 | 40% |
    
    Be sure to do all of the following checklist to help us incorporate 
    your contribution quickly and easily:
    
     - [x] Any interfaces changed?
     `Yes, only internal used interfaces are changed`
     - [x] Any backward compatibility impacted?
     `Yes, backward compatibility is handled`
     - [x] Document update required?
    `Yes`
     - [x] Testing done
            Please provide details on 
            - Whether new unit test cases have been added or why no new tests are required?
    `Added tests`
            - How it is tested? Please attach test report.
    `Tested in local machine`
            - Is it a performance related change? Please attach the performance test report.
    `The size of final store has been decreased by 40% compared with default snappy`
            - Any additional information to help reviewers in testing this change.
    `NA`
           
     - [x] For large changes, please consider breaking it into sub-tasks under an umbrella JIRA. 
    `NA`


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

    $ git pull https://github.com/xuchuanyin/carbondata 0810_support_zstd_compressor_final_store

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

    https://github.com/apache/carbondata/pull/2628.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 #2628
    
----
commit c171ee13136785110f6fff8104afebc4b2f222c7
Author: xuchuanyin <xu...@...>
Date:   2018-08-10T14:02:57Z

    Support zstd as column compressor in final store
    
    1. add zstd compressor for compressing column data
    2. add zstd support in thrift
    3. legacy store is not considered in this commit
    4. since zstd does not support zero-copy while compressing, offheap will
    not take effect for zstd
    5. support lazy load for compressor

commit 6448e6f21da66172775b625730b922fdfa57822d
Author: xuchuanyin <xu...@...>
Date:   2018-08-13T13:45:42Z

    Support new compressor on legacy store
    
    In query procedure, we need to decompress the column page. Previously we
    get the compressor from system property. Now since we support new
    compressors, we should read the compressor information from the metadata
    in datafiles.
    This PR also solve the compatibility related problems on V1/V2 store where we
    only support snappy.

commit 2815c84f1d5fd99ff37ba6890d98fb2b73a95b00
Author: xuchuanyin <xu...@...>
Date:   2018-08-14T08:38:00Z

    fix comments

commit ac95c25fca1c37f10f9cce0db76062207d0d3cee
Author: xuchuanyin <xu...@...>
Date:   2018-08-23T09:35:23Z

    Determine the column compressor before data loading
    
    we will get the column compressor before data loading/compaction start,
    so that it can make all the pages use the same compressor in case of
    concurrent modifying compressor during loading.

commit a672d3baad1c476308c0aec5133e418afeaeacb2
Author: xuchuanyin <xu...@...>
Date:   2018-08-27T11:18:30Z

    set compressor in carbon load model
    
    column compressor is necessary for carbon load model, otherwise load
    will fail.

commit d05c1cc38e1fa42ef94f70577ee2a715f649ebe3
Author: xuchuanyin <xu...@...>
Date:   2018-08-30T04:02:33Z

    fix error in test

commit fb8cdfb1258b477f7d8c867ee74bd59386725d9c
Author: xuchuanyin <xu...@...>
Date:   2018-09-03T03:58:02Z

    fix review comments
    
    optimize parameters for column page, use columnPageEncodeMeta instead of
    its members

----


---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Success with Spark 2.1.0, Please check CI http://136.243.101.176:8080/job/ApacheCarbonPRBuilder1/8328/



---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Failed  with Spark 2.3.1, Please check CI http://136.243.101.176:8080/job/ApacheCarbonPRBuilder2.3/8347/



---

[GitHub] carbondata pull request #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zs...

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

    https://github.com/apache/carbondata/pull/2628#discussion_r214560410
  
    --- Diff: core/src/main/java/org/apache/carbondata/core/datastore/compression/CompressorFactory.java ---
    @@ -17,42 +17,99 @@
     
     package org.apache.carbondata.core.datastore.compression;
     
    +import java.util.Arrays;
    +import java.util.HashMap;
    +import java.util.Map;
    +
     import org.apache.carbondata.core.constants.CarbonCommonConstants;
     import org.apache.carbondata.core.util.CarbonProperties;
    +import org.apache.carbondata.format.CompressionCodec;
     
     public class CompressorFactory {
    -
       private static final CompressorFactory COMPRESSOR_FACTORY = new CompressorFactory();
     
    -  private final Compressor snappyCompressor;
    +  private final Map<String, SupportedCompressor> compressors = new HashMap<>();
    +
    +  public enum SupportedCompressor {
    +    SNAPPY(CompressionCodec.SNAPPY, "snappy", SnappyCompressor.class),
    +    ZSTD(CompressionCodec.ZSTD, "zstd", ZstdCompressor.class);
    +
    +    private CompressionCodec codec;
    +    private String name;
    +    private Class<Compressor> compressorClass;
    +    private transient Compressor compressor;
    +
    +    SupportedCompressor(CompressionCodec codec, String name, Class compressorCls) {
    +      this.codec = codec;
    +      this.name = name;
    +      this.compressorClass = compressorCls;
    +    }
    +
    +    public CompressionCodec getCodec() {
    +      return codec;
    +    }
    +
    +    public String getName() {
    +      return name;
    +    }
    +
    +    /**
    +     * we will load the compressor only if it is needed
    +     */
    +    public Compressor getCompressor() {
    +      if (this.compressor == null) {
    +        try {
    +          this.compressor = compressorClass.newInstance();
    +        } catch (InstantiationException | IllegalAccessException e) {
    +          throw new RuntimeException("Exception occurs while getting compressor for " + name);
    +        }
    +      }
    +      return this.compressor;
    +    }
    +  }
     
       private CompressorFactory() {
    -    String compressorType = CarbonProperties.getInstance()
    -        .getProperty(CarbonCommonConstants.COMPRESSOR, CarbonCommonConstants.DEFAULT_COMPRESSOR);
    -    switch (compressorType) {
    -      case "snappy":
    -        snappyCompressor = new SnappyCompressor();
    -        break;
    -      default:
    -        throw new RuntimeException(
    -            "Invalid compressor type provided! Please provide valid compressor type");
    +    for (SupportedCompressor supportedCompressor : SupportedCompressor.values()) {
    +      compressors.put(supportedCompressor.getName(), supportedCompressor);
         }
       }
     
       public static CompressorFactory getInstance() {
         return COMPRESSOR_FACTORY;
       }
     
    +  /**
    +   * get the default compressor.
    +   * This method can only be called in data load procedure to compress column page.
    +   * In query procedure, we should read the compressor information from the metadata
    +   * in datafiles when we want to decompress the content.
    +   */
       public Compressor getCompressor() {
    -    return getCompressor(CarbonCommonConstants.DEFAULT_COMPRESSOR);
    +    String compressorType = CarbonProperties.getInstance()
    --- End diff --
    
    not case sensitive, internally it will be converted to use lowercase


---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Failed  with Spark 2.1.0, Please check CI http://136.243.101.176:8080/job/ApacheCarbonPRBuilder1/8165/



---

[GitHub] carbondata issue #2628: [CARBONDATA-2851] Support zstd as column compressor ...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Failed  with Spark 2.1.0, Please check CI http://136.243.101.176:8080/job/ApacheCarbonPRBuilder1/7908/



---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Success with Spark 2.2.1, Please check CI http://88.99.58.216:8080/job/ApacheCarbonPRBuilder/6635/



---

[GitHub] carbondata issue #2628: WIP: Support zstd as column compressor in final stor...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Failed  with Spark 2.1.0, Please check CI http://136.243.101.176:8080/job/ApacheCarbonPRBuilder1/7894/



---

[GitHub] carbondata pull request #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zs...

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

    https://github.com/apache/carbondata/pull/2628#discussion_r214583592
  
    --- Diff: core/src/main/java/org/apache/carbondata/core/datastore/page/ColumnPage.java ---
    @@ -94,222 +98,240 @@ public void setStatsCollector(ColumnPageStatsCollector statsCollector) {
       }
     
       private static ColumnPage createDecimalPage(TableSpec.ColumnSpec columnSpec, DataType dataType,
    -      int pageSize) {
    +      int pageSize, String compressorName) {
         if (unsafe) {
           try {
    -        return new UnsafeDecimalColumnPage(columnSpec, dataType, pageSize);
    +        return new UnsafeDecimalColumnPage(columnSpec, dataType, pageSize, compressorName);
           } catch (MemoryException e) {
             throw new RuntimeException(e);
           }
         } else {
    -      return new SafeDecimalColumnPage(columnSpec, dataType, pageSize);
    +      return new SafeDecimalColumnPage(columnSpec, dataType, pageSize, compressorName);
         }
       }
     
       private static ColumnPage createVarLengthPage(TableSpec.ColumnSpec columnSpec, DataType dataType,
    -      int pageSize) {
    +      int pageSize, String compressorName) {
         if (unsafe) {
           try {
    -        return new UnsafeVarLengthColumnPage(columnSpec, dataType, pageSize);
    +        return new UnsafeVarLengthColumnPage(columnSpec, dataType, pageSize, compressorName);
           } catch (MemoryException e) {
             throw new RuntimeException(e);
           }
         } else {
    -      return new SafeVarLengthColumnPage(columnSpec, dataType, pageSize);
    +      return new SafeVarLengthColumnPage(columnSpec, dataType, pageSize, compressorName);
         }
       }
     
       private static ColumnPage createFixLengthPage(TableSpec.ColumnSpec columnSpec, DataType dataType,
    -      int pageSize) {
    +      int pageSize, String compressorName) {
         if (unsafe) {
           try {
    -        return new UnsafeFixLengthColumnPage(columnSpec, dataType, pageSize);
    +        return new UnsafeFixLengthColumnPage(columnSpec, dataType, pageSize, compressorName);
           } catch (MemoryException e) {
             throw new RuntimeException(e);
           }
         } else {
    -      return new SafeFixLengthColumnPage(columnSpec, dataType, pageSize);
    +      return new SafeFixLengthColumnPage(columnSpec, dataType, pageSize, compressorName);
         }
       }
     
       private static ColumnPage createFixLengthByteArrayPage(TableSpec.ColumnSpec columnSpec,
    -      DataType dataType, int pageSize, int eachValueSize) {
    +      DataType dataType, int pageSize, int eachValueSize, String compressorName) {
         if (unsafe) {
           try {
    -        return new UnsafeFixLengthColumnPage(columnSpec, dataType, pageSize, eachValueSize);
    +        return new UnsafeFixLengthColumnPage(columnSpec, dataType, pageSize, eachValueSize,
    +            compressorName);
           } catch (MemoryException e) {
             throw new RuntimeException(e);
           }
         } else {
    -      return new SafeFixLengthColumnPage(columnSpec, dataType, pageSize);
    +      return new SafeFixLengthColumnPage(columnSpec, dataType, pageSize, compressorName);
         }
       }
     
       private static ColumnPage createPage(TableSpec.ColumnSpec columnSpec, DataType dataType,
    -      int pageSize) {
    +      int pageSize, String compressorName) {
         if (DataTypes.isDecimal(dataType)) {
    -      return createDecimalPage(columnSpec, dataType, pageSize);
    +      return createDecimalPage(columnSpec, dataType, pageSize, compressorName);
         } else if (dataType.equals(BYTE_ARRAY)) {
    -      return createVarLengthPage(columnSpec, dataType, pageSize);
    +      return createVarLengthPage(columnSpec, dataType, pageSize, compressorName);
         } else {
    -      return createFixLengthPage(columnSpec, dataType, pageSize);
    +      return createFixLengthPage(columnSpec, dataType, pageSize, compressorName);
         }
       }
     
       public static ColumnPage newDecimalPage(TableSpec.ColumnSpec columnSpec, DataType dataType,
    -      int pageSize)
    +      int pageSize, String compressor)
         throws MemoryException {
    -    return newPage(columnSpec, dataType, pageSize);
    +    return newPage(columnSpec, dataType, pageSize, compressor);
       }
     
       public static ColumnPage newLocalDictPage(TableSpec.ColumnSpec columnSpec, DataType dataType,
           int pageSize, LocalDictionaryGenerator localDictionaryGenerator,
    -      boolean isComplexTypePrimitive) throws MemoryException {
    +      boolean isComplexTypePrimitive, String compressorName) throws MemoryException {
         if (unsafe) {
    -      return new LocalDictColumnPage(new UnsafeVarLengthColumnPage(columnSpec, dataType, pageSize),
    -          new UnsafeFixLengthColumnPage(columnSpec, DataTypes.BYTE_ARRAY, pageSize,
    -              CarbonCommonConstants.LOCAL_DICT_ENCODED_BYTEARRAY_SIZE),
    -          localDictionaryGenerator, isComplexTypePrimitive);
    +      ColumnPage actualPage =
    +          new UnsafeVarLengthColumnPage(columnSpec, dataType, pageSize, compressorName);
    +      ColumnPage encodedPage = new UnsafeFixLengthColumnPage(columnSpec, DataTypes.BYTE_ARRAY,
    +          pageSize, CarbonCommonConstants.LOCAL_DICT_ENCODED_BYTEARRAY_SIZE, compressorName);
    +      return new LocalDictColumnPage(actualPage, encodedPage, localDictionaryGenerator,
    +          isComplexTypePrimitive, compressorName);
         } else {
    -      return new LocalDictColumnPage(new SafeVarLengthColumnPage(columnSpec, dataType, pageSize),
    -          new SafeFixLengthColumnPage(columnSpec, DataTypes.BYTE_ARRAY, pageSize),
    -          localDictionaryGenerator, isComplexTypePrimitive);
    +      ColumnPage actualPage =
    +          new SafeVarLengthColumnPage(columnSpec, dataType, pageSize, compressorName);
    +      ColumnPage encodedPage =
    +          new SafeFixLengthColumnPage(columnSpec, DataTypes.BYTE_ARRAY, pageSize, compressorName);
    +      return new LocalDictColumnPage(actualPage, encodedPage, localDictionaryGenerator,
    +          isComplexTypePrimitive, compressorName);
         }
       }
     
       /**
        * Create a new page of dataType and number of row = pageSize
        */
       public static ColumnPage newPage(TableSpec.ColumnSpec columnSpec, DataType dataType,
    --- End diff --
    
    OK, fixed


---

[GitHub] carbondata pull request #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zs...

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

    https://github.com/apache/carbondata/pull/2628#discussion_r213942049
  
    --- Diff: core/src/main/java/org/apache/carbondata/core/util/BlockletDataMapUtil.java ---
    @@ -357,8 +357,8 @@ private static boolean isSameColumnSchemaList(List<ColumnSchema> indexFileColumn
           columnSchema.write(dataOutput);
         }
         byte[] byteArray = stream.toByteArray();
    -    // Compress with snappy to reduce the size of schema
    -    return CompressorFactory.getInstance().getCompressor().compressByte(byteArray);
    +    // Compress to reduce the size of schema
    +    return CompressorFactory.getInstance().getCompressor("snappy").compressByte(byteArray);
    --- End diff --
    
    SupportedCompressor.SNAPPY.getName


---

[GitHub] carbondata pull request #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zs...

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

    https://github.com/apache/carbondata/pull/2628#discussion_r213934733
  
    --- Diff: core/src/main/java/org/apache/carbondata/core/datastore/page/ColumnPage.java ---
    @@ -94,222 +98,240 @@ public void setStatsCollector(ColumnPageStatsCollector statsCollector) {
       }
     
       private static ColumnPage createDecimalPage(TableSpec.ColumnSpec columnSpec, DataType dataType,
    -      int pageSize) {
    +      int pageSize, String compressorName) {
         if (unsafe) {
           try {
    -        return new UnsafeDecimalColumnPage(columnSpec, dataType, pageSize);
    +        return new UnsafeDecimalColumnPage(columnSpec, dataType, pageSize, compressorName);
           } catch (MemoryException e) {
             throw new RuntimeException(e);
           }
         } else {
    -      return new SafeDecimalColumnPage(columnSpec, dataType, pageSize);
    +      return new SafeDecimalColumnPage(columnSpec, dataType, pageSize, compressorName);
         }
       }
     
       private static ColumnPage createVarLengthPage(TableSpec.ColumnSpec columnSpec, DataType dataType,
    -      int pageSize) {
    +      int pageSize, String compressorName) {
         if (unsafe) {
           try {
    -        return new UnsafeVarLengthColumnPage(columnSpec, dataType, pageSize);
    +        return new UnsafeVarLengthColumnPage(columnSpec, dataType, pageSize, compressorName);
           } catch (MemoryException e) {
             throw new RuntimeException(e);
           }
         } else {
    -      return new SafeVarLengthColumnPage(columnSpec, dataType, pageSize);
    +      return new SafeVarLengthColumnPage(columnSpec, dataType, pageSize, compressorName);
         }
       }
     
       private static ColumnPage createFixLengthPage(TableSpec.ColumnSpec columnSpec, DataType dataType,
    -      int pageSize) {
    +      int pageSize, String compressorName) {
         if (unsafe) {
           try {
    -        return new UnsafeFixLengthColumnPage(columnSpec, dataType, pageSize);
    +        return new UnsafeFixLengthColumnPage(columnSpec, dataType, pageSize, compressorName);
           } catch (MemoryException e) {
             throw new RuntimeException(e);
           }
         } else {
    -      return new SafeFixLengthColumnPage(columnSpec, dataType, pageSize);
    +      return new SafeFixLengthColumnPage(columnSpec, dataType, pageSize, compressorName);
         }
       }
     
       private static ColumnPage createFixLengthByteArrayPage(TableSpec.ColumnSpec columnSpec,
    -      DataType dataType, int pageSize, int eachValueSize) {
    +      DataType dataType, int pageSize, int eachValueSize, String compressorName) {
         if (unsafe) {
           try {
    -        return new UnsafeFixLengthColumnPage(columnSpec, dataType, pageSize, eachValueSize);
    +        return new UnsafeFixLengthColumnPage(columnSpec, dataType, pageSize, eachValueSize,
    +            compressorName);
           } catch (MemoryException e) {
             throw new RuntimeException(e);
           }
         } else {
    -      return new SafeFixLengthColumnPage(columnSpec, dataType, pageSize);
    +      return new SafeFixLengthColumnPage(columnSpec, dataType, pageSize, compressorName);
         }
       }
     
       private static ColumnPage createPage(TableSpec.ColumnSpec columnSpec, DataType dataType,
    -      int pageSize) {
    +      int pageSize, String compressorName) {
         if (DataTypes.isDecimal(dataType)) {
    -      return createDecimalPage(columnSpec, dataType, pageSize);
    +      return createDecimalPage(columnSpec, dataType, pageSize, compressorName);
         } else if (dataType.equals(BYTE_ARRAY)) {
    -      return createVarLengthPage(columnSpec, dataType, pageSize);
    +      return createVarLengthPage(columnSpec, dataType, pageSize, compressorName);
         } else {
    -      return createFixLengthPage(columnSpec, dataType, pageSize);
    +      return createFixLengthPage(columnSpec, dataType, pageSize, compressorName);
         }
       }
     
       public static ColumnPage newDecimalPage(TableSpec.ColumnSpec columnSpec, DataType dataType,
    -      int pageSize)
    +      int pageSize, String compressor)
         throws MemoryException {
    -    return newPage(columnSpec, dataType, pageSize);
    +    return newPage(columnSpec, dataType, pageSize, compressor);
       }
     
       public static ColumnPage newLocalDictPage(TableSpec.ColumnSpec columnSpec, DataType dataType,
           int pageSize, LocalDictionaryGenerator localDictionaryGenerator,
    -      boolean isComplexTypePrimitive) throws MemoryException {
    +      boolean isComplexTypePrimitive, String compressorName) throws MemoryException {
         if (unsafe) {
    -      return new LocalDictColumnPage(new UnsafeVarLengthColumnPage(columnSpec, dataType, pageSize),
    -          new UnsafeFixLengthColumnPage(columnSpec, DataTypes.BYTE_ARRAY, pageSize,
    -              CarbonCommonConstants.LOCAL_DICT_ENCODED_BYTEARRAY_SIZE),
    -          localDictionaryGenerator, isComplexTypePrimitive);
    +      ColumnPage actualPage =
    +          new UnsafeVarLengthColumnPage(columnSpec, dataType, pageSize, compressorName);
    +      ColumnPage encodedPage = new UnsafeFixLengthColumnPage(columnSpec, DataTypes.BYTE_ARRAY,
    +          pageSize, CarbonCommonConstants.LOCAL_DICT_ENCODED_BYTEARRAY_SIZE, compressorName);
    +      return new LocalDictColumnPage(actualPage, encodedPage, localDictionaryGenerator,
    +          isComplexTypePrimitive, compressorName);
         } else {
    -      return new LocalDictColumnPage(new SafeVarLengthColumnPage(columnSpec, dataType, pageSize),
    -          new SafeFixLengthColumnPage(columnSpec, DataTypes.BYTE_ARRAY, pageSize),
    -          localDictionaryGenerator, isComplexTypePrimitive);
    +      ColumnPage actualPage =
    +          new SafeVarLengthColumnPage(columnSpec, dataType, pageSize, compressorName);
    +      ColumnPage encodedPage =
    +          new SafeFixLengthColumnPage(columnSpec, DataTypes.BYTE_ARRAY, pageSize, compressorName);
    +      return new LocalDictColumnPage(actualPage, encodedPage, localDictionaryGenerator,
    --- End diff --
    
    it is the same with 178  line


---

[GitHub] carbondata pull request #2628: [CARBONDATA-2851] Support zstd as column comp...

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

    https://github.com/apache/carbondata/pull/2628#discussion_r209543118
  
    --- Diff: core/src/main/java/org/apache/carbondata/core/datastore/compression/ZstdCompressor.java ---
    @@ -0,0 +1,200 @@
    +/*
    + * 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.core.datastore.compression;
    +
    +import java.io.IOException;
    +import java.io.Serializable;
    +import java.nio.ByteBuffer;
    +
    +import org.apache.carbondata.common.logging.LogService;
    +import org.apache.carbondata.common.logging.LogServiceFactory;
    +
    +import com.github.luben.zstd.Zstd;
    +
    +public class ZstdCompressor implements Compressor, Serializable {
    +  private static final long serialVersionUID = 8181578747306832771L;
    +  private static final LogService LOGGER =
    +      LogServiceFactory.getLogService(ZstdCompressor.class.getName());
    +  private static final int COMPRESS_LEVEL = 3;
    +
    +  public ZstdCompressor() {
    +  }
    +
    +  @Override
    +  public String getName() {
    +    return "zstd";
    +  }
    +
    +  @Override
    +  public byte[] compressByte(byte[] unCompInput) {
    +    return Zstd.compress(unCompInput, 3);
    +  }
    +
    +  @Override
    +  public byte[] compressByte(byte[] unCompInput, int byteSize) {
    +    return Zstd.compress(unCompInput, COMPRESS_LEVEL);
    +  }
    +
    +  @Override
    +  public byte[] unCompressByte(byte[] compInput) {
    +    long estimatedUncompressLength = Zstd.decompressedSize(compInput);
    +    return Zstd.decompress(compInput, (int) estimatedUncompressLength);
    +  }
    +
    +  @Override
    +  public byte[] unCompressByte(byte[] compInput, int offset, int length) {
    +    // todo: how to avoid memory copy
    +    byte[] dstBytes = new byte[length];
    +    System.arraycopy(compInput, offset, dstBytes, 0, length);
    +    return unCompressByte(dstBytes);
    +  }
    +
    +  @Override
    +  public byte[] compressShort(short[] unCompInput) {
    +    // short use 2 bytes
    +    byte[] unCompArray = new byte[unCompInput.length * 2];
    +    ByteBuffer unCompBuffer = ByteBuffer.wrap(unCompArray);
    +    for (short input : unCompInput) {
    +      unCompBuffer.putShort(input);
    +    }
    +    return Zstd.compress(unCompBuffer.array(), COMPRESS_LEVEL);
    +  }
    +
    +  @Override
    +  public short[] unCompressShort(byte[] compInput, int offset, int length) {
    +    byte[] unCompArray = unCompressByte(compInput, offset, length);
    +    ByteBuffer unCompBuffer = ByteBuffer.wrap(unCompArray);
    +    short[] shorts = new short[unCompArray.length / 2];
    +    for (int i = 0; i < shorts.length; i++) {
    +      shorts[i] = unCompBuffer.getShort();
    +    }
    +    return shorts;
    +  }
    +
    +  @Override
    +  public byte[] compressInt(int[] unCompInput) {
    +    // int use 4 bytes
    +    byte[] unCompArray = new byte[unCompInput.length * 4];
    +    ByteBuffer unCompBuffer = ByteBuffer.wrap(unCompArray);
    +    for (int input : unCompInput) {
    +      unCompBuffer.putInt(input);
    +    }
    +    return Zstd.compress(unCompBuffer.array(), COMPRESS_LEVEL);
    +  }
    +
    +  @Override
    +  public int[] unCompressInt(byte[] compInput, int offset, int length) {
    +    byte[] unCompArray = unCompressByte(compInput, offset, length);
    +    ByteBuffer unCompBuffer = ByteBuffer.wrap(unCompArray);
    +    int[] ints = new int[unCompArray.length / 4];
    +    for (int i = 0; i < ints.length; i++) {
    +      ints[i] = unCompBuffer.getInt();
    +    }
    +    return ints;
    +  }
    --- End diff --
    
    can try following code style to convert unCompress byte result to target datatype:
    (take Int for example)
    ```
        byte[] unCompArray = unCompressByte(compInput, offset, length);
        IntBuffer buf = ByteBuffer.wrap(unCompArray).asIntBuffer();
        int[] dest = new int[buf.remaining()];
        buf.get(dest);
        return dest;
    ```


---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    @ravipesala fine, I'll rework on this.
    The bad news is that the Enum 'CompressionCodec' in thrift is 'required', so even we do use it, we cannot get rid of it.
    The good news is that for legacy store, it is always snappy which makes it easier if we bypass this Enum.


---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    @xuchuanyin It is very promissing in term of compression, great!
    Have you also tested for decompression speed? It will be great if it also improves decompression speed.


---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    retest this please


---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Success with Spark 2.1.0, Please check CI http://136.243.101.176:8080/job/ApacheCarbonPRBuilder1/8324/



---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Success with Spark 2.1.0, Please check CI http://136.243.101.176:8080/job/ApacheCarbonPRBuilder2.1/226/



---

[GitHub] carbondata issue #2628: [CARBONDATA-2851][CARBONDATA-2852] Support zstd as c...

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

    https://github.com/apache/carbondata/pull/2628
  
    Build Success with Spark 2.1.0, Please check CI http://136.243.101.176:8080/job/ApacheCarbonPRBuilder2.1/109/



---

[GitHub] carbondata pull request #2628: [CARBONDATA-2851] Support zstd as column comp...

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

    https://github.com/apache/carbondata/pull/2628#discussion_r209545381
  
    --- Diff: core/src/main/java/org/apache/carbondata/core/datastore/compression/ZstdCompressor.java ---
    @@ -0,0 +1,200 @@
    +/*
    + * 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.core.datastore.compression;
    +
    +import java.io.IOException;
    +import java.io.Serializable;
    +import java.nio.ByteBuffer;
    +
    +import org.apache.carbondata.common.logging.LogService;
    +import org.apache.carbondata.common.logging.LogServiceFactory;
    +
    +import com.github.luben.zstd.Zstd;
    +
    +public class ZstdCompressor implements Compressor, Serializable {
    +  private static final long serialVersionUID = 8181578747306832771L;
    +  private static final LogService LOGGER =
    +      LogServiceFactory.getLogService(ZstdCompressor.class.getName());
    +  private static final int COMPRESS_LEVEL = 3;
    +
    +  public ZstdCompressor() {
    +  }
    +
    +  @Override
    +  public String getName() {
    +    return "zstd";
    +  }
    +
    +  @Override
    +  public byte[] compressByte(byte[] unCompInput) {
    +    return Zstd.compress(unCompInput, 3);
    +  }
    +
    +  @Override
    +  public byte[] compressByte(byte[] unCompInput, int byteSize) {
    +    return Zstd.compress(unCompInput, COMPRESS_LEVEL);
    +  }
    +
    +  @Override
    +  public byte[] unCompressByte(byte[] compInput) {
    +    long estimatedUncompressLength = Zstd.decompressedSize(compInput);
    +    return Zstd.decompress(compInput, (int) estimatedUncompressLength);
    +  }
    +
    +  @Override
    +  public byte[] unCompressByte(byte[] compInput, int offset, int length) {
    +    // todo: how to avoid memory copy
    +    byte[] dstBytes = new byte[length];
    +    System.arraycopy(compInput, offset, dstBytes, 0, length);
    +    return unCompressByte(dstBytes);
    +  }
    +
    +  @Override
    +  public byte[] compressShort(short[] unCompInput) {
    +    // short use 2 bytes
    +    byte[] unCompArray = new byte[unCompInput.length * 2];
    +    ByteBuffer unCompBuffer = ByteBuffer.wrap(unCompArray);
    +    for (short input : unCompInput) {
    +      unCompBuffer.putShort(input);
    +    }
    +    return Zstd.compress(unCompBuffer.array(), COMPRESS_LEVEL);
    +  }
    +
    +  @Override
    +  public short[] unCompressShort(byte[] compInput, int offset, int length) {
    +    byte[] unCompArray = unCompressByte(compInput, offset, length);
    +    ByteBuffer unCompBuffer = ByteBuffer.wrap(unCompArray);
    +    short[] shorts = new short[unCompArray.length / 2];
    +    for (int i = 0; i < shorts.length; i++) {
    +      shorts[i] = unCompBuffer.getShort();
    +    }
    +    return shorts;
    +  }
    +
    +  @Override
    +  public byte[] compressInt(int[] unCompInput) {
    +    // int use 4 bytes
    +    byte[] unCompArray = new byte[unCompInput.length * 4];
    +    ByteBuffer unCompBuffer = ByteBuffer.wrap(unCompArray);
    +    for (int input : unCompInput) {
    +      unCompBuffer.putInt(input);
    +    }
    +    return Zstd.compress(unCompBuffer.array(), COMPRESS_LEVEL);
    +  }
    +
    +  @Override
    +  public int[] unCompressInt(byte[] compInput, int offset, int length) {
    +    byte[] unCompArray = unCompressByte(compInput, offset, length);
    +    ByteBuffer unCompBuffer = ByteBuffer.wrap(unCompArray);
    +    int[] ints = new int[unCompArray.length / 4];
    +    for (int i = 0; i < ints.length; i++) {
    +      ints[i] = unCompBuffer.getInt();
    +    }
    +    return ints;
    +  }
    +
    +  @Override
    +  public byte[] compressLong(long[] unCompInput) {
    +    // long use 8 bytes
    +    byte[] unCompArray = new byte[unCompInput.length * 8];
    +    ByteBuffer unCompBuffer = ByteBuffer.wrap(unCompArray);
    +    for (long input : unCompInput) {
    +      unCompBuffer.putLong(input);
    +    }
    +    return Zstd.compress(unCompBuffer.array(), COMPRESS_LEVEL);
    +  }
    +
    +  @Override
    +  public long[] unCompressLong(byte[] compInput, int offset, int length) {
    +    byte[] unCompArray = unCompressByte(compInput, offset, length);
    +    ByteBuffer unCompBuffer = ByteBuffer.wrap(unCompArray);
    +    long[] longs = new long[unCompArray.length / 8];
    +    for (int i = 0; i < longs.length; i++) {
    +      longs[i] = unCompBuffer.getLong();
    +    }
    +    return longs;
    +  }
    +
    +  @Override
    +  public byte[] compressFloat(float[] unCompInput) {
    +    // float use 4 bytes
    +    byte[] unCompArray = new byte[unCompInput.length * 4];
    +    ByteBuffer unCompBuffer = ByteBuffer.wrap(unCompArray);
    +    for (float input : unCompInput) {
    +      unCompBuffer.putFloat(input);
    +    }
    +    return Zstd.compress(unCompBuffer.array(), COMPRESS_LEVEL);
    +  }
    +
    +  @Override
    +  public float[] unCompressFloat(byte[] compInput, int offset, int length) {
    +    byte[] unCompArray = unCompressByte(compInput, offset, length);
    +    ByteBuffer unCompBuffer = ByteBuffer.wrap(unCompArray);
    +    float[] floats = new float[unCompArray.length / 4];
    +    for (int i = 0; i < floats.length; i++) {
    +      floats[i] = unCompBuffer.getFloat();
    +    }
    +    return floats;
    +  }
    +
    +  @Override
    +  public byte[] compressDouble(double[] unCompInput) {
    +    // double use 8 bytes
    +    byte[] unCompArray = new byte[unCompInput.length * 8];
    +    ByteBuffer unCompBuffer = ByteBuffer.wrap(unCompArray);
    +    for (double input : unCompInput) {
    +      unCompBuffer.putDouble(input);
    +    }
    +    return Zstd.compress(unCompBuffer.array(), COMPRESS_LEVEL);
    +  }
    +
    +  @Override
    +  public double[] unCompressDouble(byte[] compInput, int offset, int length) {
    +    byte[] unCompArray = unCompressByte(compInput, offset, length);
    +    ByteBuffer unCompBuffer = ByteBuffer.wrap(unCompArray);
    +    double[] doubles = new double[unCompArray.length / 8];
    +    for (int i = 0; i < doubles.length; i++) {
    +      doubles[i] = unCompBuffer.getDouble();
    +    }
    +    return doubles;
    +  }
    +
    +  @Override
    +  public long rawCompress(long inputAddress, int inputSize, long outputAddress) throws IOException {
    +    throw new RuntimeException("Not implemented rawCompress for zstd yet");
    +  }
    +
    +  @Override
    +  public long rawUncompress(byte[] input, byte[] output) throws IOException {
    +    return Zstd.decompress(output, input);
    +  }
    +
    +  @Override
    +  public int maxCompressedLength(int inputSize) {
    +    throw new RuntimeException("Not implemented maxCompressedLength for zstd yet");
    +  }
    +
    --- End diff --
    
    Zstd.compressBound


---