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

[GitHub] carbondata pull request #916: [CARBONDATA-938] Prune partitions for filter q...

GitHub user QiangCai opened a pull request:

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

    [CARBONDATA-938] Prune partitions for filter query on partition column

    1. generate partition filter by basing on pushed-down filter
    
    2. apply partition filter to get the map of required partition (use BitSet)
    support EqualTo, In, GreaterThan/EqualTo(>, >=), LessThan/EqualTo(<, <=)
    
    3. in CarbonInputFormat, use partition map to skip some non-required splits
    
    4. add query test case

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

    $ git pull https://github.com/QiangCai/incubator-carbondata newprunepartition

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

    https://github.com/apache/carbondata/pull/916.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 #916
    
----
commit 32a5ae41a0d4ef0372bbcb662ad78ae2c89fe637
Author: QiangCai <qi...@qq.com>
Date:   2017-05-04T03:26:05Z

    prunepartition

----


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

[GitHub] carbondata pull request #916: [CARBONDATA-938] Prune partitions for filter q...

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

    https://github.com/apache/carbondata/pull/916#discussion_r117394297
  
    --- Diff: core/src/main/java/org/apache/carbondata/core/scan/filter/FilterExpressionProcessor.java ---
    @@ -137,6 +155,119 @@ public FilterResolverIntf getFilterResolver(Expression expressionTree,
       }
     
       /**
    +   * Get the map of required partitions
    +   * The value of "1" in BitSet represent the required partition
    +   * @param expressionTree
    +   * @param partitionInfo
    +   * @param partitioner
    +   * @return
    +   */
    +  @Override public BitSet getFilteredPartitions(Expression expressionTree,
    +      PartitionInfo partitionInfo, Partitioner partitioner) {
    +    return createPartitionFilterTree(expressionTree, partitionInfo).applyFilter(partitioner);
    +  }
    +
    +  /**
    +   * create partition filter by basing on pushed-down filter
    +   * @param expressionTree
    +   * @param partitionInfo
    +   * @return
    +   */
    +  public PartitionFilterIntf createPartitionFilterTree(Expression expressionTree,
    --- End diff --
    
    ok


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

[GitHub] carbondata pull request #916: [CARBONDATA-938] Prune partitions for filter q...

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

    https://github.com/apache/carbondata/pull/916#discussion_r117146059
  
    --- Diff: core/src/main/java/org/apache/carbondata/core/scan/filter/FilterExpressionProcessor.java ---
    @@ -137,6 +155,119 @@ public FilterResolverIntf getFilterResolver(Expression expressionTree,
       }
     
       /**
    +   * Get the map of required partitions
    +   * The value of "1" in BitSet represent the required partition
    +   * @param expressionTree
    +   * @param partitionInfo
    +   * @param partitioner
    +   * @return
    +   */
    +  @Override public BitSet getFilteredPartitions(Expression expressionTree,
    +      PartitionInfo partitionInfo, Partitioner partitioner) {
    +    return createPartitionFilterTree(expressionTree, partitionInfo).applyFilter(partitioner);
    +  }
    +
    +  /**
    +   * create partition filter by basing on pushed-down filter
    +   * @param expressionTree
    +   * @param partitionInfo
    +   * @return
    +   */
    +  public PartitionFilterIntf createPartitionFilterTree(Expression expressionTree,
    --- End diff --
    
    make it private


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

[GitHub] carbondata pull request #916: [CARBONDATA-938] Prune partitions for filter q...

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

    https://github.com/apache/carbondata/pull/916#discussion_r117394103
  
    --- Diff: hadoop/src/main/java/org/apache/carbondata/hadoop/CarbonInputFormat.java ---
    @@ -255,11 +258,32 @@ private static AbsoluteTableIdentifier getAbsoluteTableIdentifier(Configuration
           if (null == carbonTable) {
             throw new IOException("Missing/Corrupt schema file for table.");
           }
    +
           CarbonInputFormatUtil.processFilterExpression(filter, carbonTable);
    +
    +      // prune partitions for filter query on partition table
    +      BitSet partitionMap = null;
    +      if (null != filter) {
    +        PartitionInfo partitionInfo = carbonTable.getPartitionInfo(carbonTable.getFactTableName());
    +        if (null != partitionInfo) {
    +          Partitioner partitioner = PartitionUtil.getPartitioner(partitionInfo);
    +          partitionMap = new FilterExpressionProcessor()
    +              .getFilteredPartitions(filter, partitionInfo, partitioner);
    +          if (partitionMap.cardinality() == 0) {
    +            // no partition is required
    +            return new ArrayList<InputSplit>();
    +          }
    +          if (partitionMap.cardinality() == partitioner.numPartitions()) {
    --- End diff --
    
    If all partitions are matched or not matched, we can release BitSet object.


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

[GitHub] carbondata issue #916: [CARBONDATA-938] Prune partitions for filter query on...

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

    https://github.com/apache/carbondata/pull/916
  
    Build Success with Spark 1.6.2, Please check CI http://136.243.101.176:8080/job/ApacheCarbonPRBuilder/2091/



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

[GitHub] carbondata pull request #916: [CARBONDATA-938] Prune partitions for filter q...

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

    https://github.com/apache/carbondata/pull/916#discussion_r117388756
  
    --- Diff: hadoop/src/main/java/org/apache/carbondata/hadoop/CarbonInputFormat.java ---
    @@ -255,11 +258,32 @@ private static AbsoluteTableIdentifier getAbsoluteTableIdentifier(Configuration
           if (null == carbonTable) {
             throw new IOException("Missing/Corrupt schema file for table.");
           }
    +
           CarbonInputFormatUtil.processFilterExpression(filter, carbonTable);
    +
    +      // prune partitions for filter query on partition table
    +      BitSet partitionMap = null;
    +      if (null != filter) {
    +        PartitionInfo partitionInfo = carbonTable.getPartitionInfo(carbonTable.getFactTableName());
    +        if (null != partitionInfo) {
    +          Partitioner partitioner = PartitionUtil.getPartitioner(partitionInfo);
    +          partitionMap = new FilterExpressionProcessor()
    +              .getFilteredPartitions(filter, partitionInfo, partitioner);
    +          if (partitionMap.cardinality() == 0) {
    +            // no partition is required
    +            return new ArrayList<InputSplit>();
    +          }
    +          if (partitionMap.cardinality() == partitioner.numPartitions()) {
    --- End diff --
    
    I feel it is more readable to use BitSet with all `1` than `null`


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

[GitHub] carbondata pull request #916: [CARBONDATA-938] Prune partitions for filter q...

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

    https://github.com/apache/carbondata/pull/916#discussion_r117386887
  
    --- Diff: core/src/main/java/org/apache/carbondata/core/scan/filter/partition/DefaultPartitionFilterImpl.java ---
    @@ -0,0 +1,43 @@
    +/*
    + * 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.scan.filter.partition;
    +
    +import java.util.BitSet;
    +
    +import org.apache.carbondata.core.scan.partition.PartitionUtil;
    +import org.apache.carbondata.core.scan.partition.Partitioner;
    +
    +/**
    + * the default implement of partition filter
    + */
    +public class DefaultPartitionFilterImpl implements PartitionFilterIntf {
    +
    +  /**
    +   * true: all partitions are required
    +   * false: no partition is required
    +   */
    +  private boolean isContainAll;
    +
    +  public DefaultPartitionFilterImpl(boolean isContainAll) {
    --- End diff --
    
    There is only one instance creation whose `isContainAll` is `false`
    Can you make another implementation for it? I think the name `isContainAll` is not intuitive.
    ```
          case FALSE:
            return new DefaultPartitionFilterImpl(false);
    ```


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

[GitHub] carbondata issue #916: [CARBONDATA-938] Prune partitions for filter query on...

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

    https://github.com/apache/carbondata/pull/916
  
    @jackylk @ravipesala @lionelcao please review pr


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

[GitHub] carbondata issue #916: [CARBONDATA-938] Prune partitions for filter query on...

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

    https://github.com/apache/carbondata/pull/916
  
    Build Success with Spark 1.6.2, Please check CI http://136.243.101.176:8080/job/ApacheCarbonPRBuilder/2048/



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

[GitHub] carbondata pull request #916: [CARBONDATA-938] Prune partitions for filter q...

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

    https://github.com/apache/carbondata/pull/916#discussion_r117394238
  
    --- Diff: hadoop/src/main/java/org/apache/carbondata/hadoop/CarbonInputFormat.java ---
    @@ -255,11 +258,32 @@ private static AbsoluteTableIdentifier getAbsoluteTableIdentifier(Configuration
           if (null == carbonTable) {
             throw new IOException("Missing/Corrupt schema file for table.");
           }
    +
           CarbonInputFormatUtil.processFilterExpression(filter, carbonTable);
    +
    +      // prune partitions for filter query on partition table
    +      BitSet partitionMap = null;
    --- End diff --
    
    ok


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

[GitHub] carbondata issue #916: [CARBONDATA-938] Prune partitions for filter query on...

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

    https://github.com/apache/carbondata/pull/916
  
    Build Success with Spark 1.6.2, Please check CI http://136.243.101.176:8080/job/ApacheCarbonPRBuilder/2095/



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

[GitHub] carbondata pull request #916: [CARBONDATA-938] Prune partitions for filter q...

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

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


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

[GitHub] carbondata issue #916: [CARBONDATA-938] Prune partitions for filter query on...

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

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


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

[GitHub] carbondata pull request #916: [CARBONDATA-938] Prune partitions for filter q...

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

    https://github.com/apache/carbondata/pull/916#discussion_r117387947
  
    --- Diff: hadoop/src/main/java/org/apache/carbondata/hadoop/CarbonInputFormat.java ---
    @@ -255,11 +258,32 @@ private static AbsoluteTableIdentifier getAbsoluteTableIdentifier(Configuration
           if (null == carbonTable) {
             throw new IOException("Missing/Corrupt schema file for table.");
           }
    +
           CarbonInputFormatUtil.processFilterExpression(filter, carbonTable);
    +
    +      // prune partitions for filter query on partition table
    +      BitSet partitionMap = null;
    --- End diff --
    
    suggest to rename all `partitionMap` to `matchedPartitions` in all functions


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

[GitHub] carbondata issue #916: [CARBONDATA-938] Prune partitions for filter query on...

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

    https://github.com/apache/carbondata/pull/916
  
    Build Success with Spark 1.6.2, Please check CI http://136.243.101.176:8080/job/ApacheCarbonPRBuilder/2044/



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

[GitHub] carbondata issue #916: [CARBONDATA-938] Prune partitions for filter query on...

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

    https://github.com/apache/carbondata/pull/916
  
    Build Failed  with Spark 1.6.2, Please check CI http://136.243.101.176:8080/job/ApacheCarbonPRBuilder/2094/



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

[GitHub] carbondata pull request #916: [CARBONDATA-938] Prune partitions for filter q...

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

    https://github.com/apache/carbondata/pull/916#discussion_r117395210
  
    --- Diff: core/src/main/java/org/apache/carbondata/core/scan/filter/partition/DefaultPartitionFilterImpl.java ---
    @@ -0,0 +1,43 @@
    +/*
    + * 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.scan.filter.partition;
    +
    +import java.util.BitSet;
    +
    +import org.apache.carbondata.core.scan.partition.PartitionUtil;
    +import org.apache.carbondata.core.scan.partition.Partitioner;
    +
    +/**
    + * the default implement of partition filter
    + */
    +public class DefaultPartitionFilterImpl implements PartitionFilterIntf {
    +
    +  /**
    +   * true: all partitions are required
    +   * false: no partition is required
    +   */
    +  private boolean isContainAll;
    +
    +  public DefaultPartitionFilterImpl(boolean isContainAll) {
    --- End diff --
    
    We need to create BitSet for each filter. We will invoke logical operate on these BitSets.  It means filters can't share the same BitSet.


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