You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@carbondata.apache.org by jb...@apache.org on 2016/06/23 14:16:19 UTC

[31/56] [abbrv] incubator-carbondata git commit: Refactor org.carbondata.query package (#692)

http://git-wip-us.apache.org/repos/asf/incubator-carbondata/blob/1c725f5b/core/src/main/java/org/carbondata/scan/executor/infos/SortInfo.java
----------------------------------------------------------------------
diff --git a/core/src/main/java/org/carbondata/scan/executor/infos/SortInfo.java b/core/src/main/java/org/carbondata/scan/executor/infos/SortInfo.java
new file mode 100644
index 0000000..53584f5
--- /dev/null
+++ b/core/src/main/java/org/carbondata/scan/executor/infos/SortInfo.java
@@ -0,0 +1,125 @@
+/*
+ * 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.carbondata.scan.executor.infos;
+
+import java.util.List;
+
+import org.carbondata.scan.model.QueryDimension;
+
+/**
+ * Below class holds the order by information about the query
+ */
+public class SortInfo {
+
+  /**
+   * sorting order of a dimension
+   */
+  private byte[] dimensionSortOrder;
+
+  /**
+   * byte range of each dimension present in the order by
+   */
+  private int[][] maskedByteRangeForSorting;
+
+  /**
+   * dimension indexes which is used in order bye
+   */
+  private byte[] sortDimensionIndex;
+
+  /**
+   * mask key of each dimension
+   * this will be used to sort the dimension
+   */
+  private byte[][] dimensionMaskKeyForSorting;
+
+  /**
+   * sortDimension
+   */
+  private List<QueryDimension> sortDimension;
+
+  /**
+   * @return the dimensionSortOrder
+   */
+  public byte[] getDimensionSortOrder() {
+    return dimensionSortOrder;
+  }
+
+  /**
+   * @param dimensionSortOrder the dimensionSortOrder to set
+   */
+  public void setDimensionSortOrder(byte[] dimensionSortOrder) {
+    this.dimensionSortOrder = dimensionSortOrder;
+  }
+
+  /**
+   * @return the maskedByteRangeForSorting
+   */
+  public int[][] getMaskedByteRangeForSorting() {
+    return maskedByteRangeForSorting;
+  }
+
+  /**
+   * @param maskedByteRangeForSorting the maskedByteRangeForSorting to set
+   */
+  public void setMaskedByteRangeForSorting(int[][] maskedByteRangeForSorting) {
+    this.maskedByteRangeForSorting = maskedByteRangeForSorting;
+  }
+
+  /**
+   * @return the sortDimensionIndex
+   */
+  public byte[] getSortDimensionIndex() {
+    return sortDimensionIndex;
+  }
+
+  /**
+   * @param sortDimensionIndex the sortDimensionIndex to set
+   */
+  public void setSortDimensionIndex(byte[] sortDimensionIndex) {
+    this.sortDimensionIndex = sortDimensionIndex;
+  }
+
+  /**
+   * @return the dimensionMaskKeyForSorting
+   */
+  public byte[][] getDimensionMaskKeyForSorting() {
+    return dimensionMaskKeyForSorting;
+  }
+
+  /**
+   * @param dimensionMaskKeyForSorting the dimensionMaskKeyForSorting to set
+   */
+  public void setDimensionMaskKeyForSorting(byte[][] dimensionMaskKeyForSorting) {
+    this.dimensionMaskKeyForSorting = dimensionMaskKeyForSorting;
+  }
+
+  /**
+   * @return the sortDimension
+   */
+  public List<QueryDimension> getSortDimension() {
+    return sortDimension;
+  }
+
+  /**
+   * @param sortDimension the sortDimension to set
+   */
+  public void setSortDimension(List<QueryDimension> sortDimension) {
+    this.sortDimension = sortDimension;
+  }
+}

http://git-wip-us.apache.org/repos/asf/incubator-carbondata/blob/1c725f5b/core/src/main/java/org/carbondata/scan/executor/util/QueryUtil.java
----------------------------------------------------------------------
diff --git a/core/src/main/java/org/carbondata/scan/executor/util/QueryUtil.java b/core/src/main/java/org/carbondata/scan/executor/util/QueryUtil.java
new file mode 100644
index 0000000..a7d98a3
--- /dev/null
+++ b/core/src/main/java/org/carbondata/scan/executor/util/QueryUtil.java
@@ -0,0 +1,718 @@
+/*
+ * 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.carbondata.scan.executor.util;
+
+import java.util.ArrayList;
+import java.util.Arrays;
+import java.util.Collections;
+import java.util.Comparator;
+import java.util.HashMap;
+import java.util.HashSet;
+import java.util.Iterator;
+import java.util.List;
+import java.util.Map;
+import java.util.Map.Entry;
+import java.util.Set;
+import java.util.TreeSet;
+
+import org.carbondata.core.cache.Cache;
+import org.carbondata.core.cache.CacheProvider;
+import org.carbondata.core.cache.CacheType;
+import org.carbondata.core.cache.dictionary.Dictionary;
+import org.carbondata.core.cache.dictionary.DictionaryColumnUniqueIdentifier;
+import org.carbondata.core.carbon.AbsoluteTableIdentifier;
+import org.carbondata.core.carbon.CarbonTableIdentifier;
+import org.carbondata.core.carbon.datastore.block.SegmentProperties;
+import org.carbondata.core.carbon.metadata.CarbonMetadata;
+import org.carbondata.core.carbon.metadata.encoder.Encoding;
+import org.carbondata.core.carbon.metadata.schema.table.CarbonTable;
+import org.carbondata.core.carbon.metadata.schema.table.column.CarbonDimension;
+import org.carbondata.core.carbon.metadata.schema.table.column.CarbonMeasure;
+import org.carbondata.core.constants.CarbonCommonConstants;
+import org.carbondata.core.keygenerator.KeyGenException;
+import org.carbondata.core.keygenerator.KeyGenerator;
+import org.carbondata.core.util.CarbonUtil;
+import org.carbondata.core.util.CarbonUtilException;
+import org.carbondata.scan.executor.exception.QueryExecutionException;
+import org.carbondata.scan.executor.infos.KeyStructureInfo;
+import org.carbondata.scan.model.QueryDimension;
+import org.carbondata.scan.model.QueryMeasure;
+import org.carbondata.scan.model.QueryModel;
+
+import org.apache.commons.lang3.ArrayUtils;
+
+/**
+ * Utility class for query execution
+ */
+public class QueryUtil {
+
+  /**
+   * Below method will be used to get the masked byte range based on the query
+   * dimension. It will give the range in the mdkey. This will be used to get
+   * the actual key array from masked mdkey
+   *
+   * @param queryDimensions query dimension selected in query
+   * @param keyGenerator    key generator
+   * @return masked key
+   */
+  public static int[] getMaskedByteRange(List<QueryDimension> queryDimensions,
+      KeyGenerator keyGenerator) {
+    Set<Integer> byteRangeSet = new TreeSet<Integer>();
+    int[] byteRange = null;
+    for (int i = 0; i < queryDimensions.size(); i++) {
+
+      // as no dictionary column and complex type columns
+      // are not selected in the mdkey
+      // so we will not select the those dimension for calculating the
+      // range
+      if (queryDimensions.get(i).getDimension().getKeyOrdinal() == -1) {
+        continue;
+      }
+      // get the offset of the dimension in the mdkey
+      byteRange =
+          keyGenerator.getKeyByteOffsets(queryDimensions.get(i).getDimension().getKeyOrdinal());
+      for (int j = byteRange[0]; j <= byteRange[1]; j++) {
+        byteRangeSet.add(j);
+      }
+    }
+    int[] maksedByteRange = new int[byteRangeSet.size()];
+    int index = 0;
+    Iterator<Integer> iterator = byteRangeSet.iterator();
+    // add the masked byte range
+    while (iterator.hasNext()) {
+      maksedByteRange[index++] = iterator.next();
+    }
+    return maksedByteRange;
+  }
+
+  public static int[] getMaskedByteRangeBasedOrdinal(List<Integer> ordinals,
+      KeyGenerator keyGenerator) {
+    Set<Integer> byteRangeSet = new TreeSet<Integer>();
+    int[] byteRange = null;
+    for (int i = 0; i < ordinals.size(); i++) {
+
+      // get the offset of the dimension in the mdkey
+      byteRange = keyGenerator.getKeyByteOffsets(ordinals.get(i));
+      for (int j = byteRange[0]; j <= byteRange[1]; j++) {
+        byteRangeSet.add(j);
+      }
+    }
+    int[] maksedByteRange = new int[byteRangeSet.size()];
+    int index = 0;
+    Iterator<Integer> iterator = byteRangeSet.iterator();
+    // add the masked byte range
+    while (iterator.hasNext()) {
+      maksedByteRange[index++] = iterator.next();
+    }
+    return maksedByteRange;
+  }
+
+  /**
+   * Below method will return the max key based on the dimension ordinal
+   *
+   * @param keyOrdinalList
+   * @param generator
+   * @return
+   * @throws KeyGenException
+   */
+  public static byte[] getMaxKeyBasedOnOrinal(List<Integer> keyOrdinalList, KeyGenerator generator)
+      throws KeyGenException {
+    long[] max = new long[generator.getDimCount()];
+    Arrays.fill(max, 0L);
+
+    for (int i = 0; i < keyOrdinalList.size(); i++) {
+      // adding for dimension which is selected in query
+      max[keyOrdinalList.get(i)] = Long.MAX_VALUE;
+    }
+    return generator.generateKey(max);
+  }
+
+  /**
+   * To get the max key based on dimensions. i.e. all other dimensions will be
+   * set to 0 bits and the required query dimension will be masked with all
+   * LONG.MAX so that we can mask key and then compare while aggregating This
+   * can be useful during filter query when only few dimensions were selected
+   * out of row group
+   *
+   * @param queryDimensions dimension selected in query
+   * @param generator       key generator
+   * @return max key for dimension
+   * @throws KeyGenException if any problem while generating the key
+   */
+  public static byte[] getMaxKeyBasedOnDimensions(List<QueryDimension> queryDimensions,
+      KeyGenerator generator) throws KeyGenException {
+    long[] max = new long[generator.getDimCount()];
+    Arrays.fill(max, 0L);
+
+    for (int i = 0; i < queryDimensions.size(); i++) {
+      // as no dictionary column and complex type columns
+      // are not selected in the mdkey
+      // so we will not select the those dimension for calculating the
+      // range
+      if (queryDimensions.get(i).getDimension().getKeyOrdinal() == -1) {
+        continue;
+      }
+      // adding for dimension which is selected in query
+      max[queryDimensions.get(i).getDimension().getKeyOrdinal()] = Long.MAX_VALUE;
+    }
+
+    return generator.generateKey(max);
+  }
+
+  /**
+   * Below method will be used to get the masked key for query
+   *
+   * @param keySize         size of the masked key
+   * @param maskedKeyRanges masked byte range
+   * @return masked bytes
+   */
+  public static int[] getMaskedByte(int keySize, int[] maskedKeyRanges) {
+    int[] maskedKey = new int[keySize];
+    // all the non selected dimension will be filled with -1
+    Arrays.fill(maskedKey, -1);
+    for (int i = 0; i < maskedKeyRanges.length; i++) {
+      maskedKey[maskedKeyRanges[i]] = i;
+    }
+    return maskedKey;
+  }
+
+  /**
+   * Below method will be used to get the dimension block index in file based
+   * on query dimension
+   *
+   * @param queryDimensions                query dimension
+   * @param dimensionOrdinalToBlockMapping mapping of dimension block in file to query dimension
+   * @return block index of file
+   */
+  public static int[] getDimensionsBlockIndexes(List<QueryDimension> queryDimensions,
+      Map<Integer, Integer> dimensionOrdinalToBlockMapping,
+      List<CarbonDimension> customAggregationDimension) {
+    // using set as in row group columns will point to same block
+    Set<Integer> dimensionBlockIndex = new HashSet<Integer>();
+    for (int i = 0; i < queryDimensions.size(); i++) {
+      dimensionBlockIndex.add(
+          dimensionOrdinalToBlockMapping.get(queryDimensions.get(i).getDimension().getOrdinal()));
+    }
+    for (int i = 0; i < customAggregationDimension.size(); i++) {
+      dimensionBlockIndex
+          .add(dimensionOrdinalToBlockMapping.get(customAggregationDimension.get(i).getOrdinal()));
+    }
+    return ArrayUtils
+        .toPrimitive(dimensionBlockIndex.toArray(new Integer[dimensionBlockIndex.size()]));
+  }
+
+  /**
+   * Below method will be used to get the dictionary mapping for all the
+   * dictionary encoded dimension present in the query
+   *
+   * @param queryDimensions            query dimension present in the query this will be used to
+   *                                   convert the result from surrogate key to actual data
+   * @param absoluteTableIdentifier    absolute table identifier
+   * @return dimension unique id to its dictionary map
+   * @throws QueryExecutionException
+   */
+  public static Map<String, Dictionary> getDimensionDictionaryDetail(
+      List<QueryDimension> queryDimensions,
+      AbsoluteTableIdentifier absoluteTableIdentifier) throws QueryExecutionException {
+    // to store dimension unique column id list, this is required as
+    // dimension can be present in
+    // query dimension, as well as some aggregation function will be applied
+    // in the same dimension
+    // so we need to get only one instance of dictionary
+    // direct dictionary skip is done only for the dictionary lookup
+    Set<String> dictionaryDimensionFromQuery = new HashSet<String>();
+    for (int i = 0; i < queryDimensions.size(); i++) {
+      List<Encoding> encodingList = queryDimensions.get(i).getDimension().getEncoder();
+      if (CarbonUtil.hasEncoding(encodingList, Encoding.DICTIONARY) && !CarbonUtil
+          .hasEncoding(encodingList, Encoding.DIRECT_DICTIONARY)) {
+        dictionaryDimensionFromQuery.add(queryDimensions.get(i).getDimension().getColumnId());
+      }
+    }
+    // converting to list as api exposed needed list which i think
+    // is not correct
+    List<String> dictionaryColumnIdList =
+        new ArrayList<String>(dictionaryDimensionFromQuery.size());
+    dictionaryColumnIdList.addAll(dictionaryDimensionFromQuery);
+    return getDictionaryMap(dictionaryColumnIdList, absoluteTableIdentifier);
+  }
+
+  /**
+   * Below method will be used to get the column id to its dictionary mapping
+   *
+   * @param dictionaryColumnIdList  dictionary column list
+   * @param absoluteTableIdentifier absolute table identifier
+   * @return dictionary mapping
+   * @throws QueryExecutionException
+   */
+  private static Map<String, Dictionary> getDictionaryMap(List<String> dictionaryColumnIdList,
+      AbsoluteTableIdentifier absoluteTableIdentifier) throws QueryExecutionException {
+    // this for dictionary unique identifier
+    List<DictionaryColumnUniqueIdentifier> dictionaryColumnUniqueIdentifiers =
+        getDictionaryColumnUniqueIdentifierList(dictionaryColumnIdList,
+            absoluteTableIdentifier.getCarbonTableIdentifier());
+    CacheProvider cacheProvider = CacheProvider.getInstance();
+    Cache forwardDictionaryCache = cacheProvider
+        .createCache(CacheType.FORWARD_DICTIONARY, absoluteTableIdentifier.getStorePath());
+    List<Dictionary> columnDictionaryList = null;
+    try {
+      columnDictionaryList = forwardDictionaryCache.getAll(dictionaryColumnUniqueIdentifiers);
+    } catch (CarbonUtilException e) {
+      throw new QueryExecutionException(e);
+    }
+    Map<String, Dictionary> columnDictionaryMap = new HashMap<>(columnDictionaryList.size());
+    for (int i = 0; i < dictionaryColumnUniqueIdentifiers.size(); i++) {
+      // TODO: null check for column dictionary, if cache size is less it
+      // might return null here, in that case throw exception
+      columnDictionaryMap.put(dictionaryColumnIdList.get(i), columnDictionaryList.get(i));
+    }
+    return columnDictionaryMap;
+  }
+
+  /**
+   * Below method will be used to get the dictionary column unique identifier
+   *
+   * @param dictionaryColumnIdList dictionary
+   * @param carbonTableIdentifier
+   * @return
+   */
+  private static List<DictionaryColumnUniqueIdentifier> getDictionaryColumnUniqueIdentifierList(
+      List<String> dictionaryColumnIdList, CarbonTableIdentifier carbonTableIdentifier) {
+    CarbonTable carbonTable =
+        CarbonMetadata.getInstance().getCarbonTable(carbonTableIdentifier.getTableUniqueName());
+    List<DictionaryColumnUniqueIdentifier> dictionaryColumnUniqueIdentifiers =
+        new ArrayList<>(dictionaryColumnIdList.size());
+    for (String columnIdentifier : dictionaryColumnIdList) {
+      CarbonDimension dimension = CarbonMetadata.getInstance()
+          .getCarbonDimensionBasedOnColIdentifier(carbonTable, columnIdentifier);
+      DictionaryColumnUniqueIdentifier dictionaryColumnUniqueIdentifier =
+          new DictionaryColumnUniqueIdentifier(carbonTableIdentifier, columnIdentifier,
+              dimension.getDataType());
+      dictionaryColumnUniqueIdentifiers.add(dictionaryColumnUniqueIdentifier);
+    }
+    return dictionaryColumnUniqueIdentifiers;
+  }
+
+  /**
+   * Below method will used to get the method will be used to get the measure
+   * block indexes to be read from the file
+   *
+   * @param queryMeasures              query measure
+   * @param expressionMeasure          measure present in the expression
+   * @param ordinalToBlockIndexMapping measure ordinal to block mapping
+   * @return block indexes
+   */
+  public static int[] getMeasureBlockIndexes(List<QueryMeasure> queryMeasures,
+      List<CarbonMeasure> expressionMeasure, Map<Integer, Integer> ordinalToBlockIndexMapping) {
+    Set<Integer> measureBlockIndex = new HashSet<Integer>();
+    for (int i = 0; i < queryMeasures.size(); i++) {
+      measureBlockIndex
+          .add(ordinalToBlockIndexMapping.get(queryMeasures.get(i).getMeasure().getOrdinal()));
+    }
+    for (int i = 0; i < expressionMeasure.size(); i++) {
+      measureBlockIndex.add(ordinalToBlockIndexMapping.get(expressionMeasure.get(i).getOrdinal()));
+    }
+    return ArrayUtils.toPrimitive(measureBlockIndex.toArray(new Integer[measureBlockIndex.size()]));
+  }
+
+  /**
+   * Below method will be used to get the masked byte range for dimension
+   * which is present in order by
+   *
+   * @param orderByDimensions order by dimension
+   * @param generator         key generator
+   * @param maskedRanges      masked byte range for dimension
+   * @return range of masked byte for order by dimension
+   */
+  public static int[][] getMaskedByteRangeForSorting(List<QueryDimension> orderByDimensions,
+      KeyGenerator generator, int[] maskedRanges) {
+    int[][] dimensionCompareIndex = new int[orderByDimensions.size()][];
+    int index = 0;
+    for (int i = 0; i < dimensionCompareIndex.length; i++) {
+      Set<Integer> integers = new TreeSet<Integer>();
+      if (!orderByDimensions.get(i).getDimension().getEncoder().contains(Encoding.DICTIONARY)
+          || orderByDimensions.get(i).getDimension().numberOfChild() > 0) {
+        continue;
+      }
+      int[] range =
+          generator.getKeyByteOffsets(orderByDimensions.get(i).getDimension().getKeyOrdinal());
+      for (int j = range[0]; j <= range[1]; j++) {
+        integers.add(j);
+      }
+      dimensionCompareIndex[index] = new int[integers.size()];
+      int j = 0;
+      for (Iterator<Integer> iterator = integers.iterator(); iterator.hasNext(); ) {
+        Integer integer = (Integer) iterator.next();
+        dimensionCompareIndex[index][j++] = integer.intValue();
+      }
+      index++;
+    }
+    for (int i = 0; i < dimensionCompareIndex.length; i++) {
+      if (null == dimensionCompareIndex[i]) {
+        continue;
+      }
+      int[] range = dimensionCompareIndex[i];
+      if (null != range) {
+        for (int j = 0; j < range.length; j++) {
+          for (int k = 0; k < maskedRanges.length; k++) {
+            if (range[j] == maskedRanges[k]) {
+              range[j] = k;
+              break;
+            }
+          }
+        }
+      }
+
+    }
+    return dimensionCompareIndex;
+  }
+
+  /**
+   * Below method will be used to get the masked key for sorting
+   *
+   * @param orderDimensions           query dimension
+   * @param generator                 key generator
+   * @param maskedByteRangeForSorting masked byte range for sorting
+   * @param maskedRanges              masked range
+   * @return masked byte range
+   * @throws QueryExecutionException
+   */
+  public static byte[][] getMaksedKeyForSorting(List<QueryDimension> orderDimensions,
+      KeyGenerator generator, int[][] maskedByteRangeForSorting, int[] maskedRanges)
+      throws QueryExecutionException {
+    byte[][] maskedKey = new byte[orderDimensions.size()][];
+    byte[] mdKey = null;
+    long[] key = null;
+    byte[] maskedMdKey = null;
+    try {
+      if (null != maskedByteRangeForSorting) {
+        for (int i = 0; i < maskedByteRangeForSorting.length; i++) {
+          if (null == maskedByteRangeForSorting[i]) {
+            continue;
+          }
+          key = new long[generator.getDimCount()];
+          maskedKey[i] = new byte[maskedByteRangeForSorting[i].length];
+          key[orderDimensions.get(i).getDimension().getKeyOrdinal()] = Long.MAX_VALUE;
+          mdKey = generator.generateKey(key);
+          maskedMdKey = new byte[maskedRanges.length];
+          for (int k = 0; k < maskedMdKey.length; k++) { // CHECKSTYLE:OFF
+            // Approval
+            // No:Approval-V1R2C10_001
+            maskedMdKey[k] = mdKey[maskedRanges[k]];
+          }
+          for (int j = 0; j < maskedByteRangeForSorting[i].length; j++) {
+            maskedKey[i][j] = maskedMdKey[maskedByteRangeForSorting[i][j]];
+          }// CHECKSTYLE:ON
+
+        }
+      }
+    } catch (KeyGenException e) {
+      throw new QueryExecutionException(e);
+    }
+    return maskedKey;
+  }
+
+  /**
+   * Below method will be used to get mapping whether dimension is present in
+   * order by or not
+   *
+   * @param sortedDimensions sort dimension present in order by query
+   * @param queryDimensions  query dimension
+   * @return sort dimension indexes
+   */
+  public static byte[] getSortDimensionIndexes(List<QueryDimension> sortedDimensions,
+      List<QueryDimension> queryDimensions) {
+    byte[] sortedDims = new byte[queryDimensions.size()];
+    int indexOf = 0;
+    for (int i = 0; i < sortedDims.length; i++) {
+      indexOf = sortedDimensions.indexOf(queryDimensions.get(i));
+      if (indexOf > -1) {
+        sortedDims[i] = 1;
+      }
+    }
+    return sortedDims;
+  }
+
+  /**
+   * Below method will be used to get the mapping of block index and its
+   * restructuring info
+   *
+   * @param queryDimensions   query dimension from query model
+   * @param segmentProperties segment properties
+   * @return map of block index to its restructuring info
+   * @throws KeyGenException if problem while key generation
+   */
+  public static Map<Integer, KeyStructureInfo> getColumnGroupKeyStructureInfo(
+      List<QueryDimension> queryDimensions, SegmentProperties segmentProperties)
+      throws KeyGenException {
+    Map<Integer, KeyStructureInfo> rowGroupToItsRSInfo = new HashMap<Integer, KeyStructureInfo>();
+    // get column group id and its ordinal mapping of column group
+    Map<Integer, List<Integer>> columnGroupAndItsOrdinalMappingForQuery =
+        getColumnGroupAndItsOrdinalMapping(queryDimensions);
+    KeyGenerator keyGenerator = segmentProperties.getDimensionKeyGenerator();
+
+    Iterator<Entry<Integer, List<Integer>>> iterator =
+        columnGroupAndItsOrdinalMappingForQuery.entrySet().iterator();
+    KeyStructureInfo restructureInfos = null;
+    while (iterator.hasNext()) {
+      Entry<Integer, List<Integer>> next = iterator.next();
+      restructureInfos = new KeyStructureInfo();
+      // sort the ordinal
+      List<Integer> ordinal = next.getValue();
+      Collections.sort(ordinal);
+      // get the masked byte range for column group
+      int[] maskByteRanges = getMaskedByteRangeBasedOrdinal(ordinal, keyGenerator);
+      // max key for column group
+      byte[] maxKey = getMaxKeyBasedOnOrinal(ordinal, keyGenerator);
+      // get masked key for column group
+      int[] maksedByte = getMaskedByte(keyGenerator.getKeySizeInBytes(), maskByteRanges);
+      restructureInfos.setKeyGenerator(keyGenerator);
+      restructureInfos.setMaskByteRanges(maskByteRanges);
+      restructureInfos.setMaxKey(maxKey);
+      restructureInfos.setMaskedBytes(maksedByte);
+      restructureInfos
+          .setBlockMdKeyStartOffset(getBlockMdKeyStartOffset(segmentProperties, ordinal));
+      rowGroupToItsRSInfo
+          .put(segmentProperties.getDimensionOrdinalToBlockMapping().get(ordinal.get(0)),
+              restructureInfos);
+    }
+    return rowGroupToItsRSInfo;
+  }
+
+  /**
+   * It return mdkey start index of given column group
+   * @param segmentProperties
+   * @param ordinal : column group ordinal
+   * @return
+   */
+  public static int getBlockMdKeyStartOffset(SegmentProperties segmentProperties,
+      List<Integer> ordinal) {
+    int[][] colGroups = segmentProperties.getColumnGroups();
+    int blockMdkeyStartOffset = 0;
+    for (int i = 0; i < colGroups.length; i++) {
+      if (QueryUtil.searchInArray(colGroups[i], ordinal.get(0))) {
+        break;
+      }
+      blockMdkeyStartOffset += segmentProperties.getDimensionColumnsValueSize()[i];
+    }
+    return blockMdkeyStartOffset;
+  }
+
+  /**
+   * return true if given key is found in array
+   *
+   * @param data
+   * @param key
+   * @return
+   */
+  public static boolean searchInArray(int[] data, int key) {
+    for (int i = 0; i < data.length; i++) {
+      if (key == data[i]) {
+        return true;
+      }
+    }
+    return false;
+  }
+
+  /**
+   * Below method will be used to create a mapping of column group columns
+   * this mapping will have column group id to all the dimension ordinal
+   * present in the column group This mapping will be used during query
+   * execution, to create a mask key for the column group dimension which will
+   * be used in aggregation and filter query as column group dimension will be
+   * stored in bit level
+   */
+  private static Map<Integer, List<Integer>> getColumnGroupAndItsOrdinalMapping(
+      List<QueryDimension> origdimensions) {
+
+    List<QueryDimension> dimensions = new ArrayList<QueryDimension>(origdimensions.size());
+    dimensions.addAll(origdimensions);
+    /**
+     * sort based on column group id
+     */
+    Collections.sort(dimensions, new Comparator<QueryDimension>() {
+
+      @Override public int compare(QueryDimension o1, QueryDimension o2) {
+        return Integer
+            .compare(o1.getDimension().columnGroupId(), o2.getDimension().columnGroupId());
+      }
+    });
+    // list of row groups this will store all the row group column
+    Map<Integer, List<Integer>> columnGroupAndItsOrdinalsMapping =
+        new HashMap<Integer, List<Integer>>();
+    // to store a column group
+    List<Integer> currentColumnGroup = null;
+    // current index
+    int index = 0;
+    // previous column group to check all the column of row id has bee
+    // selected
+    int prvColumnGroupId = -1;
+    while (index < dimensions.size()) {
+      // if dimension group id is not zero and it is same as the previous
+      // column group id
+      // then we need to add ordinal of that column as it belongs to same
+      // column group
+      if (!dimensions.get(index).getDimension().isColumnar()
+          && dimensions.get(index).getDimension().columnGroupId() == prvColumnGroupId) {
+        currentColumnGroup.add(dimensions.get(index).getDimension().getOrdinal());
+      }
+
+      // if dimension is not a columnar then it is column group column
+      else if (!dimensions.get(index).getDimension().isColumnar()) {
+        currentColumnGroup = new ArrayList<Integer>();
+        columnGroupAndItsOrdinalsMapping
+            .put(dimensions.get(index).getDimension().columnGroupId(), currentColumnGroup);
+        currentColumnGroup.add(dimensions.get(index).getDimension().getOrdinal());
+      }
+      // update the row id every time,this is required to group the
+      // columns
+      // of the same row group
+      prvColumnGroupId = dimensions.get(index).getDimension().columnGroupId();
+      index++;
+    }
+    return columnGroupAndItsOrdinalsMapping;
+  }
+
+  /**
+   * Below method will be used to get masked byte
+   *
+   * @param data           actual data
+   * @param maxKey         max key
+   * @param maskByteRanges mask byte range
+   * @param byteCount
+   * @return masked byte
+   */
+  public static byte[] getMaskedKey(byte[] data, byte[] maxKey, int[] maskByteRanges,
+      int byteCount) {
+    byte[] maskedKey = new byte[byteCount];
+    int counter = 0;
+    int byteRange = 0;
+    for (int i = 0; i < byteCount; i++) {
+      byteRange = maskByteRanges[i];
+      if (byteRange != -1) {
+        maskedKey[counter++] = (byte) (data[byteRange] & maxKey[byteRange]);
+      }
+    }
+    return maskedKey;
+  }
+
+  /**
+   * Below method will be used to fill block indexes of the query dimension
+   * which will be used in creating a output row Here is method we are passing
+   * two list which store the indexes one for dictionary column other for not
+   * dictionary column. This is done for specific purpose so that in one
+   * iteration we will be able to fill both type dimension block indexes
+   *
+   * @param queryDimensions                  dimension present in the query
+   * @param columnOrdinalToBlockIndexMapping column ordinal to block index mapping
+   * @param dictionaryDimensionBlockIndex    list to store dictionary column block indexes
+   * @param noDictionaryDimensionBlockIndex  list to store no dictionary block indexes
+   */
+  public static void fillQueryDimensionsBlockIndexes(List<QueryDimension> queryDimensions,
+      Map<Integer, Integer> columnOrdinalToBlockIndexMapping,
+      Set<Integer> dictionaryDimensionBlockIndex, List<Integer> noDictionaryDimensionBlockIndex) {
+    for (QueryDimension queryDimension : queryDimensions) {
+      if (CarbonUtil.hasEncoding(queryDimension.getDimension().getEncoder(), Encoding.DICTIONARY)) {
+        dictionaryDimensionBlockIndex
+            .add(columnOrdinalToBlockIndexMapping.get(queryDimension.getDimension().getOrdinal()));
+      } else {
+        noDictionaryDimensionBlockIndex
+            .add(columnOrdinalToBlockIndexMapping.get(queryDimension.getDimension().getOrdinal()));
+      }
+    }
+  }
+
+  /**
+   * Below method will be used to resolve the query model
+   * resolve will be setting the actual dimension and measure object
+   * as from driver only column name will be passes to avoid the heavy object
+   * serialization
+   *
+   * @param queryModel query model
+   */
+  public static void resolveQueryModel(QueryModel queryModel) {
+    CarbonMetadata.getInstance().addCarbonTable(queryModel.getTable());
+    // TODO need to load the table from table identifier
+    CarbonTable carbonTable = queryModel.getTable();
+    String tableName =
+        queryModel.getAbsoluteTableIdentifier().getCarbonTableIdentifier().getTableName();
+    // resolve query dimension
+    for (QueryDimension queryDimension : queryModel.getQueryDimension()) {
+      queryDimension
+          .setDimension(carbonTable.getDimensionByName(tableName, queryDimension.getColumnName()));
+    }
+    // resolve sort dimension
+    for (QueryDimension sortDimension : queryModel.getSortDimension()) {
+      sortDimension
+          .setDimension(carbonTable.getDimensionByName(tableName, sortDimension.getColumnName()));
+    }
+    // resolve query measure
+    for (QueryMeasure queryMeasure : queryModel.getQueryMeasures()) {
+      // in case of count start column name will  be count * so
+      // for count start add first measure if measure is not present
+      // than add first dimension as a measure
+      if (queryMeasure.getColumnName().equals("count(*)")) {
+        if (carbonTable.getMeasureByTableName(tableName).size() > 0) {
+          queryMeasure.setMeasure(carbonTable.getMeasureByTableName(tableName).get(0));
+        } else {
+          CarbonMeasure dummyMeasure = new CarbonMeasure(
+              carbonTable.getDimensionByTableName(tableName).get(0).getColumnSchema(), 0);
+          queryMeasure.setMeasure(dummyMeasure);
+        }
+      } else {
+        queryMeasure
+            .setMeasure(carbonTable.getMeasureByName(tableName, queryMeasure.getColumnName()));
+      }
+    }
+    //TODO need to handle expression
+  }
+
+  /**
+   * Below method will be used to get the index of number type aggregator
+   *
+   * @param aggType
+   * @return index in aggregator
+   */
+  public static int[] getNumberTypeIndex(List<String> aggType) {
+    List<Integer> indexList = new ArrayList<Integer>();
+    for (int i = 0; i < aggType.size(); i++) {
+      if (CarbonCommonConstants.SUM.equals(aggType.get(i)) || CarbonCommonConstants.AVERAGE
+          .equals(aggType.get(i))) {
+        indexList.add(i);
+      }
+    }
+    return ArrayUtils.toPrimitive(indexList.toArray(new Integer[indexList.size()]));
+  }
+
+  /**
+   * below method will be used to get the actual type aggregator
+   *
+   * @param aggType
+   * @return index in aggrgetor
+   */
+  public static int[] getActualTypeIndex(List<String> aggType) {
+    List<Integer> indexList = new ArrayList<Integer>();
+    for (int i = 0; i < aggType.size(); i++) {
+      if (!CarbonCommonConstants.SUM.equals(aggType.get(i)) && !CarbonCommonConstants.AVERAGE
+          .equals(aggType.get(i))) {
+        indexList.add(i);
+      }
+    }
+    return ArrayUtils.toPrimitive(indexList.toArray(new Integer[indexList.size()]));
+  }
+}

http://git-wip-us.apache.org/repos/asf/incubator-carbondata/blob/1c725f5b/core/src/main/java/org/carbondata/scan/executor/util/RestructureUtil.java
----------------------------------------------------------------------
diff --git a/core/src/main/java/org/carbondata/scan/executor/util/RestructureUtil.java b/core/src/main/java/org/carbondata/scan/executor/util/RestructureUtil.java
new file mode 100644
index 0000000..92b469c
--- /dev/null
+++ b/core/src/main/java/org/carbondata/scan/executor/util/RestructureUtil.java
@@ -0,0 +1,128 @@
+/*
+ * 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.carbondata.scan.executor.util;
+
+import java.util.ArrayList;
+import java.util.List;
+
+import org.carbondata.core.carbon.metadata.schema.table.column.CarbonDimension;
+import org.carbondata.core.carbon.metadata.schema.table.column.CarbonMeasure;
+import org.carbondata.core.constants.CarbonCommonConstants;
+import org.carbondata.scan.executor.infos.AggregatorInfo;
+import org.carbondata.scan.model.QueryDimension;
+import org.carbondata.scan.model.QueryMeasure;
+
+/**
+ * Utility class for restructuring
+ */
+public class RestructureUtil {
+
+  /**
+   * Below method will be used to get the updated query dimension updation
+   * means, after restructuring some dimension will be not present in older
+   * table blocks in that case we need to select only those dimension out of
+   * query dimension which is present in the current table block
+   *
+   * @param queryDimensions
+   * @param tableBlockDimensions
+   * @return list of query dimension which is present in the table block
+   */
+  public static List<QueryDimension> getUpdatedQueryDimension(
+      List<QueryDimension> queryDimensions, List<CarbonDimension> tableBlockDimensions) {
+    List<QueryDimension> presentDimension =
+        new ArrayList<QueryDimension>(CarbonCommonConstants.DEFAULT_COLLECTION_SIZE);
+    // selecting only those dimension which is present in the query
+    for (QueryDimension queryDimimension : queryDimensions) {
+      for (CarbonDimension tableDimension : tableBlockDimensions) {
+        if (tableDimension.equals(queryDimimension.getDimension())) {
+          presentDimension.add(queryDimimension);
+        }
+      }
+    }
+    return presentDimension;
+  }
+
+  /**
+   * Below method is to add dimension children for complex type dimension as
+   * internally we are creating dimension column for each each complex
+   * dimension so when complex query dimension request will come in the query,
+   * we need to add its children as it is hidden from the user For example if
+   * complex dimension is of Array of String[2] so we are storing 3 dimension
+   * and when user will query for complex type i.e. array type we need to add
+   * its children and then we will read respective block and create a tuple
+   * based on all three dimension
+   *
+   * @param queryDimensions      current query dimensions
+   * @param tableBlockDimensions dimensions which is present in the table block
+   * @return updated dimension(after adding complex type children)
+   */
+  public static List<CarbonDimension> addChildrenForComplexTypeDimension(
+      List<CarbonDimension> queryDimensions, List<CarbonDimension> tableBlockDimensions) {
+    List<CarbonDimension> updatedQueryDimension = new ArrayList<CarbonDimension>();
+    int numberOfChildren = 0;
+    for (CarbonDimension queryDimension : queryDimensions) {
+      // if number of child is zero, then it is not a complex dimension
+      // so directly add it query dimension
+      if (queryDimension.numberOfChild() == 0) {
+        updatedQueryDimension.add(queryDimension);
+      }
+      // if number of child is more than 1 then add all its children
+      numberOfChildren = queryDimension.getOrdinal() + queryDimension.numberOfChild();
+      for (int j = queryDimension.getOrdinal(); j < numberOfChildren; j++) {
+        updatedQueryDimension.add(tableBlockDimensions.get(j));
+      }
+    }
+    return updatedQueryDimension;
+  }
+
+  /**
+   * Below method will be used to get the aggregator info object
+   * in this method some of the properties which will be extracted
+   * from query measure and current block measures will be set
+   *
+   * @param queryMeasures        measures present in query
+   * @param currentBlockMeasures current block measures
+   * @return aggregator info
+   */
+  public static AggregatorInfo getAggregatorInfos(List<QueryMeasure> queryMeasures,
+      List<CarbonMeasure> currentBlockMeasures) {
+    AggregatorInfo aggregatorInfos = new AggregatorInfo();
+    int numberOfMeasureInQuery = queryMeasures.size();
+    int[] measureOrdinals = new int[numberOfMeasureInQuery];
+    Object[] defaultValues = new Object[numberOfMeasureInQuery];
+    boolean[] measureExistsInCurrentBlock = new boolean[numberOfMeasureInQuery];
+    int index = 0;
+    for (QueryMeasure queryMeasure : queryMeasures) {
+      measureOrdinals[index] = queryMeasure.getMeasure().getOrdinal();
+      // if query measure exists in current dimension measures
+      // then setting measure exists is true
+      // otherwise adding a default value of a measure
+      if (currentBlockMeasures.contains(queryMeasure.getMeasure())) {
+        measureExistsInCurrentBlock[index] = true;
+      } else {
+        defaultValues[index] = queryMeasure.getMeasure().getDefaultValue();
+      }
+      index++;
+    }
+    aggregatorInfos.setDefaultValues(defaultValues);
+    aggregatorInfos.setMeasureOrdinals(measureOrdinals);
+    aggregatorInfos.setMeasureExists(measureExistsInCurrentBlock);
+    return aggregatorInfos;
+  }
+}

http://git-wip-us.apache.org/repos/asf/incubator-carbondata/blob/1c725f5b/core/src/main/java/org/carbondata/scan/expression/BinaryExpression.java
----------------------------------------------------------------------
diff --git a/core/src/main/java/org/carbondata/scan/expression/BinaryExpression.java b/core/src/main/java/org/carbondata/scan/expression/BinaryExpression.java
new file mode 100644
index 0000000..1ad334e
--- /dev/null
+++ b/core/src/main/java/org/carbondata/scan/expression/BinaryExpression.java
@@ -0,0 +1,59 @@
+/*
+ * 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.carbondata.scan.expression;
+
+public abstract class BinaryExpression extends Expression {
+
+  /**
+   *
+   */
+  private static final long serialVersionUID = 1L;
+  /**
+   *
+   */
+
+  protected Expression left;
+  protected Expression right;
+  protected boolean isRangeExpression;
+
+  public BinaryExpression(Expression left, Expression right) {
+    this.left = left;
+    this.right = right;
+    children.add(left);
+    children.add(right);
+  }
+
+  public Expression getLeft() {
+    return left;
+  }
+
+  public Expression getRight() {
+    return right;
+  }
+
+  public boolean isRangeExpression() {
+    return isRangeExpression;
+  }
+
+  public void setRangeExpression(boolean isRangeExpression) {
+    this.isRangeExpression = isRangeExpression;
+  }
+
+}

http://git-wip-us.apache.org/repos/asf/incubator-carbondata/blob/1c725f5b/core/src/main/java/org/carbondata/scan/expression/ColumnExpression.java
----------------------------------------------------------------------
diff --git a/core/src/main/java/org/carbondata/scan/expression/ColumnExpression.java b/core/src/main/java/org/carbondata/scan/expression/ColumnExpression.java
new file mode 100644
index 0000000..922e706
--- /dev/null
+++ b/core/src/main/java/org/carbondata/scan/expression/ColumnExpression.java
@@ -0,0 +1,112 @@
+/*
+ * 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.carbondata.scan.expression;
+
+import org.carbondata.core.carbon.metadata.schema.table.column.CarbonColumn;
+import org.carbondata.core.carbon.metadata.schema.table.column.CarbonDimension;
+import org.carbondata.scan.filter.intf.ExpressionType;
+import org.carbondata.scan.filter.intf.RowIntf;
+
+public class ColumnExpression extends LeafExpression {
+
+  private static final long serialVersionUID = 1L;
+
+  private String columnName;
+
+  private boolean isDimension;
+
+  private int colIndex = -1;
+
+  private DataType dataType;
+
+  private CarbonDimension dimension;
+
+  private CarbonColumn carbonColumn;
+
+  public ColumnExpression(String columnName, DataType dataType) {
+    this.columnName = columnName;
+    this.dataType = dataType;
+
+  }
+
+  public CarbonDimension getDimension() {
+    return dimension;
+  }
+
+  public void setDimension(CarbonDimension dimension) {
+    this.dimension = dimension;
+  }
+
+  public String getColumnName() {
+    return columnName;
+  }
+
+  public void setColumnName(String columnName) {
+    this.columnName = columnName;
+  }
+
+  public boolean isDimension() {
+    return isDimension;
+  }
+
+  public void setDimension(boolean isDimension) {
+    this.isDimension = isDimension;
+  }
+
+  public int getColIndex() {
+    return colIndex;
+  }
+
+  public void setColIndex(int colIndex) {
+    this.colIndex = colIndex;
+  }
+
+  public DataType getDataType() {
+    return dataType;
+  }
+
+  public void setDataType(DataType dataType) {
+    this.dataType = dataType;
+  }
+
+  @Override public ExpressionResult evaluate(RowIntf value) {
+    ExpressionResult expressionResult = new ExpressionResult(dataType, value.getVal(colIndex));
+    return expressionResult;
+  }
+
+  @Override public ExpressionType getFilterExpressionType() {
+    // TODO Auto-generated method stub
+    return null;
+  }
+
+  @Override public String getString() {
+    // TODO Auto-generated method stub
+    return "ColumnExpression(" + columnName + ')';
+  }
+
+  public CarbonColumn getCarbonColumn() {
+    return carbonColumn;
+  }
+
+  public void setCarbonColumn(CarbonColumn carbonColumn) {
+    this.carbonColumn = carbonColumn;
+  }
+
+}

http://git-wip-us.apache.org/repos/asf/incubator-carbondata/blob/1c725f5b/core/src/main/java/org/carbondata/scan/expression/DataType.java
----------------------------------------------------------------------
diff --git a/core/src/main/java/org/carbondata/scan/expression/DataType.java b/core/src/main/java/org/carbondata/scan/expression/DataType.java
new file mode 100644
index 0000000..fc9d0bf
--- /dev/null
+++ b/core/src/main/java/org/carbondata/scan/expression/DataType.java
@@ -0,0 +1,34 @@
+/*
+ * 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.carbondata.scan.expression;
+
+public enum DataType {
+  StringType(0), DateType(1), TimestampType(2), BooleanType(1), IntegerType(3), FloatType(
+      4), LongType(5), DoubleType(6), NullType(7), DecimalType(8), ArrayType(9), StructType(10);
+  private int presedenceOrder;
+
+  private DataType(int value) {
+    this.presedenceOrder = value;
+  }
+
+  public int getPresedenceOrder() {
+    return presedenceOrder;
+  }
+}

http://git-wip-us.apache.org/repos/asf/incubator-carbondata/blob/1c725f5b/core/src/main/java/org/carbondata/scan/expression/Expression.java
----------------------------------------------------------------------
diff --git a/core/src/main/java/org/carbondata/scan/expression/Expression.java b/core/src/main/java/org/carbondata/scan/expression/Expression.java
new file mode 100644
index 0000000..01b4fee
--- /dev/null
+++ b/core/src/main/java/org/carbondata/scan/expression/Expression.java
@@ -0,0 +1,48 @@
+/*
+ * 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.carbondata.scan.expression;
+
+import java.io.Serializable;
+import java.util.ArrayList;
+import java.util.List;
+
+import org.carbondata.core.constants.CarbonCommonConstants;
+import org.carbondata.scan.expression.exception.FilterUnsupportedException;
+import org.carbondata.scan.filter.intf.ExpressionType;
+import org.carbondata.scan.filter.intf.RowIntf;
+
+public abstract class Expression implements Serializable {
+
+  private static final long serialVersionUID = -7568676723039530713L;
+  protected List<Expression> children =
+      new ArrayList<Expression>(CarbonCommonConstants.DEFAULT_COLLECTION_SIZE);
+
+  public abstract ExpressionResult evaluate(RowIntf value) throws FilterUnsupportedException;
+
+  public abstract ExpressionType getFilterExpressionType();
+
+  public List<Expression> getChildren() {
+    return children;
+  }
+
+  public abstract String getString();
+
+  // public abstract void  accept(ExpressionVisitor visitor);
+}

http://git-wip-us.apache.org/repos/asf/incubator-carbondata/blob/1c725f5b/core/src/main/java/org/carbondata/scan/expression/ExpressionResult.java
----------------------------------------------------------------------
diff --git a/core/src/main/java/org/carbondata/scan/expression/ExpressionResult.java b/core/src/main/java/org/carbondata/scan/expression/ExpressionResult.java
new file mode 100644
index 0000000..decbdc0
--- /dev/null
+++ b/core/src/main/java/org/carbondata/scan/expression/ExpressionResult.java
@@ -0,0 +1,413 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additiona   l 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.carbondata.scan.expression;
+
+import java.math.BigDecimal;
+import java.sql.Timestamp;
+import java.text.ParseException;
+import java.text.SimpleDateFormat;
+import java.util.ArrayList;
+import java.util.Date;
+import java.util.List;
+
+import org.carbondata.core.constants.CarbonCommonConstants;
+import org.carbondata.core.util.CarbonProperties;
+import org.carbondata.scan.expression.exception.FilterUnsupportedException;
+
+public class ExpressionResult implements Comparable<ExpressionResult> {
+
+  private static final long serialVersionUID = 1L;
+  protected DataType dataType;
+
+  protected Object value;
+
+  private List<ExpressionResult> expressionResults;
+
+  public ExpressionResult(DataType dataType, Object value) {
+    this.dataType = dataType;
+    this.value = value;
+  }
+
+  public ExpressionResult(List<ExpressionResult> expressionResults) {
+    this.expressionResults = expressionResults;
+  }
+
+  public void set(DataType dataType, Object value) {
+    this.dataType = dataType;
+    this.value = value;
+    this.expressionResults = null;
+  }
+
+  public DataType getDataType() {
+    return dataType;
+  }
+
+  //CHECKSTYLE:OFF Approval No:Approval-V1R2C10_009
+  public Integer getInt() throws FilterUnsupportedException {
+    if (value == null) {
+      return null;
+    }
+    try {
+      switch (this.getDataType()) {
+        case StringType:
+          try {
+            return Integer.parseInt(value.toString());
+          } catch (NumberFormatException e) {
+            throw new FilterUnsupportedException(e);
+          }
+
+        case IntegerType:
+        case DoubleType:
+
+          if (value instanceof Double) {
+            return ((Double) value).intValue();
+          }
+          return (Integer) value;
+
+        case TimestampType:
+
+          if (value instanceof Timestamp) {
+            return (int) (((Timestamp) value).getTime() % 1000);
+          } else {
+            return (Integer) value;
+          }
+
+        default:
+          throw new FilterUnsupportedException(
+              "Cannot convert" + this.getDataType().name() + " to integer type value");
+      }
+
+    } catch (ClassCastException e) {
+      throw new FilterUnsupportedException(
+          "Cannot convert" + this.getDataType().name() + " to Integer type value");
+    }
+  }
+
+  public String getString() {
+    if (value == null) {
+      return null;
+    }
+    switch (this.getDataType()) {
+      case TimestampType:
+        SimpleDateFormat parser = new SimpleDateFormat(CarbonProperties.getInstance()
+            .getProperty(CarbonCommonConstants.CARBON_TIMESTAMP_FORMAT,
+                CarbonCommonConstants.CARBON_TIMESTAMP_DEFAULT_FORMAT));
+        if (value instanceof Timestamp) {
+          return parser.format((Timestamp) value);
+        } else {
+          return parser.format(new Timestamp((long) value / 1000));
+        }
+
+      default:
+        return value.toString();
+    }
+  }
+
+  public Double getDouble() throws FilterUnsupportedException {
+    if (value == null) {
+      return null;
+    }
+    try {
+      switch (this.getDataType()) {
+        case StringType:
+          try {
+            return Double.parseDouble(value.toString());
+          } catch (NumberFormatException e) {
+            throw new FilterUnsupportedException(e);
+          }
+
+        case IntegerType:
+          return ((Integer) value).doubleValue();
+        case LongType:
+          return ((Long) value).doubleValue();
+        case DoubleType:
+          return (Double) value;
+        case TimestampType:
+          if (value instanceof Timestamp) {
+            return (double) ((Timestamp) value).getTime() * 1000;
+          } else {
+            return (Double) (value);
+          }
+        default:
+          throw new FilterUnsupportedException(
+              "Cannot convert" + this.getDataType().name() + " to double type value");
+      }
+    } catch (ClassCastException e) {
+      throw new FilterUnsupportedException(
+          "Cannot convert" + this.getDataType().name() + " to Double type value");
+    }
+  }
+  //CHECKSTYLE:ON
+
+  public Long getLong() throws FilterUnsupportedException {
+    if (value == null) {
+      return null;
+    }
+    try {
+      switch (this.getDataType()) {
+        case StringType:
+          try {
+            return Long.parseLong(value.toString());
+          } catch (NumberFormatException e) {
+            throw new FilterUnsupportedException(e);
+          }
+
+        case IntegerType:
+          return (Long) value;
+        case LongType:
+          return (Long) value;
+        case DoubleType:
+          return (Long) value;
+        case TimestampType:
+          if (value instanceof Timestamp) {
+            return 1000 * ((Timestamp) value).getTime();
+          } else {
+            return (Long) value;
+          }
+        default:
+          throw new FilterUnsupportedException(
+              "Cannot convert" + this.getDataType().name() + " to Long type value");
+      }
+    } catch (ClassCastException e) {
+      throw new FilterUnsupportedException(
+          "Cannot convert" + this.getDataType().name() + " to Long type value");
+    }
+
+  }
+
+  //Add to judge for BigDecimal
+  public BigDecimal getDecimal() throws FilterUnsupportedException {
+    if (value == null) {
+      return null;
+    }
+    try {
+      switch (this.getDataType()) {
+        case StringType:
+          try {
+            return new BigDecimal(value.toString());
+          } catch (NumberFormatException e) {
+            throw new FilterUnsupportedException(e);
+          }
+
+        case IntegerType:
+          return new BigDecimal((int) value);
+        case LongType:
+          return new BigDecimal((long) value);
+        case DoubleType:
+          return new BigDecimal((double) value);
+        case DecimalType:
+          return new BigDecimal(value.toString());
+        case TimestampType:
+          if (value instanceof Timestamp) {
+            return new BigDecimal(1000 * ((Timestamp) value).getTime());
+          } else {
+            return new BigDecimal((long) value);
+          }
+        default:
+          throw new FilterUnsupportedException(
+              "Cannot convert" + this.getDataType().name() + " to Long type value");
+      }
+    } catch (ClassCastException e) {
+      throw new FilterUnsupportedException(
+          "Cannot convert" + this.getDataType().name() + " to Long type value");
+    }
+
+  }
+
+  public Long getTime() throws FilterUnsupportedException {
+    if (value == null) {
+      return null;
+    }
+    try {
+      switch (this.getDataType()) {
+        case StringType:
+          SimpleDateFormat parser = new SimpleDateFormat(CarbonProperties.getInstance()
+              .getProperty(CarbonCommonConstants.CARBON_TIMESTAMP_FORMAT,
+                  CarbonCommonConstants.CARBON_TIMESTAMP_DEFAULT_FORMAT));
+          Date dateToStr;
+          try {
+            dateToStr = parser.parse(value.toString());
+            return dateToStr.getTime() * 1000;
+          } catch (ParseException e) {
+            throw new FilterUnsupportedException(
+                "Cannot convert" + this.getDataType().name() + " to Time/Long type value");
+          }
+        case IntegerType:
+        case LongType:
+          return (Long) value;
+        case DoubleType:
+          return (Long) value;
+        case TimestampType:
+          if (value instanceof Timestamp) {
+            return ((Timestamp) value).getTime() * 1000;
+          } else {
+            return (Long) value;
+          }
+        default:
+          throw new FilterUnsupportedException(
+              "Cannot convert" + this.getDataType().name() + " to Time/Long type value");
+      }
+    } catch (ClassCastException e) {
+      throw new FilterUnsupportedException(
+          "Cannot convert" + this.getDataType().name() + " to Time/Long type value");
+    }
+
+  }
+
+  public Boolean getBoolean() throws FilterUnsupportedException {
+    if (value == null) {
+      return null;
+    }
+    try {
+      switch (this.getDataType()) {
+        case StringType:
+          try {
+            return Boolean.parseBoolean(value.toString());
+          } catch (NumberFormatException e) {
+            throw new FilterUnsupportedException(e);
+          }
+
+        case BooleanType:
+          return Boolean.parseBoolean(value.toString());
+
+        default:
+          throw new FilterUnsupportedException(
+              "Cannot convert" + this.getDataType().name() + " to boolean type value");
+      }
+    } catch (ClassCastException e) {
+      throw new FilterUnsupportedException(
+          "Cannot convert" + this.getDataType().name() + " to Boolean type value");
+    }
+  }
+
+  public List<ExpressionResult> getList() {
+    if (null == expressionResults) {
+      List<ExpressionResult> a = new ArrayList<ExpressionResult>(20);
+      a.add(new ExpressionResult(dataType, value));
+      return a;
+    } else {
+      return expressionResults;
+    }
+  }
+
+  public List<String> getListAsString() {
+    List<String> evaluateResultListFinal = new ArrayList<String>(20);
+    List<ExpressionResult> evaluateResultList = getList();
+    for (ExpressionResult result : evaluateResultList) {
+      if (result.getString() == null) {
+        evaluateResultListFinal.add(CarbonCommonConstants.MEMBER_DEFAULT_VAL);
+        continue;
+      }
+      evaluateResultListFinal.add(result.getString());
+    }
+    return evaluateResultListFinal;
+  }
+
+  @Override public int hashCode() {
+    final int prime = 31;
+    int result = 1;
+    if (null != expressionResults) {
+      result = prime * result + expressionResults.hashCode();
+    } else if (null != value) {
+      result = prime * result + value.toString().hashCode();
+    } else {
+      result = prime * result + "".hashCode();
+    }
+
+    return result;
+  }
+
+  @Override public boolean equals(Object obj) {
+    if (!(obj instanceof ExpressionResult)) {
+      return false;
+    }
+    if (this == obj) {
+      return true;
+    }
+    if (getClass() != obj.getClass()) {
+      return false;
+    }
+    ExpressionResult objToCompare = (ExpressionResult) obj;
+    boolean result = false;
+    if (this.value == objToCompare.value) {
+      return true;
+    }
+    try {
+      switch (this.getDataType()) {
+        case StringType:
+          result = this.getString().equals(objToCompare.getString());
+          break;
+        case IntegerType:
+          result = this.getInt().equals(objToCompare.getInt());
+          break;
+
+        case DoubleType:
+          result = this.getDouble().equals(objToCompare.getDouble());
+          break;
+        case TimestampType:
+          result = this.getLong().equals(objToCompare.getLong());
+          break;
+        default:
+          break;
+      }
+    } catch (FilterUnsupportedException ex) {
+      return false;
+    }
+
+    return result;
+  }
+
+  public boolean isNull() {
+    return value == null;
+  }
+
+  @Override public int compareTo(ExpressionResult o) {
+    try {
+      switch (o.dataType) {
+        case IntegerType:
+        case LongType:
+        case DoubleType:
+
+          Double d1 = this.getDouble();
+          Double d2 = o.getDouble();
+          return d1.compareTo(d2);
+        case DecimalType:
+          java.math.BigDecimal val1 = this.getDecimal();
+          java.math.BigDecimal val2 = o.getDecimal();
+          return val1.compareTo(val2);
+        case TimestampType:
+          SimpleDateFormat parser = new SimpleDateFormat(CarbonProperties.getInstance()
+              .getProperty(CarbonCommonConstants.CARBON_TIMESTAMP_FORMAT,
+                  CarbonCommonConstants.CARBON_TIMESTAMP_DEFAULT_FORMAT));
+          Date date1 = null;
+          Date date2 = null;
+          date1 = parser.parse(this.getString());
+          date2 = parser.parse(o.getString());
+          return date1.compareTo(date2);
+        case StringType:
+        default:
+          return this.getString().compareTo(o.getString());
+      }
+    } catch (Exception e) {
+      return -1;
+    }
+  }
+
+}

http://git-wip-us.apache.org/repos/asf/incubator-carbondata/blob/1c725f5b/core/src/main/java/org/carbondata/scan/expression/LeafExpression.java
----------------------------------------------------------------------
diff --git a/core/src/main/java/org/carbondata/scan/expression/LeafExpression.java b/core/src/main/java/org/carbondata/scan/expression/LeafExpression.java
new file mode 100644
index 0000000..2392910
--- /dev/null
+++ b/core/src/main/java/org/carbondata/scan/expression/LeafExpression.java
@@ -0,0 +1,24 @@
+/*
+ * 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.carbondata.scan.expression;
+
+public abstract class LeafExpression extends Expression {
+
+}

http://git-wip-us.apache.org/repos/asf/incubator-carbondata/blob/1c725f5b/core/src/main/java/org/carbondata/scan/expression/LiteralExpression.java
----------------------------------------------------------------------
diff --git a/core/src/main/java/org/carbondata/scan/expression/LiteralExpression.java b/core/src/main/java/org/carbondata/scan/expression/LiteralExpression.java
new file mode 100644
index 0000000..edbaf44
--- /dev/null
+++ b/core/src/main/java/org/carbondata/scan/expression/LiteralExpression.java
@@ -0,0 +1,68 @@
+/*
+ * 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.carbondata.scan.expression;
+
+import org.carbondata.scan.filter.intf.ExpressionType;
+import org.carbondata.scan.filter.intf.RowIntf;
+
+public class LiteralExpression extends LeafExpression {
+
+  /**
+   *
+   */
+  private static final long serialVersionUID = 1L;
+  private Object value;
+  private DataType dataType;
+
+  public LiteralExpression(Object value, DataType dataType) {
+    this.value = value;
+    this.dataType = dataType;
+  }
+
+  @Override public ExpressionResult evaluate(RowIntf value) {
+    ExpressionResult expressionResult = new ExpressionResult(dataType, this.value);
+    return expressionResult;
+  }
+
+  public ExpressionResult getExpressionResult() {
+    ExpressionResult expressionResult = new ExpressionResult(dataType, this.value);
+    return expressionResult;
+  }
+
+  @Override public ExpressionType getFilterExpressionType() {
+    // TODO Auto-generated method stub
+    return ExpressionType.LITERAL;
+  }
+
+  @Override public String getString() {
+    // TODO Auto-generated method stub
+    return "LiteralExpression(" + value + ')';
+  }
+
+  /**
+   * getLiteralExpDataType.
+   *
+   * @return
+   */
+  public DataType getLiteralExpDataType() {
+    return dataType;
+  }
+
+}

http://git-wip-us.apache.org/repos/asf/incubator-carbondata/blob/1c725f5b/core/src/main/java/org/carbondata/scan/expression/UnaryExpression.java
----------------------------------------------------------------------
diff --git a/core/src/main/java/org/carbondata/scan/expression/UnaryExpression.java b/core/src/main/java/org/carbondata/scan/expression/UnaryExpression.java
new file mode 100644
index 0000000..0449b66
--- /dev/null
+++ b/core/src/main/java/org/carbondata/scan/expression/UnaryExpression.java
@@ -0,0 +1,33 @@
+/*
+ * 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.carbondata.scan.expression;
+
+public abstract class UnaryExpression extends Expression {
+
+  private static final long serialVersionUID = 1L;
+  protected Expression child;
+
+  public UnaryExpression(Expression child) {
+    this.child = child;
+    children.add(child);
+
+  }
+
+}

http://git-wip-us.apache.org/repos/asf/incubator-carbondata/blob/1c725f5b/core/src/main/java/org/carbondata/scan/expression/UnknownExpression.java
----------------------------------------------------------------------
diff --git a/core/src/main/java/org/carbondata/scan/expression/UnknownExpression.java b/core/src/main/java/org/carbondata/scan/expression/UnknownExpression.java
new file mode 100644
index 0000000..3f24e1c
--- /dev/null
+++ b/core/src/main/java/org/carbondata/scan/expression/UnknownExpression.java
@@ -0,0 +1,9 @@
+package org.carbondata.scan.expression;
+
+import java.util.List;
+
+public abstract class UnknownExpression extends Expression {
+
+  public abstract List<ColumnExpression> getAllColumnList();
+
+}

http://git-wip-us.apache.org/repos/asf/incubator-carbondata/blob/1c725f5b/core/src/main/java/org/carbondata/scan/expression/arithmetic/AddExpression.java
----------------------------------------------------------------------
diff --git a/core/src/main/java/org/carbondata/scan/expression/arithmetic/AddExpression.java b/core/src/main/java/org/carbondata/scan/expression/arithmetic/AddExpression.java
new file mode 100644
index 0000000..e7ce159
--- /dev/null
+++ b/core/src/main/java/org/carbondata/scan/expression/arithmetic/AddExpression.java
@@ -0,0 +1,81 @@
+/*
+ * 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.carbondata.scan.expression.arithmetic;
+
+import org.carbondata.scan.expression.DataType;
+import org.carbondata.scan.expression.Expression;
+import org.carbondata.scan.expression.ExpressionResult;
+import org.carbondata.scan.expression.exception.FilterUnsupportedException;
+import org.carbondata.scan.filter.intf.ExpressionType;
+import org.carbondata.scan.filter.intf.RowIntf;
+
+public class AddExpression extends BinaryArithmeticExpression {
+  private static final long serialVersionUID = 7999436055420911612L;
+
+  public AddExpression(Expression left, Expression right) {
+    super(left, right);
+  }
+
+  @Override public ExpressionResult evaluate(RowIntf value) throws FilterUnsupportedException {
+    ExpressionResult addExprLeftRes = left.evaluate(value);
+    ExpressionResult addExprRightRes = right.evaluate(value);
+    ExpressionResult val1 = addExprLeftRes;
+    ExpressionResult val2 = addExprRightRes;
+    if (addExprLeftRes.isNull() || addExprRightRes.isNull()) {
+      addExprLeftRes.set(addExprLeftRes.getDataType(), null);
+      return addExprLeftRes;
+    }
+
+    if (addExprLeftRes.getDataType() != addExprRightRes.getDataType()) {
+      if (addExprLeftRes.getDataType().getPresedenceOrder() < addExprRightRes.getDataType()
+          .getPresedenceOrder()) {
+        val2 = addExprLeftRes;
+        val1 = addExprRightRes;
+      }
+    }
+    switch (val1.getDataType()) {
+      case StringType:
+      case DoubleType:
+        addExprRightRes.set(DataType.DoubleType, val1.getDouble() + val2.getDouble());
+        break;
+      case IntegerType:
+        addExprRightRes.set(DataType.IntegerType, val1.getInt() + val2.getInt());
+        break;
+      case LongType:
+        addExprRightRes.set(DataType.LongType, val1.getLong() + val2.getLong());
+        break;
+      case DecimalType:
+        addExprRightRes.set(DataType.DecimalType, val1.getDecimal().add(val2.getDecimal()));
+        break;
+      default:
+        throw new FilterUnsupportedException(
+            "Incompatible datatype for applying Add Expression Filter " + val1.getDataType());
+    }
+    return addExprRightRes;
+  }
+
+  @Override public ExpressionType getFilterExpressionType() {
+    return ExpressionType.ADD;
+  }
+
+  @Override public String getString() {
+    return "Add(" + left.getString() + ',' + right.getString() + ',';
+  }
+}

http://git-wip-us.apache.org/repos/asf/incubator-carbondata/blob/1c725f5b/core/src/main/java/org/carbondata/scan/expression/arithmetic/BinaryArithmeticExpression.java
----------------------------------------------------------------------
diff --git a/core/src/main/java/org/carbondata/scan/expression/arithmetic/BinaryArithmeticExpression.java b/core/src/main/java/org/carbondata/scan/expression/arithmetic/BinaryArithmeticExpression.java
new file mode 100644
index 0000000..9c109f7
--- /dev/null
+++ b/core/src/main/java/org/carbondata/scan/expression/arithmetic/BinaryArithmeticExpression.java
@@ -0,0 +1,34 @@
+/*
+ * 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.carbondata.scan.expression.arithmetic;
+
+import org.carbondata.scan.expression.BinaryExpression;
+import org.carbondata.scan.expression.Expression;
+
+public abstract class BinaryArithmeticExpression extends BinaryExpression {
+
+  private static final long serialVersionUID = 1L;
+
+  public BinaryArithmeticExpression(Expression left, Expression right) {
+    super(left, right);
+    // TODO Auto-generated constructor stub
+  }
+
+}

http://git-wip-us.apache.org/repos/asf/incubator-carbondata/blob/1c725f5b/core/src/main/java/org/carbondata/scan/expression/arithmetic/DivideExpression.java
----------------------------------------------------------------------
diff --git a/core/src/main/java/org/carbondata/scan/expression/arithmetic/DivideExpression.java b/core/src/main/java/org/carbondata/scan/expression/arithmetic/DivideExpression.java
new file mode 100644
index 0000000..f4df604
--- /dev/null
+++ b/core/src/main/java/org/carbondata/scan/expression/arithmetic/DivideExpression.java
@@ -0,0 +1,81 @@
+/*
+ * 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.carbondata.scan.expression.arithmetic;
+
+import org.carbondata.scan.expression.DataType;
+import org.carbondata.scan.expression.Expression;
+import org.carbondata.scan.expression.ExpressionResult;
+import org.carbondata.scan.expression.exception.FilterUnsupportedException;
+import org.carbondata.scan.filter.intf.ExpressionType;
+import org.carbondata.scan.filter.intf.RowIntf;
+
+public class DivideExpression extends BinaryArithmeticExpression {
+  private static final long serialVersionUID = -7269266926782365612L;
+
+  public DivideExpression(Expression left, Expression right) {
+    super(left, right);
+  }
+
+  @Override public ExpressionResult evaluate(RowIntf value) throws FilterUnsupportedException {
+    ExpressionResult divideExprLeftRes = left.evaluate(value);
+    ExpressionResult divideExprRightRes = right.evaluate(value);
+    ExpressionResult val1 = divideExprLeftRes;
+    ExpressionResult val2 = divideExprRightRes;
+    if (divideExprLeftRes.isNull() || divideExprRightRes.isNull()) {
+      divideExprLeftRes.set(divideExprLeftRes.getDataType(), null);
+      return divideExprLeftRes;
+    }
+    if (divideExprLeftRes.getDataType() != divideExprRightRes.getDataType()) {
+      if (divideExprLeftRes.getDataType().getPresedenceOrder() < divideExprRightRes.getDataType()
+          .getPresedenceOrder()) {
+        val2 = divideExprLeftRes;
+        val1 = divideExprRightRes;
+      }
+    }
+    switch (val1.getDataType()) {
+      case StringType:
+      case DoubleType:
+        divideExprRightRes.set(DataType.DoubleType, val1.getDouble() / val2.getDouble());
+        break;
+      case IntegerType:
+        divideExprRightRes.set(DataType.IntegerType, val1.getInt() / val2.getInt());
+        break;
+      case LongType:
+        divideExprRightRes.set(DataType.LongType, val1.getLong() / val2.getLong());
+        break;
+      case DecimalType:
+        divideExprRightRes.set(DataType.DecimalType, val1.getDecimal().divide(val2.getDecimal()));
+        break;
+      default:
+        throw new FilterUnsupportedException(
+            "Incompatible datatype for applying Add Expression Filter " + divideExprLeftRes
+                .getDataType());
+    }
+    return divideExprRightRes;
+  }
+
+  @Override public ExpressionType getFilterExpressionType() {
+    return ExpressionType.DIVIDE;
+  }
+
+  @Override public String getString() {
+    return "Divide(" + left.getString() + ',' + right.getString() + ')';
+  }
+}

http://git-wip-us.apache.org/repos/asf/incubator-carbondata/blob/1c725f5b/core/src/main/java/org/carbondata/scan/expression/arithmetic/MultiplyExpression.java
----------------------------------------------------------------------
diff --git a/core/src/main/java/org/carbondata/scan/expression/arithmetic/MultiplyExpression.java b/core/src/main/java/org/carbondata/scan/expression/arithmetic/MultiplyExpression.java
new file mode 100644
index 0000000..3347250
--- /dev/null
+++ b/core/src/main/java/org/carbondata/scan/expression/arithmetic/MultiplyExpression.java
@@ -0,0 +1,83 @@
+/*
+ * 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.carbondata.scan.expression.arithmetic;
+
+import org.carbondata.scan.expression.DataType;
+import org.carbondata.scan.expression.Expression;
+import org.carbondata.scan.expression.ExpressionResult;
+import org.carbondata.scan.expression.exception.FilterUnsupportedException;
+import org.carbondata.scan.filter.intf.ExpressionType;
+import org.carbondata.scan.filter.intf.RowIntf;
+
+public class MultiplyExpression extends BinaryArithmeticExpression {
+  private static final long serialVersionUID = 1L;
+
+  public MultiplyExpression(Expression left, Expression right) {
+    super(left, right);
+  }
+
+  @Override public ExpressionResult evaluate(RowIntf value) throws FilterUnsupportedException {
+    ExpressionResult multiplyExprLeftRes = left.evaluate(value);
+    ExpressionResult multiplyExprRightRes = right.evaluate(value);
+    ExpressionResult val1 = multiplyExprLeftRes;
+    ExpressionResult val2 = multiplyExprRightRes;
+    if (multiplyExprLeftRes.isNull() || multiplyExprRightRes.isNull()) {
+      multiplyExprLeftRes.set(multiplyExprLeftRes.getDataType(), null);
+      return multiplyExprLeftRes;
+    }
+
+    if (multiplyExprLeftRes.getDataType() != multiplyExprRightRes.getDataType()) {
+      if (multiplyExprLeftRes.getDataType().getPresedenceOrder() < multiplyExprRightRes
+          .getDataType().getPresedenceOrder()) {
+        val2 = multiplyExprLeftRes;
+        val1 = multiplyExprRightRes;
+      }
+    }
+    switch (val1.getDataType()) {
+      case StringType:
+      case DoubleType:
+        multiplyExprRightRes.set(DataType.DoubleType, val1.getDouble() * val2.getDouble());
+        break;
+      case IntegerType:
+        multiplyExprRightRes.set(DataType.IntegerType, val1.getInt() * val2.getInt());
+        break;
+      case LongType:
+        multiplyExprRightRes.set(DataType.LongType, val1.getLong() * val2.getLong());
+        break;
+      case DecimalType:
+        multiplyExprRightRes
+            .set(DataType.DecimalType, val1.getDecimal().multiply(val2.getDecimal()));
+        break;
+      default:
+        throw new FilterUnsupportedException(
+            "Incompatible datatype for applying Add Expression Filter " + multiplyExprLeftRes
+                .getDataType());
+    }
+    return multiplyExprRightRes;
+  }
+
+  @Override public ExpressionType getFilterExpressionType() {
+    return ExpressionType.MULTIPLY;
+  }
+
+  @Override public String getString() {
+    return "Substract(" + left.getString() + ',' + right.getString() + ')';
+  }
+}

http://git-wip-us.apache.org/repos/asf/incubator-carbondata/blob/1c725f5b/core/src/main/java/org/carbondata/scan/expression/arithmetic/SubstractExpression.java
----------------------------------------------------------------------
diff --git a/core/src/main/java/org/carbondata/scan/expression/arithmetic/SubstractExpression.java b/core/src/main/java/org/carbondata/scan/expression/arithmetic/SubstractExpression.java
new file mode 100644
index 0000000..35fec1c
--- /dev/null
+++ b/core/src/main/java/org/carbondata/scan/expression/arithmetic/SubstractExpression.java
@@ -0,0 +1,83 @@
+/*
+ * 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.carbondata.scan.expression.arithmetic;
+
+import org.carbondata.scan.expression.DataType;
+import org.carbondata.scan.expression.Expression;
+import org.carbondata.scan.expression.ExpressionResult;
+import org.carbondata.scan.expression.exception.FilterUnsupportedException;
+import org.carbondata.scan.filter.intf.ExpressionType;
+import org.carbondata.scan.filter.intf.RowIntf;
+
+public class SubstractExpression extends BinaryArithmeticExpression {
+
+  private static final long serialVersionUID = -8304726440185363102L;
+
+  public SubstractExpression(Expression left, Expression right) {
+    super(left, right);
+  }
+
+  @Override public ExpressionResult evaluate(RowIntf value) throws FilterUnsupportedException {
+    ExpressionResult subtractExprLeftRes = left.evaluate(value);
+    ExpressionResult subtractExprRightRes = right.evaluate(value);
+    ExpressionResult val1 = subtractExprLeftRes;
+    ExpressionResult val2 = subtractExprRightRes;
+    if (subtractExprLeftRes.isNull() || subtractExprRightRes.isNull()) {
+      subtractExprLeftRes.set(subtractExprLeftRes.getDataType(), null);
+      return subtractExprLeftRes;
+    }
+    if (subtractExprLeftRes.getDataType() != subtractExprRightRes.getDataType()) {
+      if (subtractExprLeftRes.getDataType().getPresedenceOrder() < subtractExprRightRes
+          .getDataType().getPresedenceOrder()) {
+        val2 = subtractExprLeftRes;
+        val1 = subtractExprRightRes;
+      }
+    }
+    switch (val1.getDataType()) {
+      case StringType:
+      case DoubleType:
+        subtractExprRightRes.set(DataType.DoubleType, val1.getDouble() - val2.getDouble());
+        break;
+      case IntegerType:
+        subtractExprRightRes.set(DataType.IntegerType, val1.getInt() - val2.getInt());
+        break;
+      case LongType:
+        subtractExprRightRes.set(DataType.LongType, val1.getLong() - val2.getLong());
+        break;
+      case DecimalType:
+        subtractExprRightRes
+            .set(DataType.DecimalType, val1.getDecimal().subtract(val2.getDecimal()));
+        break;
+      default:
+        throw new FilterUnsupportedException(
+            "Incompatible datatype for applying Add Expression Filter " + subtractExprLeftRes
+                .getDataType());
+    }
+    return subtractExprRightRes;
+  }
+
+  @Override public ExpressionType getFilterExpressionType() {
+    return ExpressionType.SUBSTRACT;
+  }
+
+  @Override public String getString() {
+    return "Substract(" + left.getString() + ',' + right.getString() + ')';
+  }
+}