You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@carbondata.apache.org by ja...@apache.org on 2016/12/13 11:14:56 UTC
[1/4] incubator-carbondata git commit: Added unsafe on-heap/off-heap
sort to improve loading performance
Repository: incubator-carbondata
Updated Branches:
refs/heads/master 894051432 -> 910155d42
http://git-wip-us.apache.org/repos/asf/incubator-carbondata/blob/f1f9348d/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/sort/TimSort.java
----------------------------------------------------------------------
diff --git a/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/sort/TimSort.java b/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/sort/TimSort.java
new file mode 100644
index 0000000..d9ff7e5
--- /dev/null
+++ b/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/sort/TimSort.java
@@ -0,0 +1,943 @@
+/*
+ * 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.processing.newflow.sort.unsafe.sort;
+
+import java.util.Comparator;
+
+import org.apache.spark.util.collection.SortDataFormat;
+
+/**
+ * A port of the Apache Spark's TimSort and they originally ported from Android TimSort class,
+ * which utilizes a "stable, adaptive, iterative mergesort."
+ * See the method comment on sort() for more details.
+ *
+ * This has been kept in Java with the original style in order to match very closely with the
+ * Android source code, and thus be easy to verify correctness. The class is package private. We put
+ * a simple Scala wrapper {@link org.apache.spark.util.collection.Sorter}, which is available to
+ * package org.apache.spark.
+ *
+ * The purpose of the port is to generalize the interface to the sort to accept input data formats
+ * besides simple arrays where every element is sorted individually. For instance, the AppendOnlyMap
+ * uses this to sort an Array with alternating elements of the form [key, value, key, value].
+ * This generalization comes with minimal overhead -- see SortDataFormat for more information.
+ *
+ * We allow key reuse to prevent creating many key objects -- see SortDataFormat.
+ *
+ * @see SortDataFormat
+ * @see org.apache.spark.util.collection.Sorter
+ */
+public class TimSort<K, Buffer> {
+
+ /**
+ * This is the minimum sized sequence that will be merged. Shorter
+ * sequences will be lengthened by calling binarySort. If the entire
+ * array is less than this length, no merges will be performed.
+ *
+ * This constant should be a power of two. It was 64 in Tim Peter's C
+ * implementation, but 32 was empirically determined to work better in
+ * this implementation. In the unlikely event that you set this constant
+ * to be a number that's not a power of two, you'll need to change the
+ * minRunLength computation.
+ *
+ * If you decrease this constant, you must change the stackLen
+ * computation in the TimSort constructor, or you risk an
+ * ArrayOutOfBounds exception. See listsort.txt for a discussion
+ * of the minimum stack length required as a function of the length
+ * of the array being sorted and the minimum merge sequence length.
+ */
+ private static final int MIN_MERGE = 32;
+
+ private final SortDataFormat<K, Buffer> s;
+
+ public TimSort(SortDataFormat<K, Buffer> sortDataFormat) {
+ this.s = sortDataFormat;
+ }
+
+ /**
+ * A stable, adaptive, iterative mergesort that requires far fewer than
+ * n lg(n) comparisons when running on partially sorted arrays, while
+ * offering performance comparable to a traditional mergesort when run
+ * on random arrays. Like all proper mergesorts, this sort is stable and
+ * runs O(n log n) time (worst case). In the worst case, this sort requires
+ * temporary storage space for n/2 object references; in the best case,
+ * it requires only a small constant amount of space.
+ *
+ * This implementation was adapted from Tim Peters's list sort for
+ * Python, which is described in detail here:
+ *
+ * http://svn.python.org/projects/python/trunk/Objects/listsort.txt
+ *
+ * Tim's C code may be found here:
+ *
+ * http://svn.python.org/projects/python/trunk/Objects/listobject.c
+ *
+ * The underlying techniques are described in this paper (and may have
+ * even earlier origins):
+ *
+ * "Optimistic Sorting and Information Theoretic Complexity"
+ * Peter McIlroy
+ * SODA (Fourth Annual ACM-SIAM Symposium on Discrete Algorithms),
+ * pp 467-474, Austin, Texas, 25-27 January 1993.
+ *
+ * While the API to this class consists solely of static methods, it is
+ * (privately) instantiable; a TimSort instance holds the state of an ongoing
+ * sort, assuming the input array is large enough to warrant the full-blown
+ * TimSort. Small arrays are sorted in place, using a binary insertion sort.
+ *
+ * @author Josh Bloch
+ */
+ public void sort(Buffer a, int lo, int hi, Comparator<? super K> c) {
+ assert c != null;
+
+ int nRemaining = hi - lo;
+ if (nRemaining < 2)
+ return; // Arrays of size 0 and 1 are always sorted
+
+ // If array is small, do a "mini-TimSort" with no merges
+ if (nRemaining < MIN_MERGE) {
+ int initRunLen = countRunAndMakeAscending(a, lo, hi, c);
+ binarySort(a, lo, hi, lo + initRunLen, c);
+ return;
+ }
+
+ /**
+ * March over the array once, left to right, finding natural runs,
+ * extending short natural runs to minRun elements, and merging runs
+ * to maintain stack invariant.
+ */
+ SortState sortState = new SortState(a, c, hi - lo);
+ int minRun = minRunLength(nRemaining);
+ do {
+ // Identify next run
+ int runLen = countRunAndMakeAscending(a, lo, hi, c);
+
+ // If run is short, extend to min(minRun, nRemaining)
+ if (runLen < minRun) {
+ int force = nRemaining <= minRun ? nRemaining : minRun;
+ binarySort(a, lo, lo + force, lo + runLen, c);
+ runLen = force;
+ }
+
+ // Push run onto pending-run stack, and maybe merge
+ sortState.pushRun(lo, runLen);
+ sortState.mergeCollapse();
+
+ // Advance to find next run
+ lo += runLen;
+ nRemaining -= runLen;
+ } while (nRemaining != 0);
+
+ // Merge all remaining runs to complete sort
+ assert lo == hi;
+ sortState.mergeForceCollapse();
+ assert sortState.stackSize == 1;
+ }
+
+ /**
+ * Sorts the specified portion of the specified array using a binary
+ * insertion sort. This is the best method for sorting small numbers
+ * of elements. It requires O(n log n) compares, but O(n^2) data
+ * movement (worst case).
+ *
+ * If the initial part of the specified range is already sorted,
+ * this method can take advantage of it: the method assumes that the
+ * elements from index {@code lo}, inclusive, to {@code start},
+ * exclusive are already sorted.
+ *
+ * @param a the array in which a range is to be sorted
+ * @param lo the index of the first element in the range to be sorted
+ * @param hi the index after the last element in the range to be sorted
+ * @param start the index of the first element in the range that is
+ * not already known to be sorted ({@code lo <= start <= hi})
+ * @param c comparator to used for the sort
+ */
+ @SuppressWarnings("fallthrough")
+ private void binarySort(Buffer a, int lo, int hi, int start, Comparator<? super K> c) {
+ assert lo <= start && start <= hi;
+ if (start == lo)
+ start++;
+
+ K key0 = s.newKey();
+ K key1 = s.newKey();
+
+ Buffer pivotStore = s.allocate(1);
+ for ( ; start < hi; start++) {
+ s.copyElement(a, start, pivotStore, 0);
+ K pivot = s.getKey(pivotStore, 0, key0);
+
+ // Set left (and right) to the index where a[start] (pivot) belongs
+ int left = lo;
+ int right = start;
+ assert left <= right;
+ /*
+ * Invariants:
+ * pivot >= all in [lo, left).
+ * pivot < all in [right, start).
+ */
+ while (left < right) {
+ int mid = (left + right) >>> 1;
+ if (c.compare(pivot, s.getKey(a, mid, key1)) < 0)
+ right = mid;
+ else
+ left = mid + 1;
+ }
+ assert left == right;
+
+ /*
+ * The invariants still hold: pivot >= all in [lo, left) and
+ * pivot < all in [left, start), so pivot belongs at left. Note
+ * that if there are elements equal to pivot, left points to the
+ * first slot after them -- that's why this sort is stable.
+ * Slide elements over to make room for pivot.
+ */
+ int n = start - left; // The number of elements to move
+ // Switch is just an optimization for arraycopy in default case
+ switch (n) {
+ case 2: s.copyElement(a, left + 1, a, left + 2);
+ case 1: s.copyElement(a, left, a, left + 1);
+ break;
+ default: s.copyRange(a, left, a, left + 1, n);
+ }
+ s.copyElement(pivotStore, 0, a, left);
+ }
+ }
+
+ /**
+ * Returns the length of the run beginning at the specified position in
+ * the specified array and reverses the run if it is descending (ensuring
+ * that the run will always be ascending when the method returns).
+ *
+ * A run is the longest ascending sequence with:
+ *
+ * a[lo] <= a[lo + 1] <= a[lo + 2] <= ...
+ *
+ * or the longest descending sequence with:
+ *
+ * a[lo] > a[lo + 1] > a[lo + 2] > ...
+ *
+ * For its intended use in a stable mergesort, the strictness of the
+ * definition of "descending" is needed so that the call can safely
+ * reverse a descending sequence without violating stability.
+ *
+ * @param a the array in which a run is to be counted and possibly reversed
+ * @param lo index of the first element in the run
+ * @param hi index after the last element that may be contained in the run.
+ It is required that {@code lo < hi}.
+ * @param c the comparator to used for the sort
+ * @return the length of the run beginning at the specified position in
+ * the specified array
+ */
+ private int countRunAndMakeAscending(Buffer a, int lo, int hi, Comparator<? super K> c) {
+ assert lo < hi;
+ int runHi = lo + 1;
+ if (runHi == hi)
+ return 1;
+
+ K key0 = s.newKey();
+ K key1 = s.newKey();
+
+ // Find end of run, and reverse range if descending
+ if (c.compare(s.getKey(a, runHi++, key0), s.getKey(a, lo, key1)) < 0) { // Descending
+ while (runHi < hi && c.compare(s.getKey(a, runHi, key0), s.getKey(a, runHi - 1, key1)) < 0)
+ runHi++;
+ reverseRange(a, lo, runHi);
+ } else { // Ascending
+ while (runHi < hi && c.compare(s.getKey(a, runHi, key0), s.getKey(a, runHi - 1, key1)) >= 0)
+ runHi++;
+ }
+
+ return runHi - lo;
+ }
+
+ /**
+ * Reverse the specified range of the specified array.
+ *
+ * @param a the array in which a range is to be reversed
+ * @param lo the index of the first element in the range to be reversed
+ * @param hi the index after the last element in the range to be reversed
+ */
+ private void reverseRange(Buffer a, int lo, int hi) {
+ hi--;
+ while (lo < hi) {
+ s.swap(a, lo, hi);
+ lo++;
+ hi--;
+ }
+ }
+
+ /**
+ * Returns the minimum acceptable run length for an array of the specified
+ * length. Natural runs shorter than this will be extended with
+ * {@link #binarySort}.
+ *
+ * Roughly speaking, the computation is:
+ *
+ * If n < MIN_MERGE, return n (it's too small to bother with fancy stuff).
+ * Else if n is an exact power of 2, return MIN_MERGE/2.
+ * Else return an int k, MIN_MERGE/2 <= k <= MIN_MERGE, such that n/k
+ * is close to, but strictly less than, an exact power of 2.
+ *
+ * For the rationale, see listsort.txt.
+ *
+ * @param n the length of the array to be sorted
+ * @return the length of the minimum run to be merged
+ */
+ private int minRunLength(int n) {
+ assert n >= 0;
+ int r = 0; // Becomes 1 if any 1 bits are shifted off
+ while (n >= MIN_MERGE) {
+ r |= (n & 1);
+ n >>= 1;
+ }
+ return n + r;
+ }
+
+ private class SortState {
+
+ /**
+ * The Buffer being sorted.
+ */
+ private final Buffer a;
+
+ /**
+ * Length of the sort Buffer.
+ */
+ private final int aLength;
+
+ /**
+ * The comparator for this sort.
+ */
+ private final Comparator<? super K> c;
+
+ /**
+ * When we get into galloping mode, we stay there until both runs win less
+ * often than MIN_GALLOP consecutive times.
+ */
+ private static final int MIN_GALLOP = 7;
+
+ /**
+ * This controls when we get *into* galloping mode. It is initialized
+ * to MIN_GALLOP. The mergeLo and mergeHi methods nudge it higher for
+ * random data, and lower for highly structured data.
+ */
+ private int minGallop = MIN_GALLOP;
+
+ /**
+ * Maximum initial size of tmp array, which is used for merging. The array
+ * can grow to accommodate demand.
+ *
+ * Unlike Tim's original C version, we do not allocate this much storage
+ * when sorting smaller arrays. This change was required for performance.
+ */
+ private static final int INITIAL_TMP_STORAGE_LENGTH = 256;
+
+ /**
+ * Temp storage for merges.
+ */
+ private Buffer tmp; // Actual runtime type will be Object[], regardless of T
+
+ /**
+ * Length of the temp storage.
+ */
+ private int tmpLength = 0;
+
+ /**
+ * A stack of pending runs yet to be merged. Run i starts at
+ * address base[i] and extends for len[i] elements. It's always
+ * true (so long as the indices are in bounds) that:
+ *
+ * runBase[i] + runLen[i] == runBase[i + 1]
+ *
+ * so we could cut the storage for this, but it's a minor amount,
+ * and keeping all the info explicit simplifies the code.
+ */
+ private int stackSize = 0; // Number of pending runs on stack
+ private final int[] runBase;
+ private final int[] runLen;
+
+ /**
+ * Creates a TimSort instance to maintain the state of an ongoing sort.
+ *
+ * @param a the array to be sorted
+ * @param c the comparator to determine the order of the sort
+ */
+ private SortState(Buffer a, Comparator<? super K> c, int len) {
+ this.aLength = len;
+ this.a = a;
+ this.c = c;
+
+ // Allocate temp storage (which may be increased later if necessary)
+ tmpLength = len < 2 * INITIAL_TMP_STORAGE_LENGTH ? len >>> 1 : INITIAL_TMP_STORAGE_LENGTH;
+ tmp = s.allocate(tmpLength);
+
+ /*
+ * Allocate runs-to-be-merged stack (which cannot be expanded). The
+ * stack length requirements are described in listsort.txt. The C
+ * version always uses the same stack length (85), but this was
+ * measured to be too expensive when sorting "mid-sized" arrays (e.g.,
+ * 100 elements) in Java. Therefore, we use smaller (but sufficiently
+ * large) stack lengths for smaller arrays. The "magic numbers" in the
+ * computation below must be changed if MIN_MERGE is decreased. See
+ * the MIN_MERGE declaration above for more information.
+ */
+ int stackLen = (len < 120 ? 5 :
+ len < 1542 ? 10 :
+ len < 119151 ? 19 : 40);
+ runBase = new int[stackLen];
+ runLen = new int[stackLen];
+ }
+
+ /**
+ * Pushes the specified run onto the pending-run stack.
+ *
+ * @param runBase index of the first element in the run
+ * @param runLen the number of elements in the run
+ */
+ private void pushRun(int runBase, int runLen) {
+ this.runBase[stackSize] = runBase;
+ this.runLen[stackSize] = runLen;
+ stackSize++;
+ }
+
+ /**
+ * Examines the stack of runs waiting to be merged and merges adjacent runs
+ * until the stack invariants are reestablished:
+ *
+ * 1. runLen[i - 3] > runLen[i - 2] + runLen[i - 1]
+ * 2. runLen[i - 2] > runLen[i - 1]
+ *
+ * This method is called each time a new run is pushed onto the stack,
+ * so the invariants are guaranteed to hold for i < stackSize upon
+ * entry to the method.
+ */
+ private void mergeCollapse() {
+ while (stackSize > 1) {
+ int n = stackSize - 2;
+ if ( (n >= 1 && runLen[n-1] <= runLen[n] + runLen[n+1])
+ || (n >= 2 && runLen[n-2] <= runLen[n] + runLen[n-1])) {
+ if (runLen[n - 1] < runLen[n + 1])
+ n--;
+ } else if (runLen[n] > runLen[n + 1]) {
+ break; // Invariant is established
+ }
+ mergeAt(n);
+ }
+ }
+
+ /**
+ * Merges all runs on the stack until only one remains. This method is
+ * called once, to complete the sort.
+ */
+ private void mergeForceCollapse() {
+ while (stackSize > 1) {
+ int n = stackSize - 2;
+ if (n > 0 && runLen[n - 1] < runLen[n + 1])
+ n--;
+ mergeAt(n);
+ }
+ }
+
+ /**
+ * Merges the two runs at stack indices i and i+1. Run i must be
+ * the penultimate or antepenultimate run on the stack. In other words,
+ * i must be equal to stackSize-2 or stackSize-3.
+ *
+ * @param i stack index of the first of the two runs to merge
+ */
+ private void mergeAt(int i) {
+ assert stackSize >= 2;
+ assert i >= 0;
+ assert i == stackSize - 2 || i == stackSize - 3;
+
+ int base1 = runBase[i];
+ int len1 = runLen[i];
+ int base2 = runBase[i + 1];
+ int len2 = runLen[i + 1];
+ assert len1 > 0 && len2 > 0;
+ assert base1 + len1 == base2;
+
+ /*
+ * Record the length of the combined runs; if i is the 3rd-last
+ * run now, also slide over the last run (which isn't involved
+ * in this merge). The current run (i+1) goes away in any case.
+ */
+ runLen[i] = len1 + len2;
+ if (i == stackSize - 3) {
+ runBase[i + 1] = runBase[i + 2];
+ runLen[i + 1] = runLen[i + 2];
+ }
+ stackSize--;
+
+ K key0 = s.newKey();
+
+ /*
+ * Find where the first element of run2 goes in run1. Prior elements
+ * in run1 can be ignored (because they're already in place).
+ */
+ int k = gallopRight(s.getKey(a, base2, key0), a, base1, len1, 0, c);
+ assert k >= 0;
+ base1 += k;
+ len1 -= k;
+ if (len1 == 0)
+ return;
+
+ /*
+ * Find where the last element of run1 goes in run2. Subsequent elements
+ * in run2 can be ignored (because they're already in place).
+ */
+ len2 = gallopLeft(s.getKey(a, base1 + len1 - 1, key0), a, base2, len2, len2 - 1, c);
+ assert len2 >= 0;
+ if (len2 == 0)
+ return;
+
+ // Merge remaining runs, using tmp array with min(len1, len2) elements
+ if (len1 <= len2)
+ mergeLo(base1, len1, base2, len2);
+ else
+ mergeHi(base1, len1, base2, len2);
+ }
+
+ /**
+ * Locates the position at which to insert the specified key into the
+ * specified sorted range; if the range contains an element equal to key,
+ * returns the index of the leftmost equal element.
+ *
+ * @param key the key whose insertion point to search for
+ * @param a the array in which to search
+ * @param base the index of the first element in the range
+ * @param len the length of the range; must be > 0
+ * @param hint the index at which to begin the search, 0 <= hint < n.
+ * The closer hint is to the result, the faster this method will run.
+ * @param c the comparator used to order the range, and to search
+ * @return the int k, 0 <= k <= n such that a[b + k - 1] < key <= a[b + k],
+ * pretending that a[b - 1] is minus infinity and a[b + n] is infinity.
+ * In other words, key belongs at index b + k; or in other words,
+ * the first k elements of a should precede key, and the last n - k
+ * should follow it.
+ */
+ private int gallopLeft(K key, Buffer a, int base, int len, int hint, Comparator<? super K> c) {
+ assert len > 0 && hint >= 0 && hint < len;
+ int lastOfs = 0;
+ int ofs = 1;
+ K key0 = s.newKey();
+
+ if (c.compare(key, s.getKey(a, base + hint, key0)) > 0) {
+ // Gallop right until a[base+hint+lastOfs] < key <= a[base+hint+ofs]
+ int maxOfs = len - hint;
+ while (ofs < maxOfs && c.compare(key, s.getKey(a, base + hint + ofs, key0)) > 0) {
+ lastOfs = ofs;
+ ofs = (ofs << 1) + 1;
+ if (ofs <= 0) // int overflow
+ ofs = maxOfs;
+ }
+ if (ofs > maxOfs)
+ ofs = maxOfs;
+
+ // Make offsets relative to base
+ lastOfs += hint;
+ ofs += hint;
+ } else { // key <= a[base + hint]
+ // Gallop left until a[base+hint-ofs] < key <= a[base+hint-lastOfs]
+ final int maxOfs = hint + 1;
+ while (ofs < maxOfs && c.compare(key, s.getKey(a, base + hint - ofs, key0)) <= 0) {
+ lastOfs = ofs;
+ ofs = (ofs << 1) + 1;
+ if (ofs <= 0) // int overflow
+ ofs = maxOfs;
+ }
+ if (ofs > maxOfs)
+ ofs = maxOfs;
+
+ // Make offsets relative to base
+ int tmp = lastOfs;
+ lastOfs = hint - ofs;
+ ofs = hint - tmp;
+ }
+ assert -1 <= lastOfs && lastOfs < ofs && ofs <= len;
+
+ /*
+ * Now a[base+lastOfs] < key <= a[base+ofs], so key belongs somewhere
+ * to the right of lastOfs but no farther right than ofs. Do a binary
+ * search, with invariant a[base + lastOfs - 1] < key <= a[base + ofs].
+ */
+ lastOfs++;
+ while (lastOfs < ofs) {
+ int m = lastOfs + ((ofs - lastOfs) >>> 1);
+
+ if (c.compare(key, s.getKey(a, base + m, key0)) > 0)
+ lastOfs = m + 1; // a[base + m] < key
+ else
+ ofs = m; // key <= a[base + m]
+ }
+ assert lastOfs == ofs; // so a[base + ofs - 1] < key <= a[base + ofs]
+ return ofs;
+ }
+
+ /**
+ * Like gallopLeft, except that if the range contains an element equal to
+ * key, gallopRight returns the index after the rightmost equal element.
+ *
+ * @param key the key whose insertion point to search for
+ * @param a the array in which to search
+ * @param base the index of the first element in the range
+ * @param len the length of the range; must be > 0
+ * @param hint the index at which to begin the search, 0 <= hint < n.
+ * The closer hint is to the result, the faster this method will run.
+ * @param c the comparator used to order the range, and to search
+ * @return the int k, 0 <= k <= n such that a[b + k - 1] <= key < a[b + k]
+ */
+ private int gallopRight(K key, Buffer a, int base, int len, int hint, Comparator<? super K> c) {
+ assert len > 0 && hint >= 0 && hint < len;
+
+ int ofs = 1;
+ int lastOfs = 0;
+ K key1 = s.newKey();
+
+ if (c.compare(key, s.getKey(a, base + hint, key1)) < 0) {
+ // Gallop left until a[b+hint - ofs] <= key < a[b+hint - lastOfs]
+ int maxOfs = hint + 1;
+ while (ofs < maxOfs && c.compare(key, s.getKey(a, base + hint - ofs, key1)) < 0) {
+ lastOfs = ofs;
+ ofs = (ofs << 1) + 1;
+ if (ofs <= 0) // int overflow
+ ofs = maxOfs;
+ }
+ if (ofs > maxOfs)
+ ofs = maxOfs;
+
+ // Make offsets relative to b
+ int tmp = lastOfs;
+ lastOfs = hint - ofs;
+ ofs = hint - tmp;
+ } else { // a[b + hint] <= key
+ // Gallop right until a[b+hint + lastOfs] <= key < a[b+hint + ofs]
+ int maxOfs = len - hint;
+ while (ofs < maxOfs && c.compare(key, s.getKey(a, base + hint + ofs, key1)) >= 0) {
+ lastOfs = ofs;
+ ofs = (ofs << 1) + 1;
+ if (ofs <= 0) // int overflow
+ ofs = maxOfs;
+ }
+ if (ofs > maxOfs)
+ ofs = maxOfs;
+
+ // Make offsets relative to b
+ lastOfs += hint;
+ ofs += hint;
+ }
+ assert -1 <= lastOfs && lastOfs < ofs && ofs <= len;
+
+ /*
+ * Now a[b + lastOfs] <= key < a[b + ofs], so key belongs somewhere to
+ * the right of lastOfs but no farther right than ofs. Do a binary
+ * search, with invariant a[b + lastOfs - 1] <= key < a[b + ofs].
+ */
+ lastOfs++;
+ while (lastOfs < ofs) {
+ int m = lastOfs + ((ofs - lastOfs) >>> 1);
+
+ if (c.compare(key, s.getKey(a, base + m, key1)) < 0)
+ ofs = m; // key < a[b + m]
+ else
+ lastOfs = m + 1; // a[b + m] <= key
+ }
+ assert lastOfs == ofs; // so a[b + ofs - 1] <= key < a[b + ofs]
+ return ofs;
+ }
+
+ /**
+ * Merges two adjacent runs in place, in a stable fashion. The first
+ * element of the first run must be greater than the first element of the
+ * second run (a[base1] > a[base2]), and the last element of the first run
+ * (a[base1 + len1-1]) must be greater than all elements of the second run.
+ *
+ * For performance, this method should be called only when len1 <= len2;
+ * its twin, mergeHi should be called if len1 >= len2. (Either method
+ * may be called if len1 == len2.)
+ *
+ * @param base1 index of first element in first run to be merged
+ * @param len1 length of first run to be merged (must be > 0)
+ * @param base2 index of first element in second run to be merged
+ * (must be aBase + aLen)
+ * @param len2 length of second run to be merged (must be > 0)
+ */
+ private void mergeLo(int base1, int len1, int base2, int len2) {
+ assert len1 > 0 && len2 > 0 && base1 + len1 == base2;
+
+ // Copy first run into temp array
+ Buffer a = this.a; // For performance
+ Buffer tmp = ensureCapacity(len1);
+ s.copyRange(a, base1, tmp, 0, len1);
+
+ int cursor1 = 0; // Indexes into tmp array
+ int cursor2 = base2; // Indexes int a
+ int dest = base1; // Indexes int a
+
+ // Move first element of second run and deal with degenerate cases
+ s.copyElement(a, cursor2++, a, dest++);
+ if (--len2 == 0) {
+ s.copyRange(tmp, cursor1, a, dest, len1);
+ return;
+ }
+ if (len1 == 1) {
+ s.copyRange(a, cursor2, a, dest, len2);
+ s.copyElement(tmp, cursor1, a, dest + len2); // Last elt of run 1 to end of merge
+ return;
+ }
+
+ K key0 = s.newKey();
+ K key1 = s.newKey();
+
+ Comparator<? super K> c = this.c; // Use local variable for performance
+ int minGallop = this.minGallop; // " " " " "
+ outer:
+ while (true) {
+ int count1 = 0; // Number of times in a row that first run won
+ int count2 = 0; // Number of times in a row that second run won
+
+ /*
+ * Do the straightforward thing until (if ever) one run starts
+ * winning consistently.
+ */
+ do {
+ assert len1 > 1 && len2 > 0;
+ if (c.compare(s.getKey(a, cursor2, key0), s.getKey(tmp, cursor1, key1)) < 0) {
+ s.copyElement(a, cursor2++, a, dest++);
+ count2++;
+ count1 = 0;
+ if (--len2 == 0)
+ break outer;
+ } else {
+ s.copyElement(tmp, cursor1++, a, dest++);
+ count1++;
+ count2 = 0;
+ if (--len1 == 1)
+ break outer;
+ }
+ } while ((count1 | count2) < minGallop);
+
+ /*
+ * One run is winning so consistently that galloping may be a
+ * huge win. So try that, and continue galloping until (if ever)
+ * neither run appears to be winning consistently anymore.
+ */
+ do {
+ assert len1 > 1 && len2 > 0;
+ count1 = gallopRight(s.getKey(a, cursor2, key0), tmp, cursor1, len1, 0, c);
+ if (count1 != 0) {
+ s.copyRange(tmp, cursor1, a, dest, count1);
+ dest += count1;
+ cursor1 += count1;
+ len1 -= count1;
+ if (len1 <= 1) // len1 == 1 || len1 == 0
+ break outer;
+ }
+ s.copyElement(a, cursor2++, a, dest++);
+ if (--len2 == 0)
+ break outer;
+
+ count2 = gallopLeft(s.getKey(tmp, cursor1, key0), a, cursor2, len2, 0, c);
+ if (count2 != 0) {
+ s.copyRange(a, cursor2, a, dest, count2);
+ dest += count2;
+ cursor2 += count2;
+ len2 -= count2;
+ if (len2 == 0)
+ break outer;
+ }
+ s.copyElement(tmp, cursor1++, a, dest++);
+ if (--len1 == 1)
+ break outer;
+ minGallop--;
+ } while (count1 >= MIN_GALLOP | count2 >= MIN_GALLOP);
+ if (minGallop < 0)
+ minGallop = 0;
+ minGallop += 2; // Penalize for leaving gallop mode
+ } // End of "outer" loop
+ this.minGallop = minGallop < 1 ? 1 : minGallop; // Write back to field
+
+ if (len1 == 1) {
+ assert len2 > 0;
+ s.copyRange(a, cursor2, a, dest, len2);
+ s.copyElement(tmp, cursor1, a, dest + len2); // Last elt of run 1 to end of merge
+ } else if (len1 == 0) {
+ throw new IllegalArgumentException(
+ "Comparison method violates its general contract!");
+ } else {
+ assert len2 == 0;
+ assert len1 > 1;
+ s.copyRange(tmp, cursor1, a, dest, len1);
+ }
+ }
+
+ /**
+ * Like mergeLo, except that this method should be called only if
+ * len1 >= len2; mergeLo should be called if len1 <= len2. (Either method
+ * may be called if len1 == len2.)
+ *
+ * @param base1 index of first element in first run to be merged
+ * @param len1 length of first run to be merged (must be > 0)
+ * @param base2 index of first element in second run to be merged
+ * (must be aBase + aLen)
+ * @param len2 length of second run to be merged (must be > 0)
+ */
+ private void mergeHi(int base1, int len1, int base2, int len2) {
+ assert len1 > 0 && len2 > 0 && base1 + len1 == base2;
+
+ // Copy second run into temp array
+ Buffer a = this.a; // For performance
+ Buffer tmp = ensureCapacity(len2);
+ s.copyRange(a, base2, tmp, 0, len2);
+
+ int cursor1 = base1 + len1 - 1; // Indexes into a
+ int cursor2 = len2 - 1; // Indexes into tmp array
+ int dest = base2 + len2 - 1; // Indexes into a
+
+ K key0 = s.newKey();
+ K key1 = s.newKey();
+
+ // Move last element of first run and deal with degenerate cases
+ s.copyElement(a, cursor1--, a, dest--);
+ if (--len1 == 0) {
+ s.copyRange(tmp, 0, a, dest - (len2 - 1), len2);
+ return;
+ }
+ if (len2 == 1) {
+ dest -= len1;
+ cursor1 -= len1;
+ s.copyRange(a, cursor1 + 1, a, dest + 1, len1);
+ s.copyElement(tmp, cursor2, a, dest);
+ return;
+ }
+
+ Comparator<? super K> c = this.c; // Use local variable for performance
+ int minGallop = this.minGallop; // " " " " "
+ outer:
+ while (true) {
+ int count1 = 0; // Number of times in a row that first run won
+ int count2 = 0; // Number of times in a row that second run won
+
+ /*
+ * Do the straightforward thing until (if ever) one run
+ * appears to win consistently.
+ */
+ do {
+ assert len1 > 0 && len2 > 1;
+ if (c.compare(s.getKey(tmp, cursor2, key0), s.getKey(a, cursor1, key1)) < 0) {
+ s.copyElement(a, cursor1--, a, dest--);
+ count1++;
+ count2 = 0;
+ if (--len1 == 0)
+ break outer;
+ } else {
+ s.copyElement(tmp, cursor2--, a, dest--);
+ count2++;
+ count1 = 0;
+ if (--len2 == 1)
+ break outer;
+ }
+ } while ((count1 | count2) < minGallop);
+
+ /*
+ * One run is winning so consistently that galloping may be a
+ * huge win. So try that, and continue galloping until (if ever)
+ * neither run appears to be winning consistently anymore.
+ */
+ do {
+ assert len1 > 0 && len2 > 1;
+ count1 = len1 - gallopRight(s.getKey(tmp, cursor2, key0), a, base1, len1, len1 - 1, c);
+ if (count1 != 0) {
+ dest -= count1;
+ cursor1 -= count1;
+ len1 -= count1;
+ s.copyRange(a, cursor1 + 1, a, dest + 1, count1);
+ if (len1 == 0)
+ break outer;
+ }
+ s.copyElement(tmp, cursor2--, a, dest--);
+ if (--len2 == 1)
+ break outer;
+
+ count2 = len2 - gallopLeft(s.getKey(a, cursor1, key0), tmp, 0, len2, len2 - 1, c);
+ if (count2 != 0) {
+ dest -= count2;
+ cursor2 -= count2;
+ len2 -= count2;
+ s.copyRange(tmp, cursor2 + 1, a, dest + 1, count2);
+ if (len2 <= 1) // len2 == 1 || len2 == 0
+ break outer;
+ }
+ s.copyElement(a, cursor1--, a, dest--);
+ if (--len1 == 0)
+ break outer;
+ minGallop--;
+ } while (count1 >= MIN_GALLOP | count2 >= MIN_GALLOP);
+ if (minGallop < 0)
+ minGallop = 0;
+ minGallop += 2; // Penalize for leaving gallop mode
+ } // End of "outer" loop
+ this.minGallop = minGallop < 1 ? 1 : minGallop; // Write back to field
+
+ if (len2 == 1) {
+ assert len1 > 0;
+ dest -= len1;
+ cursor1 -= len1;
+ s.copyRange(a, cursor1 + 1, a, dest + 1, len1);
+ s.copyElement(tmp, cursor2, a, dest); // Move first elt of run2 to front of merge
+ } else if (len2 == 0) {
+ throw new IllegalArgumentException(
+ "Comparison method violates its general contract!");
+ } else {
+ assert len1 == 0;
+ assert len2 > 0;
+ s.copyRange(tmp, 0, a, dest - (len2 - 1), len2);
+ }
+ }
+
+ /**
+ * Ensures that the external array tmp has at least the specified
+ * number of elements, increasing its size if necessary. The size
+ * increases exponentially to ensure amortized linear time complexity.
+ *
+ * @param minCapacity the minimum required capacity of the tmp array
+ * @return tmp, whether or not it grew
+ */
+ private Buffer ensureCapacity(int minCapacity) {
+ if (tmpLength < minCapacity) {
+ // Compute smallest power of 2 > minCapacity
+ int newSize = minCapacity;
+ newSize |= newSize >> 1;
+ newSize |= newSize >> 2;
+ newSize |= newSize >> 4;
+ newSize |= newSize >> 8;
+ newSize |= newSize >> 16;
+ newSize++;
+
+ if (newSize < 0) // Not bloody likely!
+ newSize = minCapacity;
+ else
+ newSize = Math.min(newSize, aLength >>> 1);
+
+ tmp = s.allocate(newSize);
+ tmpLength = newSize;
+ }
+ return tmp;
+ }
+ }
+}
http://git-wip-us.apache.org/repos/asf/incubator-carbondata/blob/f1f9348d/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/sort/UnsafeIntSortDataFormat.java
----------------------------------------------------------------------
diff --git a/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/sort/UnsafeIntSortDataFormat.java b/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/sort/UnsafeIntSortDataFormat.java
new file mode 100644
index 0000000..4a52144
--- /dev/null
+++ b/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/sort/UnsafeIntSortDataFormat.java
@@ -0,0 +1,74 @@
+/*
+ * 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.processing.newflow.sort.unsafe.sort;
+
+import org.apache.carbondata.processing.newflow.sort.unsafe.IntPointerBuffer;
+import org.apache.carbondata.processing.newflow.sort.unsafe.UnsafeCarbonRowPage;
+import org.apache.carbondata.processing.newflow.sort.unsafe.holder.UnsafeCarbonRow;
+
+import org.apache.spark.util.collection.SortDataFormat;
+
+/**
+ * Interface implementation for utilities to sort the data.
+ */
+public class UnsafeIntSortDataFormat
+ extends SortDataFormat<UnsafeCarbonRow, IntPointerBuffer> {
+
+ private UnsafeCarbonRowPage page;
+
+ public UnsafeIntSortDataFormat(UnsafeCarbonRowPage page) {
+ this.page = page;
+ }
+
+ @Override public UnsafeCarbonRow getKey(IntPointerBuffer data, int pos) {
+ // Since we re-use keys, this method shouldn't be called.
+ throw new UnsupportedOperationException();
+ }
+
+ @Override public UnsafeCarbonRow newKey() {
+ return new UnsafeCarbonRow();
+ }
+
+ @Override
+ public UnsafeCarbonRow getKey(IntPointerBuffer data, int pos, UnsafeCarbonRow reuse) {
+ reuse.address = data.get(pos) + page.getDataBlock().getBaseOffset();
+ return reuse;
+ }
+
+ @Override public void swap(IntPointerBuffer data, int pos0, int pos1) {
+ int tempPointer = data.get(pos0);
+ data.set(pos0, data.get(pos1));
+ data.set(pos1, tempPointer);
+ }
+
+ @Override
+ public void copyElement(IntPointerBuffer src, int srcPos, IntPointerBuffer dst, int dstPos) {
+ dst.set(dstPos, src.get(srcPos));
+ }
+
+ @Override
+ public void copyRange(IntPointerBuffer src, int srcPos, IntPointerBuffer dst, int dstPos,
+ int length) {
+ System.arraycopy(src.getPointerBlock(), srcPos, dst.getPointerBlock(), dstPos, length);
+ }
+
+ @Override public IntPointerBuffer allocate(int length) {
+ return new IntPointerBuffer(length);
+ }
+}
http://git-wip-us.apache.org/repos/asf/incubator-carbondata/blob/f1f9348d/processing/src/main/java/org/apache/carbondata/processing/newflow/steps/SortProcessorStepImpl.java
----------------------------------------------------------------------
diff --git a/processing/src/main/java/org/apache/carbondata/processing/newflow/steps/SortProcessorStepImpl.java b/processing/src/main/java/org/apache/carbondata/processing/newflow/steps/SortProcessorStepImpl.java
index 19d099b..99b7894 100644
--- a/processing/src/main/java/org/apache/carbondata/processing/newflow/steps/SortProcessorStepImpl.java
+++ b/processing/src/main/java/org/apache/carbondata/processing/newflow/steps/SortProcessorStepImpl.java
@@ -20,6 +20,8 @@ package org.apache.carbondata.processing.newflow.steps;
import java.util.Iterator;
+import org.apache.carbondata.core.constants.CarbonCommonConstants;
+import org.apache.carbondata.core.util.CarbonProperties;
import org.apache.carbondata.processing.newflow.AbstractDataLoadProcessorStep;
import org.apache.carbondata.processing.newflow.CarbonDataLoadConfiguration;
import org.apache.carbondata.processing.newflow.DataField;
@@ -28,6 +30,7 @@ import org.apache.carbondata.processing.newflow.row.CarbonRow;
import org.apache.carbondata.processing.newflow.row.CarbonRowBatch;
import org.apache.carbondata.processing.newflow.sort.Sorter;
import org.apache.carbondata.processing.newflow.sort.impl.ParallelReadMergeSorterImpl;
+import org.apache.carbondata.processing.newflow.sort.impl.UnsafeParallelReadMergeSorterImpl;
import org.apache.carbondata.processing.sortandgroupby.sortdata.SortParameters;
/**
@@ -52,7 +55,12 @@ public class SortProcessorStepImpl extends AbstractDataLoadProcessorStep {
public void initialize() throws CarbonDataLoadingException {
child.initialize();
SortParameters sortParameters = SortParameters.createSortParameters(configuration);
- sorter = new ParallelReadMergeSorterImpl(child.getOutput());
+ boolean offheapsort = Boolean.parseBoolean(CarbonProperties.getInstance()
+ .getProperty(CarbonCommonConstants.ENABLE_UNSAFE_SORT,
+ CarbonCommonConstants.ENABLE_UNSAFE_SORT_DEFAULT));
+ if (offheapsort) {
+ sorter = new UnsafeParallelReadMergeSorterImpl(child.getOutput());
+ } else sorter = new ParallelReadMergeSorterImpl(child.getOutput());
sorter.initialize(sortParameters);
}
[2/4] incubator-carbondata git commit: Added unsafe on-heap/off-heap
sort to improve loading performance
Posted by ja...@apache.org.
http://git-wip-us.apache.org/repos/asf/incubator-carbondata/blob/f1f9348d/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/holder/UnsafeInmemoryHolder.java
----------------------------------------------------------------------
diff --git a/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/holder/UnsafeInmemoryHolder.java b/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/holder/UnsafeInmemoryHolder.java
new file mode 100644
index 0000000..d512349
--- /dev/null
+++ b/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/holder/UnsafeInmemoryHolder.java
@@ -0,0 +1,80 @@
+/*
+ * 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.processing.newflow.sort.unsafe.holder;
+
+import org.apache.carbondata.common.logging.LogService;
+import org.apache.carbondata.common.logging.LogServiceFactory;
+import org.apache.carbondata.processing.newflow.sort.unsafe.UnsafeCarbonRowPage;
+import org.apache.carbondata.processing.sortandgroupby.sortdata.NewRowComparator;
+
+public class UnsafeInmemoryHolder implements SortTempChunkHolder {
+
+ private static final LogService LOGGER =
+ LogServiceFactory.getLogService(UnsafeInmemoryHolder.class.getName());
+
+ private int counter;
+
+ private int actualSize;
+
+ private UnsafeCarbonRowPage rowPage;
+
+ private Object[] currentRow;
+
+ private long address;
+
+ private NewRowComparator comparator;
+
+ private int columnSize;
+
+ public UnsafeInmemoryHolder(UnsafeCarbonRowPage rowPage, int columnSize) {
+ this.actualSize = rowPage.getBuffer().getActualSize();
+ this.rowPage = rowPage;
+ LOGGER.audit("Processing unsafe inmemory rows page with size : " + actualSize);
+ this.comparator = new NewRowComparator(rowPage.getNoDictionaryDimensionMapping());
+ this.columnSize = columnSize;
+ }
+
+ public boolean hasNext() {
+ if (counter < actualSize) {
+ return true;
+ }
+ return false;
+ }
+
+ public void readRow() {
+ currentRow = new Object[columnSize];
+ address = rowPage.getBuffer().get(counter);
+ rowPage.getRow(address + rowPage.getDataBlock().getBaseOffset(), currentRow);
+ counter++;
+ }
+
+ public Object[] getRow() {
+ return currentRow;
+ }
+
+ @Override public int compareTo(SortTempChunkHolder o) {
+ return comparator.compare(currentRow, o.getRow());
+ }
+
+ public int numberOfRows() {
+ return actualSize;
+ }
+
+ public void close() {
+ rowPage.freeMemory();
+ }
+}
http://git-wip-us.apache.org/repos/asf/incubator-carbondata/blob/f1f9348d/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/holder/UnsafeInmemoryMergeHolder.java
----------------------------------------------------------------------
diff --git a/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/holder/UnsafeInmemoryMergeHolder.java b/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/holder/UnsafeInmemoryMergeHolder.java
new file mode 100644
index 0000000..9f157a0
--- /dev/null
+++ b/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/holder/UnsafeInmemoryMergeHolder.java
@@ -0,0 +1,90 @@
+/*
+ * 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.processing.newflow.sort.unsafe.holder;
+
+import org.apache.carbondata.common.logging.LogService;
+import org.apache.carbondata.common.logging.LogServiceFactory;
+import org.apache.carbondata.processing.newflow.sort.unsafe.UnsafeCarbonRowPage;
+import org.apache.carbondata.processing.newflow.sort.unsafe.comparator.UnsafeRowComparator;
+
+/**
+ * It is used for merging unsafe inmemory intermediate data
+ */
+public class UnsafeInmemoryMergeHolder implements Comparable<UnsafeInmemoryMergeHolder> {
+
+ private static final LogService LOGGER =
+ LogServiceFactory.getLogService(UnsafeInmemoryMergeHolder.class.getName());
+
+ private int counter;
+
+ private int actualSize;
+
+ private UnsafeCarbonRowPage rowPage;
+
+ private UnsafeCarbonRowForMerge currentRow;
+
+ private long address;
+
+ private UnsafeRowComparator comparator;
+
+ private Object baseObject;
+
+ public UnsafeInmemoryMergeHolder(UnsafeCarbonRowPage rowPage, byte index) {
+ this.actualSize = rowPage.getBuffer().getActualSize();
+ this.rowPage = rowPage;
+ LOGGER.audit("Processing unsafe inmemory rows page with size : " + actualSize);
+ this.comparator = new UnsafeRowComparator(rowPage);
+ this.baseObject = rowPage.getDataBlock().getBaseObject();
+ currentRow = new UnsafeCarbonRowForMerge();
+ currentRow.index = index;
+ }
+
+ public boolean hasNext() {
+ if (counter < actualSize) {
+ return true;
+ }
+ return false;
+ }
+
+ public void readRow() {
+ address = rowPage.getBuffer().get(counter);
+ currentRow.address = address + rowPage.getDataBlock().getBaseOffset();
+ counter++;
+ }
+
+ public UnsafeCarbonRowForMerge getRow() {
+ return currentRow;
+ }
+
+ @Override public int compareTo(UnsafeInmemoryMergeHolder o) {
+ return comparator.compare(currentRow, baseObject, o.getRow(), o.getBaseObject());
+ }
+
+ public int numberOfRows() {
+ return actualSize;
+ }
+
+ public Object getBaseObject() {
+ return baseObject;
+ }
+
+ public void close() {
+ rowPage.freeMemory();
+ }
+}
http://git-wip-us.apache.org/repos/asf/incubator-carbondata/blob/f1f9348d/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/holder/UnsafeSortTempFileChunkHolder.java
----------------------------------------------------------------------
diff --git a/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/holder/UnsafeSortTempFileChunkHolder.java b/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/holder/UnsafeSortTempFileChunkHolder.java
new file mode 100644
index 0000000..30ef9ee
--- /dev/null
+++ b/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/holder/UnsafeSortTempFileChunkHolder.java
@@ -0,0 +1,455 @@
+/*
+ * 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.processing.newflow.sort.unsafe.holder;
+
+import java.io.BufferedInputStream;
+import java.io.DataInputStream;
+import java.io.File;
+import java.io.FileInputStream;
+import java.io.FileNotFoundException;
+import java.io.IOException;
+import java.util.Comparator;
+import java.util.concurrent.Callable;
+import java.util.concurrent.ExecutorService;
+import java.util.concurrent.Executors;
+import java.util.concurrent.Future;
+
+import org.apache.carbondata.common.logging.LogService;
+import org.apache.carbondata.common.logging.LogServiceFactory;
+import org.apache.carbondata.core.constants.CarbonCommonConstants;
+import org.apache.carbondata.core.util.CarbonProperties;
+import org.apache.carbondata.core.util.CarbonUtil;
+import org.apache.carbondata.core.util.DataTypeUtil;
+import org.apache.carbondata.processing.newflow.sort.unsafe.UnsafeCarbonRowPage;
+import org.apache.carbondata.processing.sortandgroupby.exception.CarbonSortKeyAndGroupByException;
+import org.apache.carbondata.processing.sortandgroupby.sortdata.NewRowComparator;
+import org.apache.carbondata.processing.sortandgroupby.sortdata.SortParameters;
+
+public class UnsafeSortTempFileChunkHolder implements SortTempChunkHolder {
+
+ /**
+ * LOGGER
+ */
+ private static final LogService LOGGER =
+ LogServiceFactory.getLogService(UnsafeSortTempFileChunkHolder.class.getName());
+
+ /**
+ * temp file
+ */
+ private File tempFile;
+
+ /**
+ * read stream
+ */
+ private DataInputStream stream;
+
+ /**
+ * entry count
+ */
+ private int entryCount;
+
+ /**
+ * return row
+ */
+ private Object[] returnRow;
+
+ /**
+ * number of measures
+ */
+ private int measureCount;
+
+ /**
+ * number of dimensionCount
+ */
+ private int dimensionCount;
+
+ /**
+ * number of complexDimensionCount
+ */
+ private int complexDimensionCount;
+
+ /**
+ * fileBufferSize for file reader stream size
+ */
+ private int fileBufferSize;
+
+ private Object[][] currentBuffer;
+
+ private Object[][] backupBuffer;
+
+ private boolean isBackupFilled;
+
+ private boolean prefetch;
+
+ private int bufferSize;
+
+ private int bufferRowCounter;
+
+ private ExecutorService executorService;
+
+ private Future<Void> submit;
+
+ private int prefetchRecordsProceesed;
+
+ /**
+ * sortTempFileNoOFRecordsInCompression
+ */
+ private int sortTempFileNoOFRecordsInCompression;
+
+ /**
+ * isSortTempFileCompressionEnabled
+ */
+ private boolean isSortTempFileCompressionEnabled;
+
+ /**
+ * totalRecordFetch
+ */
+ private int totalRecordFetch;
+
+ private int noDictionaryCount;
+
+ private char[] aggType;
+
+ private int numberOfObjectRead;
+ /**
+ * to store whether dimension is of dictionary type or not
+ */
+ private boolean[] isNoDictionaryDimensionColumn;
+
+ private int nullSetWordsLength;
+
+ private Comparator<Object[]> comparator;
+
+ /**
+ * Constructor to initialize
+ */
+ public UnsafeSortTempFileChunkHolder(File tempFile, SortParameters parameters) {
+ // set temp file
+ this.tempFile = tempFile;
+
+ // set measure and dimension count
+ this.measureCount = parameters.getMeasureColCount();
+ this.dimensionCount = parameters.getDimColCount();
+ this.complexDimensionCount = parameters.getComplexDimColCount();
+
+ this.noDictionaryCount = parameters.getNoDictionaryCount();
+ // set mdkey length
+ this.fileBufferSize = parameters.getFileBufferSize();
+ this.executorService = Executors.newFixedThreadPool(1);
+ this.aggType = parameters.getAggType();
+ this.isNoDictionaryDimensionColumn = parameters.getNoDictionaryDimnesionColumn();
+ this.nullSetWordsLength = ((measureCount - 1) >> 6) + 1;
+ comparator = new NewRowComparator(isNoDictionaryDimensionColumn);
+ initialize();
+ }
+
+ /**
+ * This method will be used to initialize
+ *
+ * @throws CarbonSortKeyAndGroupByException problem while initializing
+ */
+ public void initialize() {
+ prefetch = Boolean.parseBoolean(CarbonProperties.getInstance()
+ .getProperty(CarbonCommonConstants.CARBON_MERGE_SORT_PREFETCH,
+ CarbonCommonConstants.CARBON_MERGE_SORT_PREFETCH_DEFAULT));
+ bufferSize = CarbonCommonConstants.CARBON_PREFETCH_BUFFERSIZE;
+ this.isSortTempFileCompressionEnabled = Boolean.parseBoolean(CarbonProperties.getInstance()
+ .getProperty(CarbonCommonConstants.IS_SORT_TEMP_FILE_COMPRESSION_ENABLED,
+ CarbonCommonConstants.IS_SORT_TEMP_FILE_COMPRESSION_ENABLED_DEFAULTVALUE));
+ if (this.isSortTempFileCompressionEnabled) {
+ LOGGER.info("Compression was used while writing the sortTempFile");
+ }
+
+ try {
+ this.sortTempFileNoOFRecordsInCompression = Integer.parseInt(CarbonProperties.getInstance()
+ .getProperty(CarbonCommonConstants.SORT_TEMP_FILE_NO_OF_RECORDS_FOR_COMPRESSION,
+ CarbonCommonConstants.SORT_TEMP_FILE_NO_OF_RECORD_FOR_COMPRESSION_DEFAULTVALUE));
+ if (this.sortTempFileNoOFRecordsInCompression < 1) {
+ LOGGER.error("Invalid value for: "
+ + CarbonCommonConstants.SORT_TEMP_FILE_NO_OF_RECORDS_FOR_COMPRESSION
+ + ": Only Positive Integer value(greater than zero) is allowed.Default value will"
+ + " be used");
+
+ this.sortTempFileNoOFRecordsInCompression = Integer.parseInt(
+ CarbonCommonConstants.SORT_TEMP_FILE_NO_OF_RECORD_FOR_COMPRESSION_DEFAULTVALUE);
+ }
+ } catch (NumberFormatException e) {
+ LOGGER.error(
+ "Invalid value for: " + CarbonCommonConstants.SORT_TEMP_FILE_NO_OF_RECORDS_FOR_COMPRESSION
+ + ", only Positive Integer value is allowed.Default value will be used");
+ this.sortTempFileNoOFRecordsInCompression = Integer
+ .parseInt(CarbonCommonConstants.SORT_TEMP_FILE_NO_OF_RECORD_FOR_COMPRESSION_DEFAULTVALUE);
+ }
+
+ initialise();
+ }
+
+ private void initialise() {
+ try {
+ if (isSortTempFileCompressionEnabled) {
+ this.bufferSize = sortTempFileNoOFRecordsInCompression;
+ }
+ stream = new DataInputStream(
+ new BufferedInputStream(new FileInputStream(tempFile), this.fileBufferSize));
+ this.entryCount = stream.readInt();
+ LOGGER.audit("Processing unsafe mode file rows with size : " + entryCount);
+ if (prefetch) {
+ new DataFetcher(false).call();
+ totalRecordFetch += currentBuffer.length;
+ if (totalRecordFetch < this.entryCount) {
+ submit = executorService.submit(new DataFetcher(true));
+ }
+ } else {
+ if (isSortTempFileCompressionEnabled) {
+ new DataFetcher(false).call();
+ }
+ }
+
+ } catch (FileNotFoundException e) {
+ LOGGER.error(e);
+ throw new RuntimeException(tempFile + " No Found", e);
+ } catch (IOException e) {
+ LOGGER.error(e);
+ throw new RuntimeException(tempFile + " No Found", e);
+ } catch (Exception e) {
+ LOGGER.error(e);
+ throw new RuntimeException(tempFile + " Problem while reading", e);
+ }
+ }
+
+ /**
+ * This method will be used to read new row from file
+ *
+ * @throws CarbonSortKeyAndGroupByException problem while reading
+ */
+ public void readRow() throws CarbonSortKeyAndGroupByException {
+ if (prefetch) {
+ fillDataForPrefetch();
+ } else if (isSortTempFileCompressionEnabled) {
+ if (bufferRowCounter >= bufferSize) {
+ try {
+ new DataFetcher(false).call();
+ bufferRowCounter = 0;
+ } catch (Exception e) {
+ LOGGER.error(e);
+ throw new CarbonSortKeyAndGroupByException(tempFile + " Problem while reading", e);
+ }
+
+ }
+ prefetchRecordsProceesed++;
+ returnRow = currentBuffer[bufferRowCounter++];
+ } else {
+ Object[] outRow = getRowFromStream();
+ this.returnRow = outRow;
+ }
+ }
+
+ private void fillDataForPrefetch() {
+ if (bufferRowCounter >= bufferSize) {
+ if (isBackupFilled) {
+ bufferRowCounter = 0;
+ currentBuffer = backupBuffer;
+ totalRecordFetch += currentBuffer.length;
+ isBackupFilled = false;
+ if (totalRecordFetch < this.entryCount) {
+ submit = executorService.submit(new DataFetcher(true));
+ }
+ } else {
+ try {
+ submit.get();
+ } catch (Exception e) {
+ LOGGER.error(e);
+ }
+ bufferRowCounter = 0;
+ currentBuffer = backupBuffer;
+ isBackupFilled = false;
+ totalRecordFetch += currentBuffer.length;
+ if (totalRecordFetch < this.entryCount) {
+ submit = executorService.submit(new DataFetcher(true));
+ }
+ }
+ }
+ prefetchRecordsProceesed++;
+ returnRow = currentBuffer[bufferRowCounter++];
+ }
+
+ /**
+ * @return
+ * @throws CarbonSortKeyAndGroupByException
+ */
+ private Object[] getRowFromStream() throws CarbonSortKeyAndGroupByException {
+ Object[] row = new Object[dimensionCount + measureCount];
+ try {
+ int dimCount = 0;
+ for (; dimCount < isNoDictionaryDimensionColumn.length; dimCount++) {
+ if (isNoDictionaryDimensionColumn[dimCount]) {
+ short aShort = stream.readShort();
+ byte[] col = new byte[aShort];
+ stream.readFully(col);
+ row[dimCount] = col;
+ } else {
+ int anInt = stream.readInt();
+ row[dimCount] = anInt;
+ }
+ }
+
+ // write complex dimensions here.
+ for (; dimCount < dimensionCount; dimCount++) {
+ short aShort = stream.readShort();
+ byte[] col = new byte[aShort];
+ stream.readFully(col);
+ row[dimCount] = col;
+ }
+
+ long[] words = new long[nullSetWordsLength];
+ for (int i = 0; i < words.length; i++) {
+ words[i] = stream.readLong();
+ }
+
+ for (int mesCount = 0; mesCount < measureCount; mesCount++) {
+ if (UnsafeCarbonRowPage.isSet(words, mesCount)) {
+ if (aggType[mesCount] == CarbonCommonConstants.SUM_COUNT_VALUE_MEASURE) {
+ row[dimensionCount + mesCount] = stream.readDouble();
+ } else if (aggType[mesCount] == CarbonCommonConstants.BIG_INT_MEASURE) {
+ row[dimensionCount + mesCount] = stream.readLong();
+ } else if (aggType[mesCount] == CarbonCommonConstants.BIG_DECIMAL_MEASURE) {
+ short aShort = stream.readShort();
+ byte[] bigDecimalInBytes = new byte[aShort];
+ stream.readFully(bigDecimalInBytes);
+ row[dimensionCount + mesCount] = DataTypeUtil.byteToBigDecimal(bigDecimalInBytes);
+ }
+ }
+ }
+ return row;
+ } catch (Exception e) {
+ throw new CarbonSortKeyAndGroupByException(e);
+ }
+ }
+
+ /**
+ * below method will be used to get the row
+ *
+ * @return row
+ */
+ public Object[] getRow() {
+ return this.returnRow;
+ }
+
+ /**
+ * below method will be used to check whether any more records are present
+ * in file or not
+ *
+ * @return more row present in file
+ */
+ public boolean hasNext() {
+ if (prefetch || isSortTempFileCompressionEnabled) {
+ return this.prefetchRecordsProceesed < this.entryCount;
+ }
+ return this.numberOfObjectRead < this.entryCount;
+ }
+
+ /**
+ * Below method will be used to close streams
+ */
+ public void close() {
+ CarbonUtil.closeStreams(stream);
+ executorService.shutdown();
+ }
+
+ /**
+ * This method will number of entries
+ *
+ * @return entryCount
+ */
+ public int numberOfRows() {
+ return entryCount;
+ }
+
+ @Override public int compareTo(SortTempChunkHolder other) {
+ return comparator.compare(returnRow, other.getRow());
+ }
+
+ @Override public boolean equals(Object obj) {
+ if (!(obj instanceof UnsafeSortTempFileChunkHolder)) {
+ return false;
+ }
+ UnsafeSortTempFileChunkHolder o = (UnsafeSortTempFileChunkHolder) obj;
+
+ return o.compareTo(o) == 0;
+ }
+
+ @Override public int hashCode() {
+ int hash = 0;
+ hash += 31 * measureCount;
+ hash += 31 * dimensionCount;
+ hash += 31 * complexDimensionCount;
+ hash += 31 * noDictionaryCount;
+ hash += tempFile.hashCode();
+ return hash;
+ }
+
+ private final class DataFetcher implements Callable<Void> {
+ private boolean isBackUpFilling;
+
+ private int numberOfRecords;
+
+ private DataFetcher(boolean backUp) {
+ isBackUpFilling = backUp;
+ calculateNumberOfRecordsToBeFetched();
+ }
+
+ private void calculateNumberOfRecordsToBeFetched() {
+ int numberOfRecordsLeftToBeRead = entryCount - totalRecordFetch;
+ numberOfRecords =
+ bufferSize < numberOfRecordsLeftToBeRead ? bufferSize : numberOfRecordsLeftToBeRead;
+ }
+
+ @Override public Void call() throws Exception {
+ try {
+ if (isBackUpFilling) {
+ backupBuffer = prefetchRecordsFromFile(numberOfRecords);
+ isBackupFilled = true;
+ } else {
+ currentBuffer = prefetchRecordsFromFile(numberOfRecords);
+ }
+ } catch (Exception e) {
+ LOGGER.error(e);
+ }
+ return null;
+ }
+
+ }
+
+ /**
+ * This method will read the records from sort temp file and keep it in a buffer
+ *
+ * @param numberOfRecords
+ * @return
+ * @throws CarbonSortKeyAndGroupByException
+ */
+ private Object[][] prefetchRecordsFromFile(int numberOfRecords)
+ throws CarbonSortKeyAndGroupByException {
+ Object[][] records = new Object[numberOfRecords][];
+ for (int i = 0; i < numberOfRecords; i++) {
+ records[i] = getRowFromStream();
+ }
+ return records;
+ }
+}
http://git-wip-us.apache.org/repos/asf/incubator-carbondata/blob/f1f9348d/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/merger/UnsafeInMemoryIntermediateDataMerger.java
----------------------------------------------------------------------
diff --git a/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/merger/UnsafeInMemoryIntermediateDataMerger.java b/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/merger/UnsafeInMemoryIntermediateDataMerger.java
new file mode 100644
index 0000000..0d36d90
--- /dev/null
+++ b/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/merger/UnsafeInMemoryIntermediateDataMerger.java
@@ -0,0 +1,217 @@
+/*
+ * 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.processing.newflow.sort.unsafe.merger;
+
+import java.util.AbstractQueue;
+import java.util.PriorityQueue;
+import java.util.concurrent.Callable;
+
+import org.apache.carbondata.common.logging.LogService;
+import org.apache.carbondata.common.logging.LogServiceFactory;
+import org.apache.carbondata.processing.newflow.sort.unsafe.UnsafeCarbonRowPage;
+import org.apache.carbondata.processing.newflow.sort.unsafe.holder.UnsafeCarbonRowForMerge;
+import org.apache.carbondata.processing.newflow.sort.unsafe.holder.UnsafeInmemoryMergeHolder;
+import org.apache.carbondata.processing.sortandgroupby.exception.CarbonSortKeyAndGroupByException;
+
+public class UnsafeInMemoryIntermediateDataMerger implements Callable<Void> {
+ /**
+ * LOGGER
+ */
+ private static final LogService LOGGER =
+ LogServiceFactory.getLogService(UnsafeInMemoryIntermediateDataMerger.class.getName());
+
+ /**
+ * recordHolderHeap
+ */
+ private AbstractQueue<UnsafeInmemoryMergeHolder> recordHolderHeap;
+
+ /**
+ * fileCounter
+ */
+ private int holderCounter;
+
+ /**
+ * entryCount
+ */
+ private int entryCount;
+
+ private UnsafeCarbonRowPage[] unsafeCarbonRowPages;
+
+ private long[] mergedAddresses;
+
+ private byte[] rowPageIndexes;
+
+ /**
+ * IntermediateFileMerger Constructor
+ */
+ public UnsafeInMemoryIntermediateDataMerger(UnsafeCarbonRowPage[] unsafeCarbonRowPages,
+ int totalSize) {
+ this.holderCounter = unsafeCarbonRowPages.length;
+ this.unsafeCarbonRowPages = unsafeCarbonRowPages;
+ this.mergedAddresses = new long[totalSize];
+ this.rowPageIndexes = new byte[totalSize];
+ this.entryCount = 0;
+ }
+
+ @Override public Void call() throws Exception {
+ long intermediateMergeStartTime = System.currentTimeMillis();
+ int holderCounterConst = holderCounter;
+ boolean isFailed = false;
+ try {
+ startSorting();
+ while (hasNext()) {
+ writeDataToMemory(next());
+ }
+ double intermediateMergeCostTime =
+ (System.currentTimeMillis() - intermediateMergeStartTime) / 1000.0;
+ LOGGER.info("============================== Intermediate Merge of " + holderCounterConst
+ + " in-memory sort Cost Time: " + intermediateMergeCostTime + "(s)");
+ } catch (Exception e) {
+ LOGGER.error(e, "Problem while intermediate merging");
+ }
+ return null;
+ }
+
+ /**
+ * This method will be used to get the sorted record from file
+ *
+ * @return sorted record sorted record
+ * @throws CarbonSortKeyAndGroupByException
+ */
+ private UnsafeCarbonRowForMerge getSortedRecordFromMemory()
+ throws CarbonSortKeyAndGroupByException {
+ UnsafeCarbonRowForMerge row = null;
+
+ // poll the top object from heap
+ // heap maintains binary tree which is based on heap condition that will
+ // be based on comparator we are passing the heap
+ // when will call poll it will always delete root of the tree and then
+ // it does trickel down operation complexity is log(n)
+ UnsafeInmemoryMergeHolder poll = this.recordHolderHeap.poll();
+
+ // get the row from chunk
+ row = poll.getRow();
+
+ // check if there no entry present
+ if (!poll.hasNext()) {
+ // change the file counter
+ --this.holderCounter;
+
+ // reaturn row
+ return row;
+ }
+
+ // read new row
+ poll.readRow();
+
+ // add to heap
+ this.recordHolderHeap.add(poll);
+
+ // return row
+ return row;
+ }
+
+ /**
+ * Below method will be used to start storing process This method will get
+ * all the temp files present in sort temp folder then it will create the
+ * record holder heap and then it will read first record from each file and
+ * initialize the heap
+ *
+ * @throws CarbonSortKeyAndGroupByException
+ */
+ private void startSorting() throws CarbonSortKeyAndGroupByException {
+ LOGGER.info("Number of row pages in intermediate merger: " + this.holderCounter);
+
+ // create record holder heap
+ createRecordHolderQueue(unsafeCarbonRowPages);
+
+ // iterate over file list and create chunk holder and add to heap
+ LOGGER.info("Started adding first record from row page");
+
+ UnsafeInmemoryMergeHolder unsafePageHolder = null;
+ byte index = 0;
+ for (UnsafeCarbonRowPage unsafeCarbonRowPage : unsafeCarbonRowPages) {
+ // create chunk holder
+ unsafePageHolder = new UnsafeInmemoryMergeHolder(unsafeCarbonRowPage, index++);
+
+ // initialize
+ unsafePageHolder.readRow();
+
+ // add to heap
+ this.recordHolderHeap.add(unsafePageHolder);
+ }
+
+ LOGGER.info("Heap Size" + this.recordHolderHeap.size());
+ }
+
+ /**
+ * This method will be used to create the heap which will be used to hold
+ * the chunk of data
+ */
+ private void createRecordHolderQueue(UnsafeCarbonRowPage[] pages) {
+ // creating record holder heap
+ this.recordHolderHeap = new PriorityQueue<UnsafeInmemoryMergeHolder>(pages.length);
+ }
+
+ /**
+ * This method will be used to get the sorted row
+ *
+ * @return sorted row
+ * @throws CarbonSortKeyAndGroupByException
+ */
+ private UnsafeCarbonRowForMerge next() throws CarbonSortKeyAndGroupByException {
+ return getSortedRecordFromMemory();
+ }
+
+ /**
+ * This method will be used to check whether any more element is present or
+ * not
+ *
+ * @return more element is present
+ */
+ private boolean hasNext() {
+ return this.holderCounter > 0;
+ }
+
+ /**
+ * Below method will be used to write data to file
+ */
+ private void writeDataToMemory(UnsafeCarbonRowForMerge row) {
+ mergedAddresses[entryCount] = row.address;
+ rowPageIndexes[entryCount] = row.index;
+ entryCount++;
+ }
+
+ public int getEntryCount() {
+ return entryCount;
+ }
+
+ public UnsafeCarbonRowPage[] getUnsafeCarbonRowPages() {
+ return unsafeCarbonRowPages;
+ }
+
+ public long[] getMergedAddresses() {
+ return mergedAddresses;
+ }
+
+ public byte[] getRowPageIndexes() {
+ return rowPageIndexes;
+ }
+}
http://git-wip-us.apache.org/repos/asf/incubator-carbondata/blob/f1f9348d/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/merger/UnsafeIntermediateFileMerger.java
----------------------------------------------------------------------
diff --git a/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/merger/UnsafeIntermediateFileMerger.java b/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/merger/UnsafeIntermediateFileMerger.java
new file mode 100644
index 0000000..735243e
--- /dev/null
+++ b/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/merger/UnsafeIntermediateFileMerger.java
@@ -0,0 +1,364 @@
+/*
+ * 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.processing.newflow.sort.unsafe.merger;
+
+import java.io.BufferedOutputStream;
+import java.io.DataOutputStream;
+import java.io.File;
+import java.io.FileNotFoundException;
+import java.io.FileOutputStream;
+import java.io.IOException;
+import java.math.BigDecimal;
+import java.nio.ByteBuffer;
+import java.util.AbstractQueue;
+import java.util.Arrays;
+import java.util.PriorityQueue;
+import java.util.concurrent.Callable;
+
+import org.apache.carbondata.common.logging.LogService;
+import org.apache.carbondata.common.logging.LogServiceFactory;
+import org.apache.carbondata.core.constants.CarbonCommonConstants;
+import org.apache.carbondata.core.util.CarbonUtil;
+import org.apache.carbondata.core.util.CarbonUtilException;
+import org.apache.carbondata.core.util.DataTypeUtil;
+import org.apache.carbondata.processing.newflow.sort.unsafe.UnsafeCarbonRowPage;
+import org.apache.carbondata.processing.newflow.sort.unsafe.holder.SortTempChunkHolder;
+import org.apache.carbondata.processing.newflow.sort.unsafe.holder.UnsafeSortTempFileChunkHolder;
+import org.apache.carbondata.processing.sortandgroupby.exception.CarbonSortKeyAndGroupByException;
+import org.apache.carbondata.processing.sortandgroupby.sortdata.SortParameters;
+import org.apache.carbondata.processing.sortandgroupby.sortdata.TempSortFileWriter;
+import org.apache.carbondata.processing.sortandgroupby.sortdata.TempSortFileWriterFactory;
+
+public class UnsafeIntermediateFileMerger implements Callable<Void> {
+ /**
+ * LOGGER
+ */
+ private static final LogService LOGGER =
+ LogServiceFactory.getLogService(UnsafeIntermediateFileMerger.class.getName());
+
+ /**
+ * recordHolderHeap
+ */
+ private AbstractQueue<SortTempChunkHolder> recordHolderHeap;
+
+ /**
+ * fileCounter
+ */
+ private int fileCounter;
+
+ /**
+ * stream
+ */
+ private DataOutputStream stream;
+
+ /**
+ * totalNumberOfRecords
+ */
+ private int totalNumberOfRecords;
+
+ /**
+ * writer
+ */
+ private TempSortFileWriter writer;
+
+ private SortParameters mergerParameters;
+
+ private File[] intermediateFiles;
+
+ private File outPutFile;
+
+ private boolean[] noDictionarycolumnMapping;
+
+ private long[] nullSetWords;
+
+ private ByteBuffer rowData;
+
+ /**
+ * IntermediateFileMerger Constructor
+ */
+ public UnsafeIntermediateFileMerger(SortParameters mergerParameters, File[] intermediateFiles,
+ File outPutFile) {
+ this.mergerParameters = mergerParameters;
+ this.fileCounter = intermediateFiles.length;
+ this.intermediateFiles = intermediateFiles;
+ this.outPutFile = outPutFile;
+ noDictionarycolumnMapping = mergerParameters.getNoDictionaryDimnesionColumn();
+ this.nullSetWords = new long[((mergerParameters.getMeasureColCount() - 1) >> 6) + 1];
+ // Take size of 2 MB for each row. I think it is high enough to use
+ rowData = ByteBuffer.allocate(2*1024*1024);
+ }
+
+ @Override public Void call() throws Exception {
+ long intermediateMergeStartTime = System.currentTimeMillis();
+ int fileConterConst = fileCounter;
+ boolean isFailed = false;
+ try {
+ startSorting();
+ initialize();
+ while (hasNext()) {
+ writeDataTofile(next());
+ }
+ double intermediateMergeCostTime =
+ (System.currentTimeMillis() - intermediateMergeStartTime) / 1000.0;
+ LOGGER.info("============================== Intermediate Merge of " + fileConterConst
+ + " Sort Temp Files Cost Time: " + intermediateMergeCostTime + "(s)");
+ } catch (Exception e) {
+ LOGGER.error(e, "Problem while intermediate merging");
+ isFailed = true;
+ } finally {
+ CarbonUtil.closeStreams(this.stream);
+ if (null != writer) {
+ writer.finish();
+ }
+ if (!isFailed) {
+ try {
+ finish();
+ } catch (CarbonSortKeyAndGroupByException e) {
+ LOGGER.error(e, "Problem while deleting the merge file");
+ }
+ } else {
+ if (outPutFile.delete()) {
+ LOGGER.error("Problem while deleting the merge file");
+ }
+ }
+ }
+
+ return null;
+ }
+
+ /**
+ * This method is responsible for initializing the out stream
+ *
+ * @throws CarbonSortKeyAndGroupByException
+ */
+ private void initialize() throws CarbonSortKeyAndGroupByException {
+ if (!mergerParameters.isSortFileCompressionEnabled() && !mergerParameters.isPrefetch()) {
+ try {
+ this.stream = new DataOutputStream(
+ new BufferedOutputStream(new FileOutputStream(outPutFile),
+ mergerParameters.getFileWriteBufferSize()));
+ this.stream.writeInt(this.totalNumberOfRecords);
+ } catch (FileNotFoundException e) {
+ throw new CarbonSortKeyAndGroupByException("Problem while getting the file", e);
+ } catch (IOException e) {
+ throw new CarbonSortKeyAndGroupByException("Problem while writing the data to file", e);
+ }
+ } else {
+ writer = TempSortFileWriterFactory.getInstance()
+ .getTempSortFileWriter(mergerParameters.isSortFileCompressionEnabled(),
+ mergerParameters.getDimColCount(), mergerParameters.getComplexDimColCount(),
+ mergerParameters.getMeasureColCount(), mergerParameters.getNoDictionaryCount(),
+ mergerParameters.getFileWriteBufferSize());
+ writer.initiaize(outPutFile, totalNumberOfRecords);
+ }
+ }
+
+ /**
+ * This method will be used to get the sorted record from file
+ *
+ * @return sorted record sorted record
+ * @throws CarbonSortKeyAndGroupByException
+ */
+ private Object[] getSortedRecordFromFile() throws CarbonSortKeyAndGroupByException {
+ Object[] row = null;
+
+ // poll the top object from heap
+ // heap maintains binary tree which is based on heap condition that will
+ // be based on comparator we are passing the heap
+ // when will call poll it will always delete root of the tree and then
+ // it does trickel down operation complexity is log(n)
+ SortTempChunkHolder poll = this.recordHolderHeap.poll();
+
+ // get the row from chunk
+ row = poll.getRow();
+
+ // check if there no entry present
+ if (!poll.hasNext()) {
+ // if chunk is empty then close the stream
+ poll.close();
+
+ // change the file counter
+ --this.fileCounter;
+
+ // reaturn row
+ return row;
+ }
+
+ // read new row
+ poll.readRow();
+
+ // add to heap
+ this.recordHolderHeap.add(poll);
+
+ // return row
+ return row;
+ }
+
+ /**
+ * Below method will be used to start storing process This method will get
+ * all the temp files present in sort temp folder then it will create the
+ * record holder heap and then it will read first record from each file and
+ * initialize the heap
+ *
+ * @throws CarbonSortKeyAndGroupByException
+ */
+ private void startSorting() throws CarbonSortKeyAndGroupByException {
+ LOGGER.info("Number of temp file: " + this.fileCounter);
+
+ // create record holder heap
+ createRecordHolderQueue(intermediateFiles);
+
+ // iterate over file list and create chunk holder and add to heap
+ LOGGER.info("Started adding first record from each file");
+
+ SortTempChunkHolder sortTempFileChunkHolder = null;
+
+ for (File tempFile : intermediateFiles) {
+ // create chunk holder
+ sortTempFileChunkHolder = new UnsafeSortTempFileChunkHolder(tempFile, mergerParameters);
+
+ sortTempFileChunkHolder.readRow();
+ this.totalNumberOfRecords += sortTempFileChunkHolder.numberOfRows();
+
+ // add to heap
+ this.recordHolderHeap.add(sortTempFileChunkHolder);
+ }
+
+ LOGGER.info("Heap Size" + this.recordHolderHeap.size());
+ }
+
+ /**
+ * This method will be used to create the heap which will be used to hold
+ * the chunk of data
+ *
+ * @param listFiles list of temp files
+ */
+ private void createRecordHolderQueue(File[] listFiles) {
+ // creating record holder heap
+ this.recordHolderHeap = new PriorityQueue<SortTempChunkHolder>(listFiles.length);
+ }
+
+ /**
+ * This method will be used to get the sorted row
+ *
+ * @return sorted row
+ * @throws CarbonSortKeyAndGroupByException
+ */
+ private Object[] next() throws CarbonSortKeyAndGroupByException {
+ return getSortedRecordFromFile();
+ }
+
+ /**
+ * This method will be used to check whether any more element is present or
+ * not
+ *
+ * @return more element is present
+ */
+ private boolean hasNext() {
+ return this.fileCounter > 0;
+ }
+
+ /**
+ * Below method will be used to write data to file
+ *
+ * @throws CarbonSortKeyAndGroupByException problem while writing
+ */
+ private void writeDataTofile(Object[] row) throws CarbonSortKeyAndGroupByException, IOException {
+ int dimCount = 0;
+ int size = 0;
+ char[] aggType = mergerParameters.getAggType();
+ for (; dimCount < noDictionarycolumnMapping.length; dimCount++) {
+ if (noDictionarycolumnMapping[dimCount]) {
+ byte[] col = (byte[]) row[dimCount];
+ rowData.putShort((short) col.length);
+ size += 2;
+ rowData.put(col);
+ size += col.length;
+ } else {
+ rowData.putInt((int) row[dimCount]);
+ size += 4;
+ }
+ }
+
+ // write complex dimensions here.
+ int dimensionSize = mergerParameters.getDimColCount();
+ int measureSize = mergerParameters.getMeasureColCount();
+ for (; dimCount < dimensionSize; dimCount++) {
+ byte[] col = (byte[]) row[dimCount];
+ rowData.putShort((short)col.length);
+ size += 2;
+ rowData.put(col);
+ size += col.length;
+ }
+ Arrays.fill(nullSetWords, 0);
+ int nullSetSize = nullSetWords.length * 8;
+ int nullLoc = size;
+ size += nullSetSize;
+ for (int mesCount = 0; mesCount < measureSize; mesCount++) {
+ Object value = row[mesCount + dimensionSize];
+ if (null != value) {
+ if (aggType[mesCount] == CarbonCommonConstants.SUM_COUNT_VALUE_MEASURE) {
+ Double val = (Double) value;
+ rowData.putDouble(size, val);
+ size += 8;
+ } else if (aggType[mesCount] == CarbonCommonConstants.BIG_INT_MEASURE) {
+ Long val = (Long) value;
+ rowData.putLong(size, val);
+ size += 8;
+ } else if (aggType[mesCount] == CarbonCommonConstants.BIG_DECIMAL_MEASURE) {
+ BigDecimal val = (BigDecimal) value;
+ byte[] bigDecimalInBytes = DataTypeUtil.bigDecimalToByte(val);
+ rowData.putShort(size, (short)bigDecimalInBytes.length);
+ size += 2;
+ for (int i = 0; i < bigDecimalInBytes.length; i++) {
+ rowData.put(size++, bigDecimalInBytes[i]);
+ }
+ }
+ UnsafeCarbonRowPage.set(nullSetWords, mesCount);
+ } else {
+ UnsafeCarbonRowPage.unset(nullSetWords, mesCount);
+ }
+ }
+ for (int i = 0; i < nullSetWords.length; i++) {
+ rowData.putLong(nullLoc, nullSetWords[i]);
+ nullLoc += 8;
+ }
+ byte[] rowBytes = new byte[size];
+ rowData.position(0);
+ rowData.get(rowBytes);
+ stream.write(rowBytes);
+ rowData.clear();
+ }
+
+ private void finish() throws CarbonSortKeyAndGroupByException {
+ if (recordHolderHeap != null) {
+ int size = recordHolderHeap.size();
+ for (int i = 0; i < size; i++) {
+ recordHolderHeap.poll().close();
+ }
+ }
+ try {
+ CarbonUtil.deleteFiles(intermediateFiles);
+ rowData.clear();
+ } catch (CarbonUtilException e) {
+ throw new CarbonSortKeyAndGroupByException("Problem while deleting the intermediate files");
+ }
+ }
+}
http://git-wip-us.apache.org/repos/asf/incubator-carbondata/blob/f1f9348d/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/merger/UnsafeIntermediateMerger.java
----------------------------------------------------------------------
diff --git a/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/merger/UnsafeIntermediateMerger.java b/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/merger/UnsafeIntermediateMerger.java
new file mode 100644
index 0000000..1cb2336
--- /dev/null
+++ b/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/merger/UnsafeIntermediateMerger.java
@@ -0,0 +1,180 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements. See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership. The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License. You may obtain a copy of the License at
+ *
+ * http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied. See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+package org.apache.carbondata.processing.newflow.sort.unsafe.merger;
+
+import java.io.File;
+import java.util.ArrayList;
+import java.util.List;
+import java.util.concurrent.ExecutorService;
+import java.util.concurrent.Executors;
+import java.util.concurrent.TimeUnit;
+
+import org.apache.carbondata.common.logging.LogService;
+import org.apache.carbondata.common.logging.LogServiceFactory;
+import org.apache.carbondata.core.constants.CarbonCommonConstants;
+import org.apache.carbondata.core.util.CarbonProperties;
+import org.apache.carbondata.processing.newflow.sort.unsafe.UnsafeCarbonRowPage;
+import org.apache.carbondata.processing.sortandgroupby.exception.CarbonSortKeyAndGroupByException;
+import org.apache.carbondata.processing.sortandgroupby.sortdata.SortParameters;
+
+/**
+ * It does mergesort intermediate files to big file.
+ */
+public class UnsafeIntermediateMerger {
+
+ private static final LogService LOGGER =
+ LogServiceFactory.getLogService(UnsafeIntermediateMerger.class.getName());
+
+ /**
+ * executorService
+ */
+ private ExecutorService executorService;
+ /**
+ * rowPages
+ */
+ private List<UnsafeCarbonRowPage> rowPages;
+
+ private List<UnsafeInMemoryIntermediateDataMerger> mergedPages;
+
+ private SortParameters parameters;
+
+ private final Object lockObject = new Object();
+
+ private boolean offHeap;
+
+ private List<File> procFiles;
+
+ public UnsafeIntermediateMerger(SortParameters parameters) {
+ this.parameters = parameters;
+ // processed file list
+ this.rowPages = new ArrayList<UnsafeCarbonRowPage>(CarbonCommonConstants.CONSTANT_SIZE_TEN);
+ this.mergedPages = new ArrayList<>();
+ this.executorService = Executors.newFixedThreadPool(parameters.getNumberOfCores());
+ this.offHeap = Boolean.parseBoolean(CarbonProperties.getInstance()
+ .getProperty(CarbonCommonConstants.ENABLE_OFFHEAP_SORT,
+ CarbonCommonConstants.ENABLE_OFFHEAP_SORT_DEFAULT));
+ this.procFiles = new ArrayList<File>(CarbonCommonConstants.CONSTANT_SIZE_TEN);
+ }
+
+ public void addDataChunkToMerge(UnsafeCarbonRowPage rowPage) {
+ // add sort temp filename to and arrayList. When the list size reaches 20 then
+ // intermediate merging of sort temp files will be triggered
+ synchronized (lockObject) {
+ rowPages.add(rowPage);
+ }
+ }
+
+ public void addFileToMerge(File sortTempFile) {
+ // add sort temp filename to and arrayList. When the list size reaches 20 then
+ // intermediate merging of sort temp files will be triggered
+ synchronized (lockObject) {
+ procFiles.add(sortTempFile);
+ }
+ }
+
+ public void startFileMergingIfPossible() {
+ File[] fileList;
+ if (procFiles.size() >= parameters.getNumberOfIntermediateFileToBeMerged()) {
+ synchronized (lockObject) {
+ fileList = procFiles.toArray(new File[procFiles.size()]);
+ this.procFiles = new ArrayList<File>();
+ }
+ LOGGER.debug("Sumitting request for intermediate merging no of files: " + fileList.length);
+ startIntermediateMerging(fileList);
+ }
+ }
+
+ /**
+ * Below method will be used to start the intermediate file merging
+ *
+ * @param intermediateFiles
+ */
+ private void startIntermediateMerging(File[] intermediateFiles) {
+ File file = new File(
+ parameters.getTempFileLocation() + File.separator + parameters.getTableName() + System
+ .nanoTime() + CarbonCommonConstants.MERGERD_EXTENSION);
+ UnsafeIntermediateFileMerger merger =
+ new UnsafeIntermediateFileMerger(parameters, intermediateFiles, file);
+ executorService.submit(merger);
+ }
+
+ public void startInmemoryMergingIfPossible() throws CarbonSortKeyAndGroupByException {
+ UnsafeCarbonRowPage[] localRowPages;
+ if (rowPages.size() >= parameters.getNumberOfIntermediateFileToBeMerged()) {
+ int totalRows = 0;
+ synchronized (lockObject) {
+ totalRows = getTotalNumberOfRows(rowPages);
+ if (totalRows <= 0) {
+ return;
+ }
+ localRowPages = rowPages.toArray(new UnsafeCarbonRowPage[rowPages.size()]);
+ this.rowPages = new ArrayList<>();
+ }
+ LOGGER.debug("Sumitting request for intermediate merging of in-memory pages : "
+ + localRowPages.length);
+ startIntermediateMerging(localRowPages, totalRows);
+ }
+ }
+
+ /**
+ * Below method will be used to start the intermediate file merging
+ *
+ * @param rowPages
+ */
+ private void startIntermediateMerging(UnsafeCarbonRowPage[] rowPages, int totalRows)
+ throws CarbonSortKeyAndGroupByException {
+ UnsafeInMemoryIntermediateDataMerger merger =
+ new UnsafeInMemoryIntermediateDataMerger(rowPages, totalRows);
+ mergedPages.add(merger);
+ executorService.submit(merger);
+ }
+
+ private int getTotalNumberOfRows(List<UnsafeCarbonRowPage> unsafeCarbonRowPages) {
+ int totalSize = 0;
+ for (UnsafeCarbonRowPage unsafeCarbonRowPage : unsafeCarbonRowPages) {
+ totalSize += unsafeCarbonRowPage.getBuffer().getActualSize();
+ }
+ return totalSize;
+ }
+
+ public void finish() throws CarbonSortKeyAndGroupByException {
+ try {
+ executorService.shutdown();
+ executorService.awaitTermination(2, TimeUnit.DAYS);
+ } catch (InterruptedException e) {
+ throw new CarbonSortKeyAndGroupByException("Problem while shutdown the server ", e);
+ }
+ }
+
+ public void close() {
+ if (executorService.isShutdown()) {
+ executorService.shutdownNow();
+ }
+ rowPages.clear();
+ rowPages = null;
+ }
+
+ public List<UnsafeCarbonRowPage> getRowPages() {
+ return rowPages;
+ }
+
+ public List<UnsafeInMemoryIntermediateDataMerger> getMergedPages() {
+ return mergedPages;
+ }
+}
http://git-wip-us.apache.org/repos/asf/incubator-carbondata/blob/f1f9348d/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/merger/UnsafeSingleThreadFinalSortFilesMerger.java
----------------------------------------------------------------------
diff --git a/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/merger/UnsafeSingleThreadFinalSortFilesMerger.java b/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/merger/UnsafeSingleThreadFinalSortFilesMerger.java
new file mode 100644
index 0000000..a142823
--- /dev/null
+++ b/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/merger/UnsafeSingleThreadFinalSortFilesMerger.java
@@ -0,0 +1,313 @@
+/*
+ * 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.processing.newflow.sort.unsafe.merger;
+
+import java.io.File;
+import java.io.FileFilter;
+import java.util.AbstractQueue;
+import java.util.List;
+import java.util.PriorityQueue;
+
+import org.apache.carbondata.common.CarbonIterator;
+import org.apache.carbondata.common.logging.LogService;
+import org.apache.carbondata.common.logging.LogServiceFactory;
+import org.apache.carbondata.processing.newflow.sort.unsafe.UnsafeCarbonRowPage;
+import org.apache.carbondata.processing.newflow.sort.unsafe.holder.SortTempChunkHolder;
+import org.apache.carbondata.processing.newflow.sort.unsafe.holder.UnsafeFinalMergePageHolder;
+import org.apache.carbondata.processing.newflow.sort.unsafe.holder.UnsafeInmemoryHolder;
+import org.apache.carbondata.processing.newflow.sort.unsafe.holder.UnsafeSortTempFileChunkHolder;
+import org.apache.carbondata.processing.sortandgroupby.sortdata.SortParameters;
+import org.apache.carbondata.processing.store.writer.exception.CarbonDataWriterException;
+import org.apache.carbondata.processing.util.RemoveDictionaryUtil;
+
+public class UnsafeSingleThreadFinalSortFilesMerger extends CarbonIterator<Object[]> {
+ /**
+ * LOGGER
+ */
+ private static final LogService LOGGER =
+ LogServiceFactory.getLogService(UnsafeSingleThreadFinalSortFilesMerger.class.getName());
+
+ /**
+ * lockObject
+ */
+ private static final Object LOCKOBJECT = new Object();
+
+ /**
+ * fileCounter
+ */
+ private int fileCounter;
+
+ /**
+ * recordHolderHeap
+ */
+ private AbstractQueue<SortTempChunkHolder> recordHolderHeapLocal;
+
+ private SortParameters parameters;
+
+ /**
+ * number of measures
+ */
+ private int measureCount;
+
+ /**
+ * number of dimensionCount
+ */
+ private int dimensionCount;
+
+ /**
+ * number of complexDimensionCount
+ */
+ private int noDictionaryCount;
+
+ private int complexDimensionCount;
+
+ private boolean[] isNoDictionaryDimensionColumn;
+
+ /**
+ * tempFileLocation
+ */
+ private String tempFileLocation;
+
+ private String tableName;
+
+ public UnsafeSingleThreadFinalSortFilesMerger(SortParameters parameters) {
+ this.parameters = parameters;
+ // set measure and dimension count
+ this.measureCount = parameters.getMeasureColCount();
+ this.dimensionCount = parameters.getDimColCount();
+ this.complexDimensionCount = parameters.getComplexDimColCount();
+
+ this.noDictionaryCount = parameters.getNoDictionaryCount();
+ this.isNoDictionaryDimensionColumn = parameters.getNoDictionaryDimnesionColumn();
+ this.tempFileLocation = parameters.getTempFileLocation();
+ this.tableName = parameters.getTableName();
+ }
+
+ /**
+ * This method will be used to merger the merged files
+ *
+ */
+ public void startFinalMerge(UnsafeCarbonRowPage[] rowPages,
+ List<UnsafeInMemoryIntermediateDataMerger> merges) throws CarbonDataWriterException {
+ startSorting(rowPages, merges);
+ }
+
+ /**
+ * Below method will be used to start storing process This method will get
+ * all the temp files present in sort temp folder then it will create the
+ * record holder heap and then it will read first record from each file and
+ * initialize the heap
+ *
+ */
+ private void startSorting(UnsafeCarbonRowPage[] rowPages,
+ List<UnsafeInMemoryIntermediateDataMerger> merges) throws CarbonDataWriterException {
+ try {
+ File[] filesToMergeSort = getFilesToMergeSort();
+ this.fileCounter = rowPages.length + filesToMergeSort.length + merges.size();
+
+ LOGGER.info("Number of row pages: " + this.fileCounter);
+
+ // create record holder heap
+ createRecordHolderQueue();
+
+ // iterate over file list and create chunk holder and add to heap
+ LOGGER.info("Started adding first record from each page");
+ for (final UnsafeCarbonRowPage rowPage : rowPages) {
+
+ SortTempChunkHolder sortTempFileChunkHolder = new UnsafeInmemoryHolder(rowPage,
+ parameters.getDimColCount() + parameters.getMeasureColCount());
+
+ // initialize
+ sortTempFileChunkHolder.readRow();
+
+ recordHolderHeapLocal.add(sortTempFileChunkHolder);
+ }
+
+ for (final UnsafeInMemoryIntermediateDataMerger merger : merges) {
+
+ SortTempChunkHolder sortTempFileChunkHolder =
+ new UnsafeFinalMergePageHolder(merger, parameters.getNoDictionaryDimnesionColumn(),
+ parameters.getDimColCount() + parameters.getMeasureColCount());
+
+ // initialize
+ sortTempFileChunkHolder.readRow();
+
+ recordHolderHeapLocal.add(sortTempFileChunkHolder);
+ }
+
+ for (final File file : filesToMergeSort) {
+
+ SortTempChunkHolder sortTempFileChunkHolder =
+ new UnsafeSortTempFileChunkHolder(file, parameters);
+
+ // initialize
+ sortTempFileChunkHolder.readRow();
+
+ recordHolderHeapLocal.add(sortTempFileChunkHolder);
+ }
+
+ LOGGER.info("Heap Size" + this.recordHolderHeapLocal.size());
+ } catch (Exception e) {
+ LOGGER.error(e);
+ throw new CarbonDataWriterException(e.getMessage());
+ }
+ }
+
+ private File[] getFilesToMergeSort() {
+ // get all the merged files
+ File file = new File(tempFileLocation);
+
+ File[] fileList = file.listFiles(new FileFilter() {
+ public boolean accept(File pathname) {
+ return pathname.getName().startsWith(tableName);
+ }
+ });
+
+ if (null == fileList || fileList.length < 0) {
+ return new File[0];
+ }
+ return fileList;
+ }
+
+ /**
+ * This method will be used to create the heap which will be used to hold
+ * the chunk of data
+ */
+ private void createRecordHolderQueue() {
+ // creating record holder heap
+ this.recordHolderHeapLocal = new PriorityQueue<SortTempChunkHolder>(fileCounter);
+ }
+
+ /**
+ * This method will be used to get the sorted row
+ *
+ * @return sorted row
+ */
+ public Object[] next() {
+ return convertRow(getSortedRecordFromFile());
+ }
+
+ /**
+ * This method will be used to get the sorted record from file
+ *
+ * @return sorted record sorted record
+ */
+ private Object[] getSortedRecordFromFile() throws CarbonDataWriterException {
+ Object[] row = null;
+
+ // poll the top object from heap
+ // heap maintains binary tree which is based on heap condition that will
+ // be based on comparator we are passing the heap
+ // when will call poll it will always delete root of the tree and then
+ // it does trickel down operation complexity is log(n)
+ SortTempChunkHolder poll = this.recordHolderHeapLocal.poll();
+
+ // get the row from chunk
+ row = poll.getRow();
+
+ // check if there no entry present
+ if (!poll.hasNext()) {
+ // if chunk is empty then close the stream
+ poll.close();
+
+ // change the file counter
+ --this.fileCounter;
+
+ // reaturn row
+ return row;
+ }
+
+ // read new row
+ try {
+ poll.readRow();
+ } catch (Exception e) {
+ throw new CarbonDataWriterException(e.getMessage(), e);
+ }
+
+ // add to heap
+ this.recordHolderHeapLocal.add(poll);
+
+ // return row
+ return row;
+ }
+
+ /**
+ * This method will be used to check whether any more element is present or
+ * not
+ *
+ * @return more element is present
+ */
+ public boolean hasNext() {
+ return this.fileCounter > 0;
+ }
+
+ private Object[] convertRow(Object[] data) {
+ // create new row of size 3 (1 for dims , 1 for high card , 1 for measures)
+
+ Object[] holder = new Object[3];
+ int index = 0;
+ int nonDicIndex = 0;
+ int allCount = 0;
+ int[] dim = new int[this.dimensionCount];
+ byte[][] nonDicArray = new byte[this.noDictionaryCount + this.complexDimensionCount][];
+ Object[] measures = new Object[this.measureCount];
+ try {
+ // read dimension values
+ for (int i = 0; i < isNoDictionaryDimensionColumn.length; i++) {
+ if (isNoDictionaryDimensionColumn[i]) {
+ nonDicArray[nonDicIndex++] = (byte[]) data[i];
+ } else {
+ dim[index++] = (int) data[allCount];
+ }
+ allCount++;
+ }
+
+ for (int i = 0; i < complexDimensionCount; i++) {
+ nonDicArray[nonDicIndex++] = (byte[]) data[allCount];
+ allCount++;
+ }
+
+ index = 0;
+ // read measure values
+ for (int i = 0; i < this.measureCount; i++) {
+ measures[index++] = data[allCount];
+ allCount++;
+ }
+
+ RemoveDictionaryUtil.prepareOutObj(holder, dim, nonDicArray, measures);
+
+ // increment number if record read
+ } catch (Exception e) {
+ throw new RuntimeException("Problem while converting row ", e);
+ }
+
+ //return out row
+ return holder;
+ }
+
+ public void clear() {
+ if (null != recordHolderHeapLocal) {
+ for (SortTempChunkHolder pageHolder : recordHolderHeapLocal) {
+ pageHolder.close();
+ }
+ recordHolderHeapLocal = null;
+ }
+ }
+}
[3/4] incubator-carbondata git commit: Added unsafe on-heap/off-heap
sort to improve loading performance
Posted by ja...@apache.org.
Added unsafe on-heap/off-heap sort to improve loading performance
fixed testcase
fixed row duplicated issue.
rebased and changed the default value
Added file header for code porting.
Project: http://git-wip-us.apache.org/repos/asf/incubator-carbondata/repo
Commit: http://git-wip-us.apache.org/repos/asf/incubator-carbondata/commit/f1f9348d
Tree: http://git-wip-us.apache.org/repos/asf/incubator-carbondata/tree/f1f9348d
Diff: http://git-wip-us.apache.org/repos/asf/incubator-carbondata/diff/f1f9348d
Branch: refs/heads/master
Commit: f1f9348d0d7150c95500f8f10d3fd3adde47ecb2
Parents: 8940514
Author: ravipesala <ra...@gmail.com>
Authored: Wed Dec 7 00:29:04 2016 +0530
Committer: jackylk <ja...@huawei.com>
Committed: Tue Dec 13 19:13:25 2016 +0800
----------------------------------------------------------------------
.../core/constants/CarbonCommonConstants.java | 32 +
.../core/memory/HeapMemoryAllocator.java | 84 ++
.../carbondata/core/memory/MemoryAllocator.java | 36 +
.../carbondata/core/memory/MemoryBlock.java | 57 ++
.../carbondata/core/memory/MemoryLocation.java | 55 ++
.../core/memory/UnsafeMemoryAllocator.java | 40 +
.../carbondata/core/unsafe/CarbonUnsafe.java | 48 +
dev/javastyle-config.xml | 6 +-
dev/javastyle-suppressions.xml | 35 +
.../newflow/iterator/InputIterator.java | 40 +
.../impl/UnsafeParallelReadMergeSorterImpl.java | 219 +++++
.../newflow/sort/unsafe/IntPointerBuffer.java | 95 ++
.../sort/unsafe/UnsafeCarbonRowPage.java | 356 +++++++
.../sort/unsafe/UnsafeMemoryManager.java | 99 ++
.../newflow/sort/unsafe/UnsafeSortDataRows.java | 356 +++++++
.../unsafe/comparator/UnsafeRowComparator.java | 133 +++
.../UnsafeRowComparatorForNormalDIms.java | 61 ++
.../sort/unsafe/holder/SortTempChunkHolder.java | 35 +
.../sort/unsafe/holder/UnsafeCarbonRow.java | 23 +
.../unsafe/holder/UnsafeCarbonRowForMerge.java | 25 +
.../holder/UnsafeFinalMergePageHolder.java | 90 ++
.../unsafe/holder/UnsafeInmemoryHolder.java | 80 ++
.../holder/UnsafeInmemoryMergeHolder.java | 90 ++
.../holder/UnsafeSortTempFileChunkHolder.java | 455 +++++++++
.../UnsafeInMemoryIntermediateDataMerger.java | 217 +++++
.../merger/UnsafeIntermediateFileMerger.java | 364 +++++++
.../unsafe/merger/UnsafeIntermediateMerger.java | 180 ++++
.../UnsafeSingleThreadFinalSortFilesMerger.java | 313 ++++++
.../newflow/sort/unsafe/sort/TimSort.java | 943 +++++++++++++++++++
.../unsafe/sort/UnsafeIntSortDataFormat.java | 74 ++
.../newflow/steps/SortProcessorStepImpl.java | 10 +-
31 files changed, 4647 insertions(+), 4 deletions(-)
----------------------------------------------------------------------
http://git-wip-us.apache.org/repos/asf/incubator-carbondata/blob/f1f9348d/core/src/main/java/org/apache/carbondata/core/constants/CarbonCommonConstants.java
----------------------------------------------------------------------
diff --git a/core/src/main/java/org/apache/carbondata/core/constants/CarbonCommonConstants.java b/core/src/main/java/org/apache/carbondata/core/constants/CarbonCommonConstants.java
index 8257756..033b48d 100644
--- a/core/src/main/java/org/apache/carbondata/core/constants/CarbonCommonConstants.java
+++ b/core/src/main/java/org/apache/carbondata/core/constants/CarbonCommonConstants.java
@@ -953,6 +953,38 @@ public final class CarbonCommonConstants {
*/
public static final int CARBON_EXECUTOR_STARTUP_THREAD_SLEEP_TIME = 250;
+ /**
+ * to enable offheap sort
+ */
+ public static final String ENABLE_UNSAFE_SORT = "enable.unsafe.sort";
+
+ /**
+ * to enable offheap sort
+ */
+ public static final String ENABLE_UNSAFE_SORT_DEFAULT = "false";
+
+ /**
+ * to enable offheap sort
+ */
+ public static final String ENABLE_OFFHEAP_SORT = "enable.offheap.sort";
+
+ /**
+ * to enable offheap sort
+ */
+ public static final String ENABLE_OFFHEAP_SORT_DEFAULT = "true";
+
+ public static final String ENABLE_INMEMORY_MERGE_SORT = "enable.inmemory.merge.sort";
+
+ public static final String ENABLE_INMEMORY_MERGE_SORT_DEFAULT = "true";
+
+ public static final String OFFHEAP_SORT_CHUNK_SIZE_IN_MB = "offheap.sort.chunk.size.inmb";
+
+ public static final String OFFHEAP_SORT_CHUNK_SIZE_IN_MB_DEFAULT = "64";
+
+ public static final String IN_MEMORY_FOR_SORT_DATA_IN_MB = "sort.inmemory.size.inmb";
+
+ public static final String IN_MEMORY_FOR_SORT_DATA_IN_MB_DEFAULT = "1024";
+
private CarbonCommonConstants() {
}
}
http://git-wip-us.apache.org/repos/asf/incubator-carbondata/blob/f1f9348d/core/src/main/java/org/apache/carbondata/core/memory/HeapMemoryAllocator.java
----------------------------------------------------------------------
diff --git a/core/src/main/java/org/apache/carbondata/core/memory/HeapMemoryAllocator.java b/core/src/main/java/org/apache/carbondata/core/memory/HeapMemoryAllocator.java
new file mode 100644
index 0000000..67e766c
--- /dev/null
+++ b/core/src/main/java/org/apache/carbondata/core/memory/HeapMemoryAllocator.java
@@ -0,0 +1,84 @@
+/*
+ * 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.memory;
+
+import javax.annotation.concurrent.GuardedBy;
+import java.lang.ref.WeakReference;
+import java.util.HashMap;
+import java.util.LinkedList;
+import java.util.Map;
+
+import org.apache.carbondata.core.unsafe.CarbonUnsafe;
+
+/**
+ * Code ported from Apache Spark {org.apache.spark.unsafe.memory} package
+ * A simple {@link MemoryAllocator} that can allocate up to 16GB using a JVM long primitive array.
+ */
+public class HeapMemoryAllocator implements MemoryAllocator {
+
+ @GuardedBy("this") private final Map<Long, LinkedList<WeakReference<MemoryBlock>>>
+ bufferPoolsBySize = new HashMap<>();
+
+ private static final int POOLING_THRESHOLD_BYTES = 1024 * 1024;
+
+ /**
+ * Returns true if allocations of the given size should go through the pooling mechanism and
+ * false otherwise.
+ */
+ private boolean shouldPool(long size) {
+ // Very small allocations are less likely to benefit from pooling.
+ return size >= POOLING_THRESHOLD_BYTES;
+ }
+
+ @Override public MemoryBlock allocate(long size) throws OutOfMemoryError {
+ if (shouldPool(size)) {
+ synchronized (this) {
+ final LinkedList<WeakReference<MemoryBlock>> pool = bufferPoolsBySize.get(size);
+ if (pool != null) {
+ while (!pool.isEmpty()) {
+ final WeakReference<MemoryBlock> blockReference = pool.pop();
+ final MemoryBlock memory = blockReference.get();
+ if (memory != null) {
+ assert (memory.size() == size);
+ return memory;
+ }
+ }
+ bufferPoolsBySize.remove(size);
+ }
+ }
+ }
+ long[] array = new long[(int) ((size + 7) / 8)];
+ return new MemoryBlock(array, CarbonUnsafe.LONG_ARRAY_OFFSET, size);
+ }
+
+ @Override public void free(MemoryBlock memory) {
+ final long size = memory.size();
+ if (shouldPool(size)) {
+ synchronized (this) {
+ LinkedList<WeakReference<MemoryBlock>> pool = bufferPoolsBySize.get(size);
+ if (pool == null) {
+ pool = new LinkedList<>();
+ bufferPoolsBySize.put(size, pool);
+ }
+ pool.add(new WeakReference<>(memory));
+ }
+ } else {
+ // Do nothing
+ }
+ }
+}
http://git-wip-us.apache.org/repos/asf/incubator-carbondata/blob/f1f9348d/core/src/main/java/org/apache/carbondata/core/memory/MemoryAllocator.java
----------------------------------------------------------------------
diff --git a/core/src/main/java/org/apache/carbondata/core/memory/MemoryAllocator.java b/core/src/main/java/org/apache/carbondata/core/memory/MemoryAllocator.java
new file mode 100644
index 0000000..ccb3986
--- /dev/null
+++ b/core/src/main/java/org/apache/carbondata/core/memory/MemoryAllocator.java
@@ -0,0 +1,36 @@
+/*
+ * 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.memory;
+
+/**
+ * Code ported from Apache Spark {org.apache.spark.unsafe.memory} package
+ */
+public interface MemoryAllocator {
+
+ /**
+ * Allocates a contiguous block of memory. Note that the allocated memory is not guaranteed
+ * to be zeroed out (call `zero()` on the result if this is necessary).
+ */
+ MemoryBlock allocate(long size) throws OutOfMemoryError;
+
+ void free(MemoryBlock memory);
+
+ MemoryAllocator UNSAFE = new UnsafeMemoryAllocator();
+
+ MemoryAllocator HEAP = new HeapMemoryAllocator();
+}
http://git-wip-us.apache.org/repos/asf/incubator-carbondata/blob/f1f9348d/core/src/main/java/org/apache/carbondata/core/memory/MemoryBlock.java
----------------------------------------------------------------------
diff --git a/core/src/main/java/org/apache/carbondata/core/memory/MemoryBlock.java b/core/src/main/java/org/apache/carbondata/core/memory/MemoryBlock.java
new file mode 100644
index 0000000..ab9b3d4
--- /dev/null
+++ b/core/src/main/java/org/apache/carbondata/core/memory/MemoryBlock.java
@@ -0,0 +1,57 @@
+/*
+ * 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.memory;
+
+import javax.annotation.Nullable;
+
+import org.apache.spark.unsafe.Platform;
+
+/**
+ * Code ported from Apache Spark {org.apache.spark.unsafe.memory} package
+ * A consecutive block of memory, starting at a {@link MemoryLocation} with a fixed size.
+ */
+public class MemoryBlock extends MemoryLocation {
+
+ private final long length;
+
+ /**
+ * Optional page number; used when this MemoryBlock represents a page allocated by a
+ * TaskMemoryManager. This field is public so that it can be modified by the TaskMemoryManager,
+ * which lives in a different package.
+ */
+ public int pageNumber = -1;
+
+ public MemoryBlock(@Nullable Object obj, long offset, long length) {
+ super(obj, offset);
+ this.length = length;
+ }
+
+ /**
+ * Returns the size of the memory block.
+ */
+ public long size() {
+ return length;
+ }
+
+ /**
+ * Creates a memory block pointing to the memory used by the long array.
+ */
+ public static MemoryBlock fromLongArray(final long[] array) {
+ return new MemoryBlock(array, Platform.LONG_ARRAY_OFFSET, array.length * 8);
+ }
+}
http://git-wip-us.apache.org/repos/asf/incubator-carbondata/blob/f1f9348d/core/src/main/java/org/apache/carbondata/core/memory/MemoryLocation.java
----------------------------------------------------------------------
diff --git a/core/src/main/java/org/apache/carbondata/core/memory/MemoryLocation.java b/core/src/main/java/org/apache/carbondata/core/memory/MemoryLocation.java
new file mode 100644
index 0000000..0419e04
--- /dev/null
+++ b/core/src/main/java/org/apache/carbondata/core/memory/MemoryLocation.java
@@ -0,0 +1,55 @@
+/*
+ * 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.memory;
+
+import javax.annotation.Nullable;
+
+/**
+ * Code ported from Apache Spark {org.apache.spark.unsafe.memory} package
+ * A memory location. Tracked either by a memory address (with off-heap allocation),
+ * or by an offset from a JVM object (in-heap allocation).
+ */
+public class MemoryLocation {
+
+ @Nullable
+ Object obj;
+
+ long offset;
+
+ public MemoryLocation(@Nullable Object obj, long offset) {
+ this.obj = obj;
+ this.offset = offset;
+ }
+
+ public MemoryLocation() {
+ this(null, 0);
+ }
+
+ public void setObjAndOffset(Object newObj, long newOffset) {
+ this.obj = newObj;
+ this.offset = newOffset;
+ }
+
+ public final Object getBaseObject() {
+ return obj;
+ }
+
+ public final long getBaseOffset() {
+ return offset;
+ }
+}
http://git-wip-us.apache.org/repos/asf/incubator-carbondata/blob/f1f9348d/core/src/main/java/org/apache/carbondata/core/memory/UnsafeMemoryAllocator.java
----------------------------------------------------------------------
diff --git a/core/src/main/java/org/apache/carbondata/core/memory/UnsafeMemoryAllocator.java b/core/src/main/java/org/apache/carbondata/core/memory/UnsafeMemoryAllocator.java
new file mode 100644
index 0000000..ae4cc0a
--- /dev/null
+++ b/core/src/main/java/org/apache/carbondata/core/memory/UnsafeMemoryAllocator.java
@@ -0,0 +1,40 @@
+/*
+ * 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.memory;
+
+import org.apache.carbondata.core.unsafe.CarbonUnsafe;
+
+/**
+ * Code ported from Apache Spark {org.apache.spark.unsafe.memory} package
+ * A simple {@link MemoryAllocator} that uses {@code Unsafe} to allocate off-heap memory.
+ */
+public class UnsafeMemoryAllocator implements MemoryAllocator {
+
+ @Override
+ public MemoryBlock allocate(long size) throws OutOfMemoryError {
+ long address = CarbonUnsafe.unsafe.allocateMemory(size);
+ return new MemoryBlock(null, address, size);
+ }
+
+ @Override
+ public void free(MemoryBlock memory) {
+ assert (memory.obj == null) :
+ "baseObject not null; are you trying to use the off-heap allocator to free on-heap memory?";
+ CarbonUnsafe.unsafe.freeMemory(memory.offset);
+ }
+}
http://git-wip-us.apache.org/repos/asf/incubator-carbondata/blob/f1f9348d/core/src/main/java/org/apache/carbondata/core/unsafe/CarbonUnsafe.java
----------------------------------------------------------------------
diff --git a/core/src/main/java/org/apache/carbondata/core/unsafe/CarbonUnsafe.java b/core/src/main/java/org/apache/carbondata/core/unsafe/CarbonUnsafe.java
new file mode 100644
index 0000000..89ac14a
--- /dev/null
+++ b/core/src/main/java/org/apache/carbondata/core/unsafe/CarbonUnsafe.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.apache.carbondata.core.unsafe;
+
+import java.lang.reflect.Field;
+
+import sun.misc.Unsafe;
+
+
+public final class CarbonUnsafe {
+
+ public static final int BYTE_ARRAY_OFFSET;
+
+ public static final int LONG_ARRAY_OFFSET;
+
+ public static Unsafe unsafe;
+
+ static {
+ try {
+ Field cause = Unsafe.class.getDeclaredField("theUnsafe");
+ cause.setAccessible(true);
+ unsafe = (Unsafe) cause.get((Object) null);
+ } catch (Throwable var2) {
+ unsafe = null;
+ }
+ if (unsafe != null) {
+ BYTE_ARRAY_OFFSET = unsafe.arrayBaseOffset(byte[].class);
+ LONG_ARRAY_OFFSET = unsafe.arrayBaseOffset(long[].class);
+ } else {
+ BYTE_ARRAY_OFFSET = 0;
+ LONG_ARRAY_OFFSET = 0;
+ }
+ }
+}
http://git-wip-us.apache.org/repos/asf/incubator-carbondata/blob/f1f9348d/dev/javastyle-config.xml
----------------------------------------------------------------------
diff --git a/dev/javastyle-config.xml b/dev/javastyle-config.xml
index e77c9fd..6c9b50c 100644
--- a/dev/javastyle-config.xml
+++ b/dev/javastyle-config.xml
@@ -48,9 +48,9 @@
<property name="fileExtensions" value="java, properties, xml"/>
- <!--module name="SuppressionFilter">
- <property name="file" value="dev/checkstyle-suppressions.xml"/>
- </module-->
+ <module name="SuppressionFilter">
+ <property name="file" value="dev/javastyle-suppressions.xml"/>
+ </module>
<!-- Checks for whitespace -->
<!-- See http://checkstyle.sf.net/config_whitespace.html -->
http://git-wip-us.apache.org/repos/asf/incubator-carbondata/blob/f1f9348d/dev/javastyle-suppressions.xml
----------------------------------------------------------------------
diff --git a/dev/javastyle-suppressions.xml b/dev/javastyle-suppressions.xml
new file mode 100644
index 0000000..9780dcb
--- /dev/null
+++ b/dev/javastyle-suppressions.xml
@@ -0,0 +1,35 @@
+<!--
+ ~ 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.
+ -->
+
+<!DOCTYPE suppressions PUBLIC
+"-//Puppy Crawl//DTD Suppressions 1.1//EN"
+"http://www.puppycrawl.com/dtds/suppressions_1_1.dtd">
+
+<!--
+
+ This file contains suppression rules for Checkstyle checks.
+ Ideally only files that cannot be modified (e.g. third-party code)
+ should be added here. All other violations should be fixed.
+
+-->
+
+<suppressions>
+ <suppress checks=".*"
+ files="org/apache/carbondata/processing/newflow/sort/unsafe/sort/TimSort.java"/>
+ <suppress checks=".*"
+ files="org/apache/carbondata/core/memory/HeapMemoryAllocator"/>
+</suppressions>
http://git-wip-us.apache.org/repos/asf/incubator-carbondata/blob/f1f9348d/processing/src/main/java/org/apache/carbondata/processing/newflow/iterator/InputIterator.java
----------------------------------------------------------------------
diff --git a/processing/src/main/java/org/apache/carbondata/processing/newflow/iterator/InputIterator.java b/processing/src/main/java/org/apache/carbondata/processing/newflow/iterator/InputIterator.java
new file mode 100644
index 0000000..a87aaa2
--- /dev/null
+++ b/processing/src/main/java/org/apache/carbondata/processing/newflow/iterator/InputIterator.java
@@ -0,0 +1,40 @@
+/*
+ * 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.processing.newflow.iterator;
+
+import java.util.Iterator;
+
+/**
+ * It iterates the data of record readers
+ */
+public abstract class InputIterator<E> implements Iterator<E> {
+
+ @Override
+ public void remove() {
+ throw new UnsupportedOperationException("Not permitted");
+ }
+
+ /**
+ * Initialize the iterator
+ */
+ public abstract void initialize();
+
+ /**
+ * Close the resources
+ */
+ public abstract void close();
+}
http://git-wip-us.apache.org/repos/asf/incubator-carbondata/blob/f1f9348d/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/impl/UnsafeParallelReadMergeSorterImpl.java
----------------------------------------------------------------------
diff --git a/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/impl/UnsafeParallelReadMergeSorterImpl.java b/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/impl/UnsafeParallelReadMergeSorterImpl.java
new file mode 100644
index 0000000..3a29647
--- /dev/null
+++ b/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/impl/UnsafeParallelReadMergeSorterImpl.java
@@ -0,0 +1,219 @@
+/*
+ * 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.processing.newflow.sort.impl;
+
+import java.io.File;
+import java.util.Iterator;
+import java.util.List;
+import java.util.concurrent.Callable;
+import java.util.concurrent.ExecutorService;
+import java.util.concurrent.Executors;
+import java.util.concurrent.TimeUnit;
+
+import org.apache.carbondata.common.CarbonIterator;
+import org.apache.carbondata.common.logging.LogService;
+import org.apache.carbondata.common.logging.LogServiceFactory;
+import org.apache.carbondata.core.constants.CarbonCommonConstants;
+import org.apache.carbondata.core.util.CarbonProperties;
+import org.apache.carbondata.core.util.CarbonTimeStatisticsFactory;
+import org.apache.carbondata.processing.newflow.DataField;
+import org.apache.carbondata.processing.newflow.exception.CarbonDataLoadingException;
+import org.apache.carbondata.processing.newflow.row.CarbonRow;
+import org.apache.carbondata.processing.newflow.row.CarbonRowBatch;
+import org.apache.carbondata.processing.newflow.sort.Sorter;
+import org.apache.carbondata.processing.newflow.sort.unsafe.UnsafeCarbonRowPage;
+import org.apache.carbondata.processing.newflow.sort.unsafe.UnsafeSortDataRows;
+import org.apache.carbondata.processing.newflow.sort.unsafe.merger.UnsafeIntermediateMerger;
+import org.apache.carbondata.processing.newflow.sort.unsafe.merger.UnsafeSingleThreadFinalSortFilesMerger;
+import org.apache.carbondata.processing.sortandgroupby.exception.CarbonSortKeyAndGroupByException;
+import org.apache.carbondata.processing.sortandgroupby.sortdata.SortParameters;
+import org.apache.carbondata.processing.store.writer.exception.CarbonDataWriterException;
+import org.apache.carbondata.processing.util.CarbonDataProcessorUtil;
+
+/**
+ * It parallely reads data from array of iterates and do merge sort.
+ * First it sorts the data and write to temp files. These temp files will be merge sorted to get
+ * final merge sort result.
+ */
+public class UnsafeParallelReadMergeSorterImpl implements Sorter {
+
+ private static final LogService LOGGER =
+ LogServiceFactory.getLogService(UnsafeParallelReadMergeSorterImpl.class.getName());
+
+ private SortParameters sortParameters;
+
+ private UnsafeIntermediateMerger unsafeIntermediateFileMerger;
+
+ private ExecutorService executorService;
+
+ private UnsafeSingleThreadFinalSortFilesMerger finalMerger;
+
+ private DataField[] inputDataFields;
+
+ public UnsafeParallelReadMergeSorterImpl(DataField[] inputDataFields) {
+ this.inputDataFields = inputDataFields;
+ }
+
+ @Override public void initialize(SortParameters sortParameters) {
+ this.sortParameters = sortParameters;
+ unsafeIntermediateFileMerger = new UnsafeIntermediateMerger(sortParameters);
+ String storeLocation = CarbonDataProcessorUtil
+ .getLocalDataFolderLocation(sortParameters.getDatabaseName(), sortParameters.getTableName(),
+ String.valueOf(sortParameters.getTaskNo()), sortParameters.getPartitionID(),
+ sortParameters.getSegmentId() + "", false);
+ // Set the data file location
+ String dataFolderLocation =
+ storeLocation + File.separator + CarbonCommonConstants.SORT_TEMP_FILE_LOCATION;
+ finalMerger = new UnsafeSingleThreadFinalSortFilesMerger(sortParameters);
+ }
+
+ @Override public Iterator<CarbonRowBatch>[] sort(Iterator<CarbonRowBatch>[] iterators)
+ throws CarbonDataLoadingException {
+ UnsafeSortDataRows sortDataRow =
+ new UnsafeSortDataRows(sortParameters, unsafeIntermediateFileMerger);
+ final int batchSize = CarbonProperties.getInstance().getBatchSize();
+ try {
+ sortDataRow.initialize();
+ } catch (CarbonSortKeyAndGroupByException e) {
+ throw new CarbonDataLoadingException(e);
+ }
+ this.executorService = Executors.newFixedThreadPool(iterators.length);
+
+ try {
+ for (int i = 0; i < iterators.length; i++) {
+ executorService
+ .submit(new SortIteratorThread(iterators[i], sortDataRow, sortParameters, batchSize));
+ }
+ executorService.shutdown();
+ executorService.awaitTermination(2, TimeUnit.DAYS);
+ processRowToNextStep(sortDataRow, sortParameters);
+ } catch (Exception e) {
+ throw new CarbonDataLoadingException("Problem while shutdown the server ", e);
+ }
+ try {
+ unsafeIntermediateFileMerger.finish();
+ List<UnsafeCarbonRowPage> rowPages = unsafeIntermediateFileMerger.getRowPages();
+ finalMerger.startFinalMerge(rowPages.toArray(new UnsafeCarbonRowPage[rowPages.size()]),
+ unsafeIntermediateFileMerger.getMergedPages());
+ } catch (CarbonDataWriterException e) {
+ throw new CarbonDataLoadingException(e);
+ } catch (CarbonSortKeyAndGroupByException e) {
+ throw new CarbonDataLoadingException(e);
+ }
+
+ // Creates the iterator to read from merge sorter.
+ Iterator<CarbonRowBatch> batchIterator = new CarbonIterator<CarbonRowBatch>() {
+
+ @Override public boolean hasNext() {
+ return finalMerger.hasNext();
+ }
+
+ @Override public CarbonRowBatch next() {
+ int counter = 0;
+ CarbonRowBatch rowBatch = new CarbonRowBatch();
+ while (finalMerger.hasNext() && counter < batchSize) {
+ rowBatch.addRow(new CarbonRow(finalMerger.next()));
+ counter++;
+ }
+ return rowBatch;
+ }
+ };
+ return new Iterator[] { batchIterator };
+ }
+
+ @Override public void close() {
+ unsafeIntermediateFileMerger.close();
+ finalMerger.clear();
+ }
+
+ /**
+ * Below method will be used to process data to next step
+ */
+ private boolean processRowToNextStep(UnsafeSortDataRows sortDataRows, SortParameters parameters)
+ throws CarbonDataLoadingException {
+ if (null == sortDataRows) {
+ LOGGER.info("Record Processed For table: " + parameters.getTableName());
+ LOGGER.info("Number of Records was Zero");
+ String logMessage = "Summary: Carbon Sort Key Step: Read: " + 0 + ": Write: " + 0;
+ LOGGER.info(logMessage);
+ return false;
+ }
+
+ try {
+ // start sorting
+ sortDataRows.startSorting();
+
+ // check any more rows are present
+ LOGGER.info("Record Processed For table: " + parameters.getTableName());
+ CarbonTimeStatisticsFactory.getLoadStatisticsInstance()
+ .recordSortRowsStepTotalTime(parameters.getPartitionID(), System.currentTimeMillis());
+ CarbonTimeStatisticsFactory.getLoadStatisticsInstance()
+ .recordDictionaryValuesTotalTime(parameters.getPartitionID(), System.currentTimeMillis());
+ return false;
+ } catch (CarbonSortKeyAndGroupByException e) {
+ throw new CarbonDataLoadingException(e);
+ }
+ }
+
+ /**
+ * This thread iterates the iterator and adds the rows
+ */
+ private static class SortIteratorThread implements Callable<Void> {
+
+ private Iterator<CarbonRowBatch> iterator;
+
+ private UnsafeSortDataRows sortDataRows;
+
+ private SortParameters parameters;
+
+ private Object[][] buffer;
+
+ public SortIteratorThread(Iterator<CarbonRowBatch> iterator, UnsafeSortDataRows sortDataRows,
+ SortParameters parameters, int batchSize) {
+ this.iterator = iterator;
+ this.sortDataRows = sortDataRows;
+ this.parameters = parameters;
+ this.buffer = new Object[batchSize][];
+ }
+
+ @Override public Void call() throws CarbonDataLoadingException {
+ try {
+ while (iterator.hasNext()) {
+ CarbonRowBatch batch = iterator.next();
+ Iterator<CarbonRow> batchIterator = batch.getBatchIterator();
+ int i = 0;
+ while (batchIterator.hasNext()) {
+ CarbonRow row = batchIterator.next();
+ if (row != null) {
+ buffer[i++] = row.getData();
+ }
+ }
+ if (i > 0) {
+ sortDataRows.addRowBatch(buffer, i);
+ }
+ }
+ } catch (Exception e) {
+ LOGGER.error(e);
+ throw new CarbonDataLoadingException(e);
+ }
+ return null;
+ }
+
+ }
+}
http://git-wip-us.apache.org/repos/asf/incubator-carbondata/blob/f1f9348d/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/IntPointerBuffer.java
----------------------------------------------------------------------
diff --git a/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/IntPointerBuffer.java b/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/IntPointerBuffer.java
new file mode 100644
index 0000000..94674cf
--- /dev/null
+++ b/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/IntPointerBuffer.java
@@ -0,0 +1,95 @@
+/*
+ * 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.processing.newflow.sort.unsafe;
+
+import org.apache.carbondata.core.memory.MemoryBlock;
+
+/**
+ * Holds the pointers for rows.
+ */
+public class IntPointerBuffer {
+
+ private int length;
+
+ private int actualSize;
+
+
+ private int[] pointerBlock;
+
+ private MemoryBlock baseBlock;
+
+ public IntPointerBuffer(MemoryBlock baseBlock) {
+ // TODO can be configurable, it is initial size and it can grow automatically.
+ this.length = 100000;
+ pointerBlock = new int[length];
+ this.baseBlock = baseBlock;
+ }
+
+ public IntPointerBuffer(int length) {
+ this.length = length;
+ pointerBlock = new int[length];
+ }
+
+ public void set(int index, int value) {
+ pointerBlock[index] = value;
+ }
+
+ public void set(int value) {
+ ensureMemory();
+ pointerBlock[actualSize] = value;
+ actualSize++;
+ }
+
+ /**
+ * Returns the value at position {@code index}.
+ */
+ public int get(int index) {
+ assert index >= 0 : "index (" + index + ") should >= 0";
+ assert index < length : "index (" + index + ") should < length (" + length + ")";
+ return pointerBlock[index];
+ }
+
+ public int getActualSize() {
+ return actualSize;
+ }
+
+ public MemoryBlock getBaseBlock() {
+ return baseBlock;
+ }
+
+ public int[] getPointerBlock() {
+ return pointerBlock;
+ }
+
+ private void ensureMemory() {
+ if (actualSize >= length) {
+ // Expand by quarter, may be we can correct the logic later
+ int localLength = length + (int) (length * (0.25));
+ int[] memoryAddress = new int[localLength];
+ System.arraycopy(pointerBlock, 0, memoryAddress, 0, length);
+ pointerBlock = memoryAddress;
+ length = localLength;
+ }
+ }
+
+ public void freeMemory() {
+ pointerBlock = null;
+ if (baseBlock != null) {
+ UnsafeMemoryManager.INSTANCE.freeMemory(baseBlock);
+ }
+ }
+}
http://git-wip-us.apache.org/repos/asf/incubator-carbondata/blob/f1f9348d/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/UnsafeCarbonRowPage.java
----------------------------------------------------------------------
diff --git a/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/UnsafeCarbonRowPage.java b/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/UnsafeCarbonRowPage.java
new file mode 100644
index 0000000..7296e74
--- /dev/null
+++ b/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/UnsafeCarbonRowPage.java
@@ -0,0 +1,356 @@
+/*
+ * 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.processing.newflow.sort.unsafe;
+
+import java.io.DataOutputStream;
+import java.io.IOException;
+import java.math.BigDecimal;
+import java.util.Arrays;
+import java.util.Iterator;
+
+import org.apache.carbondata.common.CarbonIterator;
+import org.apache.carbondata.core.constants.CarbonCommonConstants;
+import org.apache.carbondata.core.memory.MemoryBlock;
+import org.apache.carbondata.core.unsafe.CarbonUnsafe;
+import org.apache.carbondata.core.util.DataTypeUtil;
+
+/**
+ * It can keep the data of prescribed size data in offheap/onheap memory and returns it when needed
+ */
+public class UnsafeCarbonRowPage {
+
+ private boolean[] noDictionaryDimensionMapping;
+
+ private int dimensionSize;
+
+ private int measureSize;
+
+ private char[] aggType;
+
+ private long[] nullSetWords;
+
+ private IntPointerBuffer buffer;
+
+ private int lastSize;
+
+ private long sizeToBeUsed;
+
+ private MemoryBlock dataBlock;
+
+ private boolean saveToDisk;
+
+ public UnsafeCarbonRowPage(boolean[] noDictionaryDimensionMapping, int dimensionSize,
+ int measureSize, char[] aggType, MemoryBlock memoryBlock, boolean saveToDisk) {
+ this.noDictionaryDimensionMapping = noDictionaryDimensionMapping;
+ this.dimensionSize = dimensionSize;
+ this.measureSize = measureSize;
+ this.aggType = aggType;
+ this.saveToDisk = saveToDisk;
+ this.nullSetWords = new long[((measureSize - 1) >> 6) + 1];
+ buffer = new IntPointerBuffer(memoryBlock);
+ this.dataBlock = buffer.getBaseBlock();
+ // TODO Only using 98% of space for safe side.May be we can have different logic.
+ sizeToBeUsed = dataBlock.size() - (dataBlock.size() * 5) / 100;
+ }
+
+ public void addRow(Object[] row) {
+ int size = addRow(row, dataBlock.getBaseOffset() + lastSize);
+ buffer.set(lastSize);
+ lastSize = lastSize + size;
+ }
+
+ public Iterator<Object[]> getIterator() {
+ return new UnsafeIterator();
+ }
+
+ private int addRow(Object[] row, long address) {
+ if (row == null) {
+ throw new RuntimeException("Row is null ??");
+ }
+ int dimCount = 0;
+ int size = 0;
+ Object baseObject = dataBlock.getBaseObject();
+ for (; dimCount < noDictionaryDimensionMapping.length; dimCount++) {
+ if (noDictionaryDimensionMapping[dimCount]) {
+ byte[] col = (byte[]) row[dimCount];
+ CarbonUnsafe.unsafe
+ .putShort(baseObject, address + size, (short) col.length);
+ size += 2;
+ CarbonUnsafe.unsafe.copyMemory(col, CarbonUnsafe.BYTE_ARRAY_OFFSET, baseObject,
+ address + size, col.length);
+ size += col.length;
+ } else {
+ int value = (int) row[dimCount];
+ CarbonUnsafe.unsafe.putInt(baseObject, address + size, value);
+ size += 4;
+ }
+ }
+
+ // write complex dimensions here.
+ for (; dimCount < dimensionSize; dimCount++) {
+ byte[] col = (byte[]) row[dimCount];
+ CarbonUnsafe.unsafe.putShort(baseObject, address + size, (short) col.length);
+ size += 2;
+ CarbonUnsafe.unsafe.copyMemory(col, CarbonUnsafe.BYTE_ARRAY_OFFSET, baseObject,
+ address + size, col.length);
+ size += col.length;
+ }
+ Arrays.fill(nullSetWords, 0);
+ int nullSetSize = nullSetWords.length * 8;
+ int nullWordLoc = size;
+ size += nullSetSize;
+ for (int mesCount = 0; mesCount < measureSize; mesCount++) {
+ Object value = row[mesCount + dimensionSize];
+ if (null != value) {
+ if (aggType[mesCount] == CarbonCommonConstants.SUM_COUNT_VALUE_MEASURE) {
+ Double val = (Double) value;
+ CarbonUnsafe.unsafe.putDouble(baseObject, address + size, val);
+ size += 8;
+ } else if (aggType[mesCount] == CarbonCommonConstants.BIG_INT_MEASURE) {
+ Long val = (Long) value;
+ CarbonUnsafe.unsafe.putLong(baseObject, address + size, val);
+ size += 8;
+ } else if (aggType[mesCount] == CarbonCommonConstants.BIG_DECIMAL_MEASURE) {
+ BigDecimal val = (BigDecimal) value;
+ byte[] bigDecimalInBytes = DataTypeUtil.bigDecimalToByte(val);
+ CarbonUnsafe.unsafe.putShort(baseObject, address + size,
+ (short) bigDecimalInBytes.length);
+ size += 2;
+ CarbonUnsafe.unsafe
+ .copyMemory(bigDecimalInBytes, CarbonUnsafe.BYTE_ARRAY_OFFSET, baseObject,
+ address + size, bigDecimalInBytes.length);
+ size += bigDecimalInBytes.length;
+ }
+ set(nullSetWords, mesCount);
+ } else {
+ unset(nullSetWords, mesCount);
+ }
+ }
+ CarbonUnsafe.unsafe.copyMemory(nullSetWords, CarbonUnsafe.LONG_ARRAY_OFFSET, baseObject,
+ address + nullWordLoc, nullSetSize);
+ return size;
+ }
+
+ public Object[] getRow(long address, Object[] rowToFill) {
+ int dimCount = 0;
+ int size = 0;
+
+ Object baseObject = dataBlock.getBaseObject();
+ for (; dimCount < noDictionaryDimensionMapping.length; dimCount++) {
+ if (noDictionaryDimensionMapping[dimCount]) {
+ short aShort = CarbonUnsafe.unsafe.getShort(baseObject, address + size);
+ byte[] col = new byte[aShort];
+ size += 2;
+ CarbonUnsafe.unsafe
+ .copyMemory(baseObject, address + size, col, CarbonUnsafe.BYTE_ARRAY_OFFSET,
+ col.length);
+ size += col.length;
+ rowToFill[dimCount] = col;
+ } else {
+ int anInt = CarbonUnsafe.unsafe.getInt(baseObject, address + size);
+ size += 4;
+ rowToFill[dimCount] = anInt;
+ }
+ }
+
+ // write complex dimensions here.
+ for (; dimCount < dimensionSize; dimCount++) {
+ short aShort = CarbonUnsafe.unsafe.getShort(baseObject, address + size);
+ byte[] col = new byte[aShort];
+ size += 2;
+ CarbonUnsafe.unsafe
+ .copyMemory(baseObject, address + size, col, CarbonUnsafe.BYTE_ARRAY_OFFSET, col.length);
+ size += col.length;
+ rowToFill[dimCount] = col;
+ }
+
+ int nullSetSize = nullSetWords.length * 8;
+ Arrays.fill(nullSetWords, 0);
+ CarbonUnsafe.unsafe
+ .copyMemory(baseObject, address + size, nullSetWords, CarbonUnsafe.LONG_ARRAY_OFFSET,
+ nullSetSize);
+ size += nullSetSize;
+
+ for (int mesCount = 0; mesCount < measureSize; mesCount++) {
+ if (isSet(nullSetWords, mesCount)) {
+ if (aggType[mesCount] == CarbonCommonConstants.SUM_COUNT_VALUE_MEASURE) {
+ Double val = CarbonUnsafe.unsafe.getDouble(baseObject, address + size);
+ size += 8;
+ rowToFill[dimensionSize + mesCount] = val;
+ } else if (aggType[mesCount] == CarbonCommonConstants.BIG_INT_MEASURE) {
+ Long val = CarbonUnsafe.unsafe.getLong(baseObject, address + size);
+ size += 8;
+ rowToFill[dimensionSize + mesCount] = val;
+ } else if (aggType[mesCount] == CarbonCommonConstants.BIG_DECIMAL_MEASURE) {
+ short aShort = CarbonUnsafe.unsafe.getShort(baseObject, address + size);
+ byte[] bigDecimalInBytes = new byte[aShort];
+ size += 2;
+ CarbonUnsafe.unsafe.copyMemory(baseObject, address + size, bigDecimalInBytes,
+ CarbonUnsafe.BYTE_ARRAY_OFFSET, bigDecimalInBytes.length);
+ BigDecimal val = DataTypeUtil.byteToBigDecimal(bigDecimalInBytes);
+ size += bigDecimalInBytes.length;
+ rowToFill[dimensionSize + mesCount] = val;
+ }
+ } else {
+ rowToFill[dimensionSize + mesCount] = null;
+ }
+ }
+ return rowToFill;
+ }
+
+ public void fillRow(long address, DataOutputStream stream) throws IOException {
+ int dimCount = 0;
+ int size = 0;
+
+ Object baseObject = dataBlock.getBaseObject();
+ for (; dimCount < noDictionaryDimensionMapping.length; dimCount++) {
+ if (noDictionaryDimensionMapping[dimCount]) {
+ short aShort = CarbonUnsafe.unsafe.getShort(baseObject, address + size);
+ byte[] col = new byte[aShort];
+ size += 2;
+ CarbonUnsafe.unsafe
+ .copyMemory(baseObject, address + size, col, CarbonUnsafe.BYTE_ARRAY_OFFSET,
+ col.length);
+ size += col.length;
+ stream.writeShort(aShort);
+ stream.write(col);
+ } else {
+ int anInt = CarbonUnsafe.unsafe.getInt(baseObject, address + size);
+ size += 4;
+ stream.writeInt(anInt);
+ }
+ }
+
+ // write complex dimensions here.
+ for (; dimCount < dimensionSize; dimCount++) {
+ short aShort = CarbonUnsafe.unsafe.getShort(baseObject, address + size);
+ byte[] col = new byte[aShort];
+ size += 2;
+ CarbonUnsafe.unsafe
+ .copyMemory(baseObject, address + size, col, CarbonUnsafe.BYTE_ARRAY_OFFSET, col.length);
+ size += col.length;
+ stream.writeShort(aShort);
+ stream.write(col);
+ }
+
+ int nullSetSize = nullSetWords.length * 8;
+ Arrays.fill(nullSetWords, 0);
+ CarbonUnsafe.unsafe
+ .copyMemory(baseObject, address + size, nullSetWords, CarbonUnsafe.LONG_ARRAY_OFFSET,
+ nullSetSize);
+ size += nullSetSize;
+ for (int i = 0; i < nullSetWords.length; i++) {
+ stream.writeLong(nullSetWords[i]);
+ }
+
+ for (int mesCount = 0; mesCount < measureSize; mesCount++) {
+ if (isSet(nullSetWords, mesCount)) {
+ if (aggType[mesCount] == CarbonCommonConstants.SUM_COUNT_VALUE_MEASURE) {
+ double val = CarbonUnsafe.unsafe.getDouble(baseObject, address + size);
+ size += 8;
+ stream.writeDouble(val);
+ } else if (aggType[mesCount] == CarbonCommonConstants.BIG_INT_MEASURE) {
+ long val = CarbonUnsafe.unsafe.getLong(baseObject, address + size);
+ size += 8;
+ stream.writeLong(val);
+ } else if (aggType[mesCount] == CarbonCommonConstants.BIG_DECIMAL_MEASURE) {
+ short aShort = CarbonUnsafe.unsafe.getShort(baseObject, address + size);
+ byte[] bigDecimalInBytes = new byte[aShort];
+ size += 2;
+ CarbonUnsafe.unsafe.copyMemory(baseObject, address + size, bigDecimalInBytes,
+ CarbonUnsafe.BYTE_ARRAY_OFFSET, bigDecimalInBytes.length);
+ size += bigDecimalInBytes.length;
+ stream.writeShort(aShort);
+ stream.write(bigDecimalInBytes);
+ }
+ }
+ }
+ }
+
+ private Object[] getRow(long address) {
+ Object[] row = new Object[dimensionSize + measureSize];
+ return getRow(address, row);
+ }
+
+ public void freeMemory() {
+ buffer.freeMemory();
+ }
+
+ public boolean isSaveToDisk() {
+ return saveToDisk;
+ }
+
+ public IntPointerBuffer getBuffer() {
+ return buffer;
+ }
+
+ public int getUsedSize() {
+ return lastSize;
+ }
+
+ public boolean canAdd() {
+ return lastSize < sizeToBeUsed;
+ }
+
+ public MemoryBlock getDataBlock() {
+ return dataBlock;
+ }
+
+ class UnsafeIterator extends CarbonIterator<Object[]> {
+
+ private int counter;
+
+ private int actualSize;
+
+ public UnsafeIterator() {
+ this.actualSize = buffer.getActualSize();
+ }
+
+ @Override public boolean hasNext() {
+ if (counter < actualSize) {
+ return true;
+ }
+ return false;
+ }
+
+ @Override public Object[] next() {
+ long address = buffer.get(counter);
+ counter++;
+ return getRow(address + dataBlock.getBaseOffset());
+ }
+ }
+
+ public static void set(long[] words, int index) {
+ int wordOffset = (index >> 6);
+ words[wordOffset] |= (1L << index);
+ }
+
+ public static void unset(long[] words, int index) {
+ int wordOffset = (index >> 6);
+ words[wordOffset] &= ~(1L << index);
+ }
+
+ public static boolean isSet(long[] words, int index) {
+ int wordOffset = (index >> 6);
+ return ((words[wordOffset] & (1L << index)) != 0);
+ }
+
+ public boolean[] getNoDictionaryDimensionMapping() {
+ return noDictionaryDimensionMapping;
+ }
+
+}
http://git-wip-us.apache.org/repos/asf/incubator-carbondata/blob/f1f9348d/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/UnsafeMemoryManager.java
----------------------------------------------------------------------
diff --git a/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/UnsafeMemoryManager.java b/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/UnsafeMemoryManager.java
new file mode 100644
index 0000000..f844fb5
--- /dev/null
+++ b/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/UnsafeMemoryManager.java
@@ -0,0 +1,99 @@
+/*
+ * 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.processing.newflow.sort.unsafe;
+
+import org.apache.carbondata.common.logging.LogService;
+import org.apache.carbondata.common.logging.LogServiceFactory;
+import org.apache.carbondata.core.constants.CarbonCommonConstants;
+import org.apache.carbondata.core.memory.MemoryAllocator;
+import org.apache.carbondata.core.memory.MemoryBlock;
+import org.apache.carbondata.core.util.CarbonProperties;
+
+/**
+ * Manages memory for instance.
+ */
+public class UnsafeMemoryManager {
+
+ private static final LogService LOGGER =
+ LogServiceFactory.getLogService(UnsafeMemoryManager.class.getName());
+
+ static {
+ long size = Long.parseLong(CarbonProperties.getInstance()
+ .getProperty(CarbonCommonConstants.IN_MEMORY_FOR_SORT_DATA_IN_MB,
+ CarbonCommonConstants.IN_MEMORY_FOR_SORT_DATA_IN_MB_DEFAULT));
+
+ boolean offHeap = Boolean.parseBoolean(CarbonProperties.getInstance()
+ .getProperty(CarbonCommonConstants.ENABLE_OFFHEAP_SORT,
+ CarbonCommonConstants.ENABLE_OFFHEAP_SORT_DEFAULT));
+ long takenSize = size * 1024 * 1024;
+ MemoryAllocator allocator;
+ if (offHeap) {
+ allocator = MemoryAllocator.UNSAFE;
+ } else {
+ long maxMemory = Runtime.getRuntime().maxMemory()*60/100;
+ if (takenSize > maxMemory) {
+ takenSize = maxMemory;
+ }
+ allocator = MemoryAllocator.HEAP;
+ }
+ INSTANCE = new UnsafeMemoryManager(takenSize, allocator);
+ }
+
+ public static final UnsafeMemoryManager INSTANCE;
+
+ private long totalMemory;
+
+ private long memoryUsed;
+
+ private MemoryAllocator allocator;
+
+ private long minimumMemory;
+
+ private UnsafeMemoryManager(long totalMemory, MemoryAllocator allocator) {
+ this.totalMemory = totalMemory;
+ this.allocator = allocator;
+ minimumMemory = (long) (totalMemory * ((double)10/100));
+ LOGGER.audit("Memory manager is created with size " + totalMemory
+ +" with "+allocator +" and minimum reserve memory "+minimumMemory);
+ }
+ public synchronized MemoryBlock allocateMemory(long memoryRequested) {
+ if (memoryUsed + memoryRequested <= totalMemory) {
+ MemoryBlock allocate = allocator.allocate(memoryRequested);
+ memoryUsed += allocate.size();
+ LOGGER.audit("Memory block is created with size " + allocate.size() +
+ " Total memory used " + memoryUsed + " memory left " + (getAvailableMemory()));
+ return allocate;
+ }
+ return null;
+ }
+
+ public synchronized void freeMemory(MemoryBlock memoryBlock) {
+ allocator.free(memoryBlock);
+ memoryUsed -= memoryBlock.size();
+ memoryUsed = memoryUsed < 0 ? 0 : memoryUsed;
+ LOGGER.audit("Memory released, memory used "+ memoryUsed
+ + " memory left "+(getAvailableMemory()));
+ }
+
+ public synchronized long getAvailableMemory() {
+ return totalMemory - memoryUsed;
+ }
+
+ public boolean isMemoryAvailable() {
+ return getAvailableMemory() > minimumMemory;
+ }
+}
http://git-wip-us.apache.org/repos/asf/incubator-carbondata/blob/f1f9348d/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/UnsafeSortDataRows.java
----------------------------------------------------------------------
diff --git a/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/UnsafeSortDataRows.java b/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/UnsafeSortDataRows.java
new file mode 100644
index 0000000..9d73ba2
--- /dev/null
+++ b/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/UnsafeSortDataRows.java
@@ -0,0 +1,356 @@
+/*
+ * 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.processing.newflow.sort.unsafe;
+
+import java.io.BufferedOutputStream;
+import java.io.DataOutputStream;
+import java.io.File;
+import java.io.FileOutputStream;
+import java.io.IOException;
+import java.util.concurrent.Callable;
+import java.util.concurrent.ExecutorService;
+import java.util.concurrent.Executors;
+import java.util.concurrent.TimeUnit;
+
+import org.apache.carbondata.common.logging.LogService;
+import org.apache.carbondata.common.logging.LogServiceFactory;
+import org.apache.carbondata.core.constants.CarbonCommonConstants;
+import org.apache.carbondata.core.memory.MemoryBlock;
+import org.apache.carbondata.core.util.CarbonProperties;
+import org.apache.carbondata.core.util.CarbonUtil;
+import org.apache.carbondata.processing.newflow.sort.unsafe.comparator.UnsafeRowComparator;
+import org.apache.carbondata.processing.newflow.sort.unsafe.comparator.UnsafeRowComparatorForNormalDIms;
+import org.apache.carbondata.processing.newflow.sort.unsafe.holder.UnsafeCarbonRow;
+import org.apache.carbondata.processing.newflow.sort.unsafe.merger.UnsafeIntermediateMerger;
+import org.apache.carbondata.processing.newflow.sort.unsafe.sort.TimSort;
+import org.apache.carbondata.processing.newflow.sort.unsafe.sort.UnsafeIntSortDataFormat;
+import org.apache.carbondata.processing.sortandgroupby.exception.CarbonSortKeyAndGroupByException;
+import org.apache.carbondata.processing.sortandgroupby.sortdata.SortParameters;
+import org.apache.carbondata.processing.sortandgroupby.sortdata.SortTempFileChunkWriter;
+import org.apache.carbondata.processing.sortandgroupby.sortdata.TempSortFileWriter;
+import org.apache.carbondata.processing.sortandgroupby.sortdata.TempSortFileWriterFactory;
+import org.apache.carbondata.processing.util.CarbonDataProcessorUtil;
+
+public class UnsafeSortDataRows {
+ /**
+ * LOGGER
+ */
+ private static final LogService LOGGER =
+ LogServiceFactory.getLogService(UnsafeSortDataRows.class.getName());
+ /**
+ * threadStatusObserver
+ */
+ private ThreadStatusObserver threadStatusObserver;
+ /**
+ * executor service for data sort holder
+ */
+ private ExecutorService dataSorterAndWriterExecutorService;
+ /**
+ * semaphore which will used for managing sorted data object arrays
+ */
+
+ private SortParameters parameters;
+
+ private UnsafeIntermediateMerger unsafeInMemoryIntermediateFileMerger;
+
+ private UnsafeCarbonRowPage rowPage;
+
+ private final Object addRowsLock = new Object();
+
+ private int inMemoryChunkSizeInMB;
+
+ private boolean enableInMemoryIntermediateMerge;
+
+ public UnsafeSortDataRows(SortParameters parameters,
+ UnsafeIntermediateMerger unsafeInMemoryIntermediateFileMerger) {
+ this.parameters = parameters;
+
+ this.unsafeInMemoryIntermediateFileMerger = unsafeInMemoryIntermediateFileMerger;
+
+ // observer of writing file in thread
+ this.threadStatusObserver = new ThreadStatusObserver();
+
+ this.inMemoryChunkSizeInMB = Integer.parseInt(CarbonProperties.getInstance()
+ .getProperty(CarbonCommonConstants.OFFHEAP_SORT_CHUNK_SIZE_IN_MB,
+ CarbonCommonConstants.OFFHEAP_SORT_CHUNK_SIZE_IN_MB_DEFAULT));
+ enableInMemoryIntermediateMerge = Boolean.parseBoolean(CarbonProperties.getInstance()
+ .getProperty(CarbonCommonConstants.ENABLE_INMEMORY_MERGE_SORT,
+ CarbonCommonConstants.ENABLE_INMEMORY_MERGE_SORT_DEFAULT));
+ }
+
+ /**
+ * This method will be used to initialize
+ */
+ public void initialize() throws CarbonSortKeyAndGroupByException {
+ MemoryBlock baseBlock = getMemoryBlock(inMemoryChunkSizeInMB * 1024 * 1024);
+ this.rowPage = new UnsafeCarbonRowPage(parameters.getNoDictionaryDimnesionColumn(),
+ parameters.getDimColCount(), parameters.getMeasureColCount(), parameters.getAggType(),
+ baseBlock, !UnsafeMemoryManager.INSTANCE.isMemoryAvailable());
+ // Delete if any older file exists in sort temp folder
+ deleteSortLocationIfExists();
+
+ // create new sort temp directory
+ if (!new File(parameters.getTempFileLocation()).mkdirs()) {
+ LOGGER.info("Sort Temp Location Already Exists");
+ }
+ this.dataSorterAndWriterExecutorService =
+ Executors.newFixedThreadPool(parameters.getNumberOfCores());
+ }
+
+ public static MemoryBlock getMemoryBlock(long size) throws CarbonSortKeyAndGroupByException {
+ MemoryBlock baseBlock = null;
+ int tries = 0;
+ while (true && tries < 100) {
+ baseBlock = UnsafeMemoryManager.INSTANCE.allocateMemory(size);
+ if (baseBlock == null) {
+ try {
+ Thread.sleep(50);
+ } catch (InterruptedException e) {
+ throw new CarbonSortKeyAndGroupByException(e);
+ }
+ } else {
+ break;
+ }
+ tries++;
+ }
+ if (baseBlock == null) {
+ throw new CarbonSortKeyAndGroupByException("Not enough memory to create page");
+ }
+ return baseBlock;
+ }
+
+ /**
+ * This method will be used to add new row
+ *
+ * @param rowBatch new rowBatch
+ * @throws CarbonSortKeyAndGroupByException problem while writing
+ */
+ public void addRowBatch(Object[][] rowBatch, int size) throws CarbonSortKeyAndGroupByException {
+ // if record holder list size is equal to sort buffer size then it will
+ // sort the list and then write current list data to file
+ synchronized (addRowsLock) {
+ for (int i = 0; i < size; i++) {
+ if (rowPage.canAdd()) {
+ rowPage.addRow(rowBatch[i]);
+ } else {
+ try {
+ if (enableInMemoryIntermediateMerge) {
+ unsafeInMemoryIntermediateFileMerger.startInmemoryMergingIfPossible();
+ }
+ unsafeInMemoryIntermediateFileMerger.startFileMergingIfPossible();
+ dataSorterAndWriterExecutorService.submit(new DataSorterAndWriter(rowPage));
+ MemoryBlock memoryBlock = getMemoryBlock(inMemoryChunkSizeInMB * 1024 * 1024);
+ boolean saveToDisk = !UnsafeMemoryManager.INSTANCE.isMemoryAvailable();
+ rowPage = new UnsafeCarbonRowPage(parameters.getNoDictionaryDimnesionColumn(),
+ parameters.getDimColCount(), parameters.getMeasureColCount(),
+ parameters.getAggType(), memoryBlock,
+ saveToDisk);
+ rowPage.addRow(rowBatch[i]);
+ } catch (Exception e) {
+ LOGGER.error(
+ "exception occurred while trying to acquire a semaphore lock: " + e.getMessage());
+ throw new CarbonSortKeyAndGroupByException(e);
+ }
+
+ }
+ }
+ }
+ }
+
+ /**
+ * Below method will be used to start storing process This method will get
+ * all the temp files present in sort temp folder then it will create the
+ * record holder heap and then it will read first record from each file and
+ * initialize the heap
+ *
+ * @throws CarbonSortKeyAndGroupByException
+ */
+ public void startSorting() throws CarbonSortKeyAndGroupByException {
+ LOGGER.info("Unsafe based sorting will be used");
+ if (this.rowPage.getUsedSize() > 0) {
+ TimSort<UnsafeCarbonRow, IntPointerBuffer> timSort = new TimSort<>(
+ new UnsafeIntSortDataFormat(rowPage));
+ if (parameters.getNoDictionaryCount() > 0) {
+ timSort.sort(rowPage.getBuffer(), 0, rowPage.getBuffer().getActualSize(),
+ new UnsafeRowComparator(rowPage));
+ } else {
+ timSort.sort(rowPage.getBuffer(), 0, rowPage.getBuffer().getActualSize(),
+ new UnsafeRowComparatorForNormalDIms(parameters.getDimColCount(), rowPage));
+ }
+ unsafeInMemoryIntermediateFileMerger.addDataChunkToMerge(rowPage);
+ } else {
+ rowPage.freeMemory();
+ }
+ startFileBasedMerge();
+ }
+
+ private void writeSortTempFile(Object[][] recordHolderList, int entryCountLocal, File file)
+ throws CarbonSortKeyAndGroupByException {
+ TempSortFileWriter writer = null;
+
+ try {
+ writer = getWriter();
+ writer.initiaize(file, entryCountLocal);
+ writer.writeSortTempFile(recordHolderList);
+ } catch (CarbonSortKeyAndGroupByException e) {
+ LOGGER.error(e, "Problem while writing the sort temp file");
+ throw e;
+ } finally {
+ if (writer != null) {
+ writer.finish();
+ }
+ }
+ }
+
+ private void writeData(UnsafeCarbonRowPage rowPage, File file)
+ throws CarbonSortKeyAndGroupByException {
+ DataOutputStream stream = null;
+ try {
+ // open stream
+ stream = new DataOutputStream(new BufferedOutputStream(new FileOutputStream(file),
+ parameters.getFileWriteBufferSize()));
+ int actualSize = rowPage.getBuffer().getActualSize();
+ // write number of entries to the file
+ stream.writeInt(actualSize);
+ for (int i = 0; i < actualSize; i++) {
+ rowPage.fillRow(rowPage.getBuffer().get(i) + rowPage.getDataBlock().getBaseOffset(),
+ stream);
+ }
+
+ } catch (IOException e) {
+ throw new CarbonSortKeyAndGroupByException("Problem while writing the file", e);
+ } finally {
+ // close streams
+ CarbonUtil.closeStreams(stream);
+ }
+ }
+
+ private TempSortFileWriter getWriter() {
+ TempSortFileWriter chunkWriter = null;
+ TempSortFileWriter writer = TempSortFileWriterFactory.getInstance()
+ .getTempSortFileWriter(parameters.isSortFileCompressionEnabled(),
+ parameters.getDimColCount(), parameters.getComplexDimColCount(),
+ parameters.getMeasureColCount(), parameters.getNoDictionaryCount(),
+ parameters.getFileWriteBufferSize());
+
+ if (parameters.isPrefetch() && !parameters.isSortFileCompressionEnabled()) {
+ chunkWriter = new SortTempFileChunkWriter(writer, parameters.getBufferSize());
+ } else {
+ chunkWriter =
+ new SortTempFileChunkWriter(writer, parameters.getSortTempFileNoOFRecordsInCompression());
+ }
+
+ return chunkWriter;
+ }
+
+ /**
+ * This method will be used to delete sort temp location is it is exites
+ *
+ * @throws CarbonSortKeyAndGroupByException
+ */
+ public void deleteSortLocationIfExists() throws CarbonSortKeyAndGroupByException {
+ CarbonDataProcessorUtil.deleteSortLocationIfExists(parameters.getTempFileLocation());
+ }
+
+ /**
+ * Below method will be used to start file based merge
+ *
+ * @throws CarbonSortKeyAndGroupByException
+ */
+ private void startFileBasedMerge() throws CarbonSortKeyAndGroupByException {
+ try {
+ dataSorterAndWriterExecutorService.shutdown();
+ dataSorterAndWriterExecutorService.awaitTermination(2, TimeUnit.DAYS);
+ } catch (InterruptedException e) {
+ throw new CarbonSortKeyAndGroupByException("Problem while shutdown the server ", e);
+ }
+ }
+
+ /**
+ * Observer class for thread execution
+ * In case of any failure we need stop all the running thread
+ */
+ private class ThreadStatusObserver {
+ /**
+ * Below method will be called if any thread fails during execution
+ *
+ * @param exception
+ * @throws CarbonSortKeyAndGroupByException
+ */
+ public void notifyFailed(Throwable exception) throws CarbonSortKeyAndGroupByException {
+ dataSorterAndWriterExecutorService.shutdownNow();
+ unsafeInMemoryIntermediateFileMerger.close();
+ parameters.getObserver().setFailed(true);
+ LOGGER.error(exception);
+ throw new CarbonSortKeyAndGroupByException(exception);
+ }
+ }
+
+ /**
+ * This class is responsible for sorting and writing the object
+ * array which holds the records equal to given array size
+ */
+ private class DataSorterAndWriter implements Callable<Void> {
+ private UnsafeCarbonRowPage page;
+
+ public DataSorterAndWriter(UnsafeCarbonRowPage rowPage) {
+ this.page = rowPage;
+ }
+
+ @Override public Void call() throws Exception {
+ try {
+ long startTime = System.currentTimeMillis();
+ TimSort<UnsafeCarbonRow, IntPointerBuffer> timSort = new TimSort<>(
+ new UnsafeIntSortDataFormat(page));
+ if (parameters.getNoDictionaryCount() > 0) {
+ timSort.sort(page.getBuffer(), 0, page.getBuffer().getActualSize(),
+ new UnsafeRowComparator(page));
+ } else {
+ timSort.sort(page.getBuffer(), 0, page.getBuffer().getActualSize(),
+ new UnsafeRowComparatorForNormalDIms(parameters.getDimColCount(), page));
+ }
+ if (rowPage.isSaveToDisk()) {
+ // create a new file every time
+ File sortTempFile = new File(
+ parameters.getTempFileLocation() + File.separator + parameters.getTableName() + System
+ .nanoTime() + CarbonCommonConstants.SORT_TEMP_FILE_EXT);
+ writeData(page, sortTempFile);
+ LOGGER.info("Time taken to sort row page with size" + page.getBuffer().getActualSize()
+ + " and write is: " + (System.currentTimeMillis() - startTime));
+ page.freeMemory();
+ // add sort temp filename to and arrayList. When the list size reaches 20 then
+ // intermediate merging of sort temp files will be triggered
+ unsafeInMemoryIntermediateFileMerger.addFileToMerge(sortTempFile);
+ } else {
+ // add sort temp filename to and arrayList. When the list size reaches 20 then
+ // intermediate merging of sort temp files will be triggered
+ unsafeInMemoryIntermediateFileMerger.addDataChunkToMerge(page);
+ LOGGER.info(
+ "Time taken to sort row page with size" + page.getBuffer().getActualSize() + "is: "
+ + (System.currentTimeMillis() - startTime));
+ }
+ } catch (Throwable e) {
+ threadStatusObserver.notifyFailed(e);
+ }
+ return null;
+ }
+ }
+}
+
http://git-wip-us.apache.org/repos/asf/incubator-carbondata/blob/f1f9348d/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/comparator/UnsafeRowComparator.java
----------------------------------------------------------------------
diff --git a/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/comparator/UnsafeRowComparator.java b/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/comparator/UnsafeRowComparator.java
new file mode 100644
index 0000000..9ab2bdd
--- /dev/null
+++ b/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/comparator/UnsafeRowComparator.java
@@ -0,0 +1,133 @@
+/*
+ * 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.processing.newflow.sort.unsafe.comparator;
+
+import java.util.Comparator;
+
+import org.apache.carbondata.core.unsafe.CarbonUnsafe;
+import org.apache.carbondata.core.util.ByteUtil.UnsafeComparer;
+import org.apache.carbondata.processing.newflow.sort.unsafe.UnsafeCarbonRowPage;
+import org.apache.carbondata.processing.newflow.sort.unsafe.holder.UnsafeCarbonRow;
+
+public class UnsafeRowComparator implements Comparator<UnsafeCarbonRow> {
+
+ /**
+ * noDictionaryColMaping mapping of dictionary dimensions and no dictionary dimensions.
+ */
+ private boolean[] noDictionaryColMaping;
+
+ private Object baseObject;
+
+ public UnsafeRowComparator(UnsafeCarbonRowPage rowPage) {
+ this.noDictionaryColMaping = rowPage.getNoDictionaryDimensionMapping();
+ this.baseObject = rowPage.getDataBlock().getBaseObject();
+ }
+
+ /**
+ * Below method will be used to compare two mdkey
+ */
+ public int compare(UnsafeCarbonRow rowL, UnsafeCarbonRow rowR) {
+ int diff = 0;
+ long rowA = rowL.address;
+ long rowB = rowR.address;
+ int sizeA = 0;
+ int sizeB = 0;
+ for (boolean isNoDictionary : noDictionaryColMaping) {
+ if (isNoDictionary) {
+ short aShort1 = CarbonUnsafe.unsafe.getShort(baseObject, rowA + sizeA);
+ byte[] byteArr1 = new byte[aShort1];
+ sizeA += 2;
+ CarbonUnsafe.unsafe.copyMemory(baseObject, rowA + sizeA, byteArr1,
+ CarbonUnsafe.BYTE_ARRAY_OFFSET, aShort1);
+ sizeA += aShort1;
+
+ short aShort2 = CarbonUnsafe.unsafe.getShort(baseObject, rowB + sizeB);
+ byte[] byteArr2 = new byte[aShort2];
+ sizeB += 2;
+ CarbonUnsafe.unsafe.copyMemory(baseObject, rowB + sizeB, byteArr2,
+ CarbonUnsafe.BYTE_ARRAY_OFFSET, aShort2);
+ sizeB += aShort2;
+
+ int difference = UnsafeComparer.INSTANCE.compareTo(byteArr1, byteArr2);
+ if (difference != 0) {
+ return difference;
+ }
+ } else {
+ int dimFieldA = CarbonUnsafe.unsafe.getInt(baseObject, rowA + sizeA);
+ sizeA += 4;
+ int dimFieldB = CarbonUnsafe.unsafe.getInt(baseObject, rowB + sizeB);
+ sizeB += 4;
+ diff = dimFieldA - dimFieldB;
+ if (diff != 0) {
+ return diff;
+ }
+ }
+ }
+
+ return diff;
+ }
+
+ /**
+ * Below method will be used to compare two mdkey
+ */
+ public int compare(UnsafeCarbonRow rowL, Object baseObjectL, UnsafeCarbonRow rowR,
+ Object baseObjectR) {
+ int diff = 0;
+ long rowA = rowL.address;
+ long rowB = rowR.address;
+ int sizeA = 0;
+ int sizeB = 0;
+ for (boolean isNoDictionary : noDictionaryColMaping) {
+ if (isNoDictionary) {
+ short aShort1 = CarbonUnsafe.unsafe.getShort(baseObjectL, rowA + sizeA);
+ byte[] byteArr1 = new byte[aShort1];
+ sizeA += 2;
+ CarbonUnsafe.unsafe
+ .copyMemory(baseObjectL, rowA + sizeA, byteArr1, CarbonUnsafe.BYTE_ARRAY_OFFSET,
+ aShort1);
+ sizeA += aShort1;
+
+ short aShort2 = CarbonUnsafe.unsafe.getShort(baseObjectR, rowB + sizeB);
+ byte[] byteArr2 = new byte[aShort2];
+ sizeB += 2;
+ CarbonUnsafe.unsafe
+ .copyMemory(baseObjectR, rowB + sizeB, byteArr2, CarbonUnsafe.BYTE_ARRAY_OFFSET,
+ aShort2);
+ sizeB += aShort2;
+
+ int difference = UnsafeComparer.INSTANCE.compareTo(byteArr1, byteArr2);
+ if (difference != 0) {
+ return difference;
+ }
+ } else {
+ int dimFieldA = CarbonUnsafe.unsafe.getInt(baseObjectL, rowA + sizeA);
+ sizeA += 4;
+ int dimFieldB = CarbonUnsafe.unsafe.getInt(baseObjectR, rowB + sizeB);
+ sizeB += 4;
+ diff = dimFieldA - dimFieldB;
+ if (diff != 0) {
+ return diff;
+ }
+ }
+ }
+
+ return diff;
+ }
+}
http://git-wip-us.apache.org/repos/asf/incubator-carbondata/blob/f1f9348d/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/comparator/UnsafeRowComparatorForNormalDIms.java
----------------------------------------------------------------------
diff --git a/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/comparator/UnsafeRowComparatorForNormalDIms.java b/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/comparator/UnsafeRowComparatorForNormalDIms.java
new file mode 100644
index 0000000..8794753
--- /dev/null
+++ b/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/comparator/UnsafeRowComparatorForNormalDIms.java
@@ -0,0 +1,61 @@
+/*
+ * 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.processing.newflow.sort.unsafe.comparator;
+
+import java.util.Comparator;
+
+import org.apache.carbondata.core.unsafe.CarbonUnsafe;
+import org.apache.carbondata.processing.newflow.sort.unsafe.UnsafeCarbonRowPage;
+import org.apache.carbondata.processing.newflow.sort.unsafe.holder.UnsafeCarbonRow;
+
+public class UnsafeRowComparatorForNormalDIms implements Comparator<UnsafeCarbonRow> {
+
+ private Object baseObject;
+
+ private int dimCount;
+
+ public UnsafeRowComparatorForNormalDIms(int dimCount, UnsafeCarbonRowPage rowPage) {
+ this.baseObject = rowPage.getDataBlock().getBaseObject();
+ this.dimCount = dimCount;
+ }
+
+ /**
+ * Below method will be used to compare two mdkey
+ */
+ public int compare(UnsafeCarbonRow rowL, UnsafeCarbonRow rowR) {
+ int diff = 0;
+ long rowA = rowL.address;
+ long rowB = rowR.address;
+ int sizeA = 0;
+ int sizeB = 0;
+ for (int i = 0; i < dimCount; i++) {
+ int dimFieldA = CarbonUnsafe.unsafe.getInt(baseObject, rowA + sizeA);
+ sizeA += 4;
+ int dimFieldB = CarbonUnsafe.unsafe.getInt(baseObject, rowB + sizeB);
+ sizeB += 4;
+ diff = dimFieldA - dimFieldB;
+ if (diff != 0) {
+ return diff;
+ }
+ }
+
+ return diff;
+ }
+}
http://git-wip-us.apache.org/repos/asf/incubator-carbondata/blob/f1f9348d/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/holder/SortTempChunkHolder.java
----------------------------------------------------------------------
diff --git a/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/holder/SortTempChunkHolder.java b/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/holder/SortTempChunkHolder.java
new file mode 100644
index 0000000..8d199e2
--- /dev/null
+++ b/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/holder/SortTempChunkHolder.java
@@ -0,0 +1,35 @@
+/*
+ * 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.processing.newflow.sort.unsafe.holder;
+
+import org.apache.carbondata.processing.sortandgroupby.exception.CarbonSortKeyAndGroupByException;
+
+/**
+ * Interface for merging temporary sort files/ inmemory data
+ */
+public interface SortTempChunkHolder extends Comparable<SortTempChunkHolder> {
+
+ boolean hasNext();
+
+ void readRow() throws CarbonSortKeyAndGroupByException;
+
+ Object[] getRow();
+
+ int numberOfRows();
+
+ void close();
+}
http://git-wip-us.apache.org/repos/asf/incubator-carbondata/blob/f1f9348d/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/holder/UnsafeCarbonRow.java
----------------------------------------------------------------------
diff --git a/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/holder/UnsafeCarbonRow.java b/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/holder/UnsafeCarbonRow.java
new file mode 100644
index 0000000..b71abf1
--- /dev/null
+++ b/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/holder/UnsafeCarbonRow.java
@@ -0,0 +1,23 @@
+/*
+ * 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.processing.newflow.sort.unsafe.holder;
+
+public class UnsafeCarbonRow {
+
+ public long address;
+
+}
http://git-wip-us.apache.org/repos/asf/incubator-carbondata/blob/f1f9348d/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/holder/UnsafeCarbonRowForMerge.java
----------------------------------------------------------------------
diff --git a/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/holder/UnsafeCarbonRowForMerge.java b/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/holder/UnsafeCarbonRowForMerge.java
new file mode 100644
index 0000000..3534b6d
--- /dev/null
+++ b/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/holder/UnsafeCarbonRowForMerge.java
@@ -0,0 +1,25 @@
+/*
+ * 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.processing.newflow.sort.unsafe.holder;
+
+public class UnsafeCarbonRowForMerge extends UnsafeCarbonRow {
+
+ public byte index;
+
+}
http://git-wip-us.apache.org/repos/asf/incubator-carbondata/blob/f1f9348d/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/holder/UnsafeFinalMergePageHolder.java
----------------------------------------------------------------------
diff --git a/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/holder/UnsafeFinalMergePageHolder.java b/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/holder/UnsafeFinalMergePageHolder.java
new file mode 100644
index 0000000..5bce745
--- /dev/null
+++ b/processing/src/main/java/org/apache/carbondata/processing/newflow/sort/unsafe/holder/UnsafeFinalMergePageHolder.java
@@ -0,0 +1,90 @@
+/*
+ * 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.processing.newflow.sort.unsafe.holder;
+
+import org.apache.carbondata.common.logging.LogService;
+import org.apache.carbondata.common.logging.LogServiceFactory;
+import org.apache.carbondata.processing.newflow.sort.unsafe.UnsafeCarbonRowPage;
+import org.apache.carbondata.processing.newflow.sort.unsafe.merger.UnsafeInMemoryIntermediateDataMerger;
+import org.apache.carbondata.processing.sortandgroupby.sortdata.NewRowComparator;
+
+public class UnsafeFinalMergePageHolder implements SortTempChunkHolder {
+
+ private static final LogService LOGGER =
+ LogServiceFactory.getLogService(UnsafeFinalMergePageHolder.class.getName());
+
+ private int counter;
+
+ private int actualSize;
+
+ private long[] mergedAddresses;
+
+ private byte[] rowPageIndexes;
+
+ private UnsafeCarbonRowPage[] rowPages;
+
+ private NewRowComparator comparator;
+
+ private Object[] currentRow;
+
+ private int columnSize;
+
+ public UnsafeFinalMergePageHolder(UnsafeInMemoryIntermediateDataMerger merger,
+ boolean[] noDictMapping, int columnSize) {
+ this.actualSize = merger.getEntryCount();
+ this.mergedAddresses = merger.getMergedAddresses();
+ this.rowPageIndexes = merger.getRowPageIndexes();
+ this.rowPages = merger.getUnsafeCarbonRowPages();
+ LOGGER.audit("Processing unsafe inmemory rows page with size : " + actualSize);
+ this.comparator = new NewRowComparator(noDictMapping);
+ this.columnSize = columnSize;
+ }
+
+ public boolean hasNext() {
+ if (counter < actualSize) {
+ return true;
+ }
+ return false;
+ }
+
+ public void readRow() {
+ currentRow = new Object[columnSize];
+ rowPages[rowPageIndexes[counter]].getRow(mergedAddresses[counter], currentRow);
+ counter++;
+ }
+
+ public Object[] getRow() {
+ return currentRow;
+ }
+
+ @Override public int compareTo(SortTempChunkHolder o) {
+ return comparator.compare(currentRow, o.getRow());
+ }
+
+ public int numberOfRows() {
+ return actualSize;
+ }
+
+ public void close() {
+ for (int i = 0; i < rowPages.length; i++) {
+ rowPages[i].freeMemory();
+ }
+ }
+}
[4/4] incubator-carbondata git commit: [CARBONDATA-470]Add unsafe
offheap and on-heap sort in carbodata loading This closes #369
Posted by ja...@apache.org.
[CARBONDATA-470]Add unsafe offheap and on-heap sort in carbodata loading This closes #369
Project: http://git-wip-us.apache.org/repos/asf/incubator-carbondata/repo
Commit: http://git-wip-us.apache.org/repos/asf/incubator-carbondata/commit/910155d4
Tree: http://git-wip-us.apache.org/repos/asf/incubator-carbondata/tree/910155d4
Diff: http://git-wip-us.apache.org/repos/asf/incubator-carbondata/diff/910155d4
Branch: refs/heads/master
Commit: 910155d42196dafed6d684df9d29b72ceb702d52
Parents: 8940514 f1f9348
Author: jackylk <ja...@huawei.com>
Authored: Tue Dec 13 19:14:07 2016 +0800
Committer: jackylk <ja...@huawei.com>
Committed: Tue Dec 13 19:14:07 2016 +0800
----------------------------------------------------------------------
.../core/constants/CarbonCommonConstants.java | 32 +
.../core/memory/HeapMemoryAllocator.java | 84 ++
.../carbondata/core/memory/MemoryAllocator.java | 36 +
.../carbondata/core/memory/MemoryBlock.java | 57 ++
.../carbondata/core/memory/MemoryLocation.java | 55 ++
.../core/memory/UnsafeMemoryAllocator.java | 40 +
.../carbondata/core/unsafe/CarbonUnsafe.java | 48 +
dev/javastyle-config.xml | 6 +-
dev/javastyle-suppressions.xml | 35 +
.../newflow/iterator/InputIterator.java | 40 +
.../impl/UnsafeParallelReadMergeSorterImpl.java | 219 +++++
.../newflow/sort/unsafe/IntPointerBuffer.java | 95 ++
.../sort/unsafe/UnsafeCarbonRowPage.java | 356 +++++++
.../sort/unsafe/UnsafeMemoryManager.java | 99 ++
.../newflow/sort/unsafe/UnsafeSortDataRows.java | 356 +++++++
.../unsafe/comparator/UnsafeRowComparator.java | 133 +++
.../UnsafeRowComparatorForNormalDIms.java | 61 ++
.../sort/unsafe/holder/SortTempChunkHolder.java | 35 +
.../sort/unsafe/holder/UnsafeCarbonRow.java | 23 +
.../unsafe/holder/UnsafeCarbonRowForMerge.java | 25 +
.../holder/UnsafeFinalMergePageHolder.java | 90 ++
.../unsafe/holder/UnsafeInmemoryHolder.java | 80 ++
.../holder/UnsafeInmemoryMergeHolder.java | 90 ++
.../holder/UnsafeSortTempFileChunkHolder.java | 455 +++++++++
.../UnsafeInMemoryIntermediateDataMerger.java | 217 +++++
.../merger/UnsafeIntermediateFileMerger.java | 364 +++++++
.../unsafe/merger/UnsafeIntermediateMerger.java | 180 ++++
.../UnsafeSingleThreadFinalSortFilesMerger.java | 313 ++++++
.../newflow/sort/unsafe/sort/TimSort.java | 943 +++++++++++++++++++
.../unsafe/sort/UnsafeIntSortDataFormat.java | 74 ++
.../newflow/steps/SortProcessorStepImpl.java | 10 +-
31 files changed, 4647 insertions(+), 4 deletions(-)
----------------------------------------------------------------------