Skip to content

Commit

Permalink
Somewhat hacky fix for descending sorts
Browse files Browse the repository at this point in the history
  • Loading branch information
JoshRosen committed Jul 7, 2015
1 parent 08701e7 commit 1d7ffaa
Show file tree
Hide file tree
Showing 2 changed files with 16 additions and 3 deletions.
Original file line number Diff line number Diff line change
Expand Up @@ -27,6 +27,7 @@ import org.apache.spark.sql.catalyst.expressions._
import org.apache.spark.sql.Row
import org.apache.spark.sql.catalyst.plans.physical._
import org.apache.spark.util.collection.ExternalSorter
import org.apache.spark.util.collection.unsafe.sort.PrefixComparator
import org.apache.spark.util.{CompletionIterator, MutablePair}
import org.apache.spark.{HashPartitioner, SparkEnv}

Expand Down Expand Up @@ -274,7 +275,18 @@ case class UnsafeExternalSort(
def doSort(iterator: Iterator[InternalRow]): Iterator[InternalRow] = {
val ordering = newOrdering(sortOrder, child.output)
val boundSortExpression = BindReferences.bindReference(sortOrder.head, child.output)
val prefixComparator = SortPrefixUtils.getPrefixComparator(boundSortExpression)
// Hack until we generate separate comparator implementations for ascending vs. descending
// (or choose to codegen them):
val prefixComparator = {
val comp = SortPrefixUtils.getPrefixComparator(boundSortExpression)
if (sortOrder.head.direction == Descending) {
new PrefixComparator {
override def compare(p1: Long, p2: Long): Int = -1 * comp.compare(p1, p2)
}
} else {
comp
}
}
val prefixComputer = {
val prefixComputer = SortPrefixUtils.getPrefixComputer(boundSortExpression)
new UnsafeExternalRowSorter.PrefixComputer {
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -38,9 +38,10 @@ class UnsafeExternalSortSuite extends SparkPlanTest with BeforeAndAfterAll {

// Test sorting on different data types
for (
dataType <- DataTypeTestUtils.atomicTypes; // Disable null type for now due to bug in SqlSerializer2 ++ Set(NullType);
dataType <- DataTypeTestUtils.atomicTypes // Disable null type for now due to bug in SqlSerializer2 ++ Set(NullType);
if !dataType.isInstanceOf[DecimalType]; // Since we don't have an unsafe representation for decimals
nullable <- Seq(true, false);
sortOrder <- Seq('a.asc :: Nil);
sortOrder <- Seq('a.asc :: Nil, 'a.desc :: Nil);
randomDataGenerator <- RandomDataGenerator.forType(dataType, nullable)
) {
test(s"sorting on $dataType with nullable=$nullable, sortOrder=$sortOrder") {
Expand Down

0 comments on commit 1d7ffaa

Please sign in to comment.