Skip to content

Commit

Permalink
code review, minor updates, and rearrange imports.
Browse files Browse the repository at this point in the history
  • Loading branch information
rxin committed Jan 8, 2015
1 parent c130913 commit e98a7c0
Show file tree
Hide file tree
Showing 36 changed files with 79 additions and 89 deletions.
Original file line number Diff line number Diff line change
Expand Up @@ -719,7 +719,7 @@ object StructType {
* // twoFields: StructType =
* // StructType(List(StructField(b,LongType,false), StructField(c,BooleanType,false)))
*
* // Those names do not have matching fields will be ignored.
* // Any names without matching fields will be ignored.
* // For the case shown below, "d" will be ignored and
* // it is treated as struct(Set("b", "c")).
* val ignoreNonExisting = struct(Set("b", "c", "d"))
Expand Down Expand Up @@ -766,8 +766,8 @@ case class StructType(fields: Array[StructField]) extends DataType with Seq[Stru
}

/**
* Returns a [[StructType]] containing [[StructField]]s of the given names.
* Those names which do not have matching fields will be ignored.
* Returns a [[StructType]] containing [[StructField]]s of the given names, preserving the
* original order of fields. Those names which do not have matching fields will be ignored.
*/
def apply(names: Set[String]): StructType = {
val nonExistFields = names -- fieldNamesSet
Expand All @@ -779,8 +779,8 @@ case class StructType(fields: Array[StructField]) extends DataType with Seq[Stru
StructType(fields.filter(f => names.contains(f.name)))
}

protected[sql] def toAttributes =
fields.map(f => AttributeReference(f.name, f.dataType, f.nullable, f.metadata)())
protected[sql] def toAttributes: Seq[AttributeReference] =
map(f => AttributeReference(f.name, f.dataType, f.nullable, f.metadata)())

def treeString: String = {
val builder = new StringBuilder
Expand All @@ -799,7 +799,7 @@ case class StructType(fields: Array[StructField]) extends DataType with Seq[Stru

override private[sql] def jsonValue =
("type" -> typeName) ~
("fields" -> fields.map(_.jsonValue).toSeq)
("fields" -> map(_.jsonValue))

override def apply(fieldIndex: Int): StructField = fields(fieldIndex)

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -21,17 +21,15 @@ import java.sql.{Date, Timestamp}

import scala.collection.immutable.HashSet

import org.apache.spark.sql.types.decimal.Decimal
import org.scalactic.TripleEqualsSupport.Spread
import org.scalatest.FunSuite
import org.scalatest.Matchers._
import org.scalactic.TripleEqualsSupport.Spread

import org.apache.spark.sql.catalyst.dsl.expressions._
import org.apache.spark.sql.types._
import org.apache.spark.sql.types.decimal.Decimal


/* Implicit conversions */
import org.apache.spark.sql.catalyst.dsl.expressions._

class ExpressionEvaluationSuite extends FunSuite {

test("literals") {
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -17,10 +17,11 @@

package org.apache.spark.sql.catalyst.util

import org.apache.spark.sql.types.{MetadataBuilder, Metadata}
import org.json4s.jackson.JsonMethods.parse
import org.scalatest.FunSuite

import org.apache.spark.sql.types.{MetadataBuilder, Metadata}

class MetadataSuite extends FunSuite {

val baseMetadata = new MetadataBuilder()
Expand Down
10 changes: 4 additions & 6 deletions sql/core/src/main/scala/org/apache/spark/sql/SQLContext.scala
Original file line number Diff line number Diff line change
Expand Up @@ -21,23 +21,21 @@ import scala.language.implicitConversions
import scala.reflect.runtime.universe.TypeTag

import org.apache.hadoop.conf.Configuration

import org.apache.spark.SparkContext
import org.apache.spark.annotation.{AlphaComponent, DeveloperApi, Experimental}
import org.apache.spark.rdd.RDD
import org.apache.spark.sql.catalyst.ScalaReflection
import org.apache.spark.sql.catalyst.analysis._
import org.apache.spark.sql.catalyst.dsl.ExpressionConversions
import org.apache.spark.sql.catalyst.expressions._
import org.apache.spark.sql.catalyst.optimizer.{Optimizer, DefaultOptimizer}
import org.apache.spark.sql.catalyst.optimizer.{DefaultOptimizer, Optimizer}
import org.apache.spark.sql.catalyst.plans.logical.LogicalPlan
import org.apache.spark.sql.catalyst.rules.RuleExecutor
import org.apache.spark.sql.types.UserDefinedType
import org.apache.spark.sql.execution.{SparkStrategies, _}
import org.apache.spark.sql.execution._
import org.apache.spark.sql.json._
import org.apache.spark.sql.types._
import org.apache.spark.sql.parquet.ParquetRelation
import org.apache.spark.sql.sources.{DataSourceStrategy, BaseRelation, DDLParser, LogicalRelation}
import org.apache.spark.sql.sources.{BaseRelation, DDLParser, DataSourceStrategy, LogicalRelation}
import org.apache.spark.sql.types._

/**
* :: AlphaComponent ::
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -20,12 +20,12 @@ package org.apache.spark.sql.api.java
import java.util.{List => JList}

import org.apache.spark.Partitioner
import org.apache.spark.api.java.{JavaRDDLike, JavaRDD}
import org.apache.spark.api.java.{JavaRDD, JavaRDDLike}
import org.apache.spark.api.java.function.{Function => JFunction}
import org.apache.spark.rdd.RDD
import org.apache.spark.sql.types.StructType
import org.apache.spark.sql.{SQLContext, SchemaRDD, SchemaRDDLike}
import org.apache.spark.sql.catalyst.plans.logical.LogicalPlan
import org.apache.spark.sql.types.StructType
import org.apache.spark.storage.StorageLevel

/**
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -17,8 +17,6 @@

package org.apache.spark.sql.api.java

import org.apache.spark.sql.types.decimal.Decimal

import scala.annotation.varargs
import scala.collection.convert.Wrappers.{JListWrapper, JMapWrapper}
import scala.collection.JavaConversions
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -17,11 +17,11 @@

package org.apache.spark.sql.columnar

import java.nio.{ByteOrder, ByteBuffer}
import java.nio.{ByteBuffer, ByteOrder}

import org.apache.spark.sql.types.{BinaryType, NativeType, DataType}
import org.apache.spark.sql.catalyst.expressions.MutableRow
import org.apache.spark.sql.columnar.compression.CompressibleColumnAccessor
import org.apache.spark.sql.types.{BinaryType, DataType, NativeType}

/**
* An `Iterator` like trait used to extract values from columnar byte buffer. When a value is
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -20,9 +20,9 @@ package org.apache.spark.sql.columnar
import java.nio.{ByteBuffer, ByteOrder}

import org.apache.spark.sql.Row
import org.apache.spark.sql.types._
import org.apache.spark.sql.columnar.ColumnBuilder._
import org.apache.spark.sql.columnar.compression.{AllCompressionSchemes, CompressibleColumnBuilder}
import org.apache.spark.sql.types._

private[sql] trait ColumnBuilder {
/**
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -24,8 +24,8 @@ import scala.reflect.runtime.universe.TypeTag

import org.apache.spark.sql.Row
import org.apache.spark.sql.catalyst.expressions.MutableRow
import org.apache.spark.sql.types._
import org.apache.spark.sql.execution.SparkSqlSerializer
import org.apache.spark.sql.types._

/**
* An abstract class that represents type of a column. Used to append/extract Java objects into/from
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -18,8 +18,8 @@
package org.apache.spark.sql.columnar.compression

import org.apache.spark.sql.catalyst.expressions.MutableRow
import org.apache.spark.sql.types.NativeType
import org.apache.spark.sql.columnar.{ColumnAccessor, NativeColumnAccessor}
import org.apache.spark.sql.types.NativeType

private[sql] trait CompressibleColumnAccessor[T <: NativeType] extends ColumnAccessor {
this: NativeColumnAccessor[T] =>
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -21,8 +21,8 @@ import java.nio.{ByteBuffer, ByteOrder}

import org.apache.spark.Logging
import org.apache.spark.sql.Row
import org.apache.spark.sql.types.NativeType
import org.apache.spark.sql.columnar.{ColumnBuilder, NativeColumnBuilder}
import org.apache.spark.sql.types.NativeType

/**
* A stackable trait that builds optionally compressed byte buffer for a column. Memory layout of
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -21,8 +21,8 @@ import java.nio.{ByteBuffer, ByteOrder}

import org.apache.spark.sql.Row
import org.apache.spark.sql.catalyst.expressions.MutableRow
import org.apache.spark.sql.types.NativeType
import org.apache.spark.sql.columnar.{ColumnType, NativeColumnType}
import org.apache.spark.sql.types.NativeType

private[sql] trait Encoder[T <: NativeType] {
def gatherCompressibilityStats(row: Row, ordinal: Int): Unit = {}
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -25,10 +25,11 @@ import scala.reflect.runtime.universe.runtimeMirror

import org.apache.spark.sql.Row
import org.apache.spark.sql.catalyst.expressions.{MutableRow, SpecificMutableRow}
import org.apache.spark.sql.types._
import org.apache.spark.sql.columnar._
import org.apache.spark.sql.types._
import org.apache.spark.util.Utils


private[sql] case object PassThrough extends CompressionScheme {
override val typeId = 0

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -17,15 +17,15 @@

package org.apache.spark.sql.execution

import org.apache.spark.sql.{SQLContext, Strategy, execution}
import org.apache.spark.sql.catalyst.expressions._
import org.apache.spark.sql.catalyst.planning._
import org.apache.spark.sql.catalyst.plans._
import org.apache.spark.sql.catalyst.plans.logical.LogicalPlan
import org.apache.spark.sql.catalyst.plans.physical._
import org.apache.spark.sql.types._
import org.apache.spark.sql.columnar.{InMemoryRelation, InMemoryColumnarTableScan}
import org.apache.spark.sql.columnar.{InMemoryColumnarTableScan, InMemoryRelation}
import org.apache.spark.sql.parquet._
import org.apache.spark.sql.types._
import org.apache.spark.sql.{SQLContext, Strategy, execution}


private[sql] abstract class SparkStrategies extends QueryPlanner[SparkPlan] {
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -19,8 +19,6 @@ package org.apache.spark.sql.execution

import java.util.{List => JList, Map => JMap}

import org.apache.spark.sql.types.decimal.Decimal

import scala.collection.JavaConversions._
import scala.collection.JavaConverters._

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -17,16 +17,15 @@

package org.apache.spark.sql.parquet

import org.apache.spark.sql.types.decimal.Decimal

import scala.collection.mutable.{Buffer, ArrayBuffer, HashMap}

import parquet.io.api.{PrimitiveConverter, GroupConverter, Binary, Converter}
import parquet.schema.MessageType

import org.apache.spark.sql.types._
import org.apache.spark.sql.catalyst.expressions._
import org.apache.spark.sql.parquet.CatalystConverter.FieldType
import org.apache.spark.sql.types._
import org.apache.spark.sql.types.decimal.Decimal

/**
* Collection of converters of Parquet types (group and primitive types) that
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -18,25 +18,24 @@ package org.apache.spark.sql.parquet

import java.util.{List => JList}

import scala.collection.JavaConversions._

import org.apache.hadoop.fs.{FileStatus, FileSystem, Path}
import org.apache.hadoop.conf.{Configurable, Configuration}
import org.apache.hadoop.io.Writable
import org.apache.hadoop.mapreduce.{JobContext, InputSplit, Job}
import org.apache.spark.sql.catalyst.expressions.codegen.GeneratePredicate

import parquet.hadoop.ParquetInputFormat
import parquet.hadoop.util.ContextUtil

import org.apache.spark.annotation.DeveloperApi
import org.apache.spark.{Partition => SparkPartition, Logging}
import org.apache.spark.rdd.{NewHadoopPartition, RDD}

import org.apache.spark.sql.{SQLConf, Row, SQLContext}
import org.apache.spark.sql.catalyst.expressions._
import org.apache.spark.sql.types.{StringType, IntegerType, StructField, StructType}
import org.apache.spark.sql.sources._
import org.apache.spark.sql.types.{IntegerType, StructField, StructType}

import scala.collection.JavaConversions._

/**
* Allows creation of parquet based tables using the syntax
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -27,7 +27,7 @@ private[sql] case class LogicalRelation(relation: BaseRelation)
extends LeafNode
with MultiInstanceRelation {

override val output: Seq[AttributeReference] = relation.schema.toAttributes.toSeq
override val output: Seq[AttributeReference] = relation.schema.toAttributes

// Logical Relations are distinct if they have different output for the sake of transformations.
override def equals(other: Any) = other match {
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -19,7 +19,6 @@ package org.apache.spark.sql

import java.sql.{Date, Timestamp}

import org.apache.spark.sql.types.decimal.Decimal
import org.scalatest.FunSuite

import org.apache.spark.sql.catalyst.expressions._
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -20,8 +20,8 @@ package org.apache.spark.sql
import scala.beans.{BeanInfo, BeanProperty}

import org.apache.spark.rdd.RDD
import org.apache.spark.sql.types._
import org.apache.spark.sql.test.TestSQLContext._
import org.apache.spark.sql.types._

@SQLUserDefinedType(udt = classOf[MyDenseVectorUDT])
private[sql] class MyDenseVector(val data: Array[Double]) extends Serializable {
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -24,9 +24,9 @@ import org.scalatest.FunSuite

import org.apache.spark.Logging
import org.apache.spark.sql.catalyst.expressions.GenericMutableRow
import org.apache.spark.sql.types._
import org.apache.spark.sql.columnar.ColumnarTestUtils._
import org.apache.spark.sql.execution.SparkSqlSerializer
import org.apache.spark.sql.types._

class ColumnTypeSuite extends FunSuite with Logging {
val DEFAULT_BUFFER_SIZE = 512
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -19,8 +19,8 @@ package org.apache.spark.sql.columnar

import org.scalatest.FunSuite

import org.apache.spark.sql.types._
import org.apache.spark.sql.execution.SparkSqlSerializer
import org.apache.spark.sql.types._

class TestNullableColumnBuilder[T <: DataType, JvmType](columnType: ColumnType[T, JvmType])
extends BasicColumnBuilder[T, JvmType](new NoopColumnStats, columnType)
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -22,9 +22,9 @@ import java.nio.ByteBuffer
import org.scalatest.FunSuite

import org.apache.spark.sql.catalyst.expressions.GenericMutableRow
import org.apache.spark.sql.types.NativeType
import org.apache.spark.sql.columnar._
import org.apache.spark.sql.columnar.ColumnarTestUtils._
import org.apache.spark.sql.types.NativeType

class DictionaryEncodingSuite extends FunSuite {
testDictionaryEncoding(new IntColumnStats, INT)
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -20,9 +20,9 @@ package org.apache.spark.sql.columnar.compression
import org.scalatest.FunSuite

import org.apache.spark.sql.catalyst.expressions.GenericMutableRow
import org.apache.spark.sql.types.IntegralType
import org.apache.spark.sql.columnar._
import org.apache.spark.sql.columnar.ColumnarTestUtils._
import org.apache.spark.sql.types.IntegralType

class IntegralDeltaSuite extends FunSuite {
testIntegralDelta(new IntColumnStats, INT, IntDelta)
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -20,9 +20,9 @@ package org.apache.spark.sql.columnar.compression
import org.scalatest.FunSuite

import org.apache.spark.sql.catalyst.expressions.GenericMutableRow
import org.apache.spark.sql.types.NativeType
import org.apache.spark.sql.columnar._
import org.apache.spark.sql.columnar.ColumnarTestUtils._
import org.apache.spark.sql.types.NativeType

class RunLengthEncodingSuite extends FunSuite {
testRunLengthEncoding(new NoopColumnStats, BOOLEAN)
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -17,8 +17,8 @@

package org.apache.spark.sql.columnar.compression

import org.apache.spark.sql.types.NativeType
import org.apache.spark.sql.columnar._
import org.apache.spark.sql.types.NativeType

class TestCompressibleColumnBuilder[T <: NativeType](
override val columnStats: ColumnStats,
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -17,19 +17,19 @@

package org.apache.spark.sql.json

import org.apache.spark.sql.types._
import org.apache.spark.sql.types.decimal.Decimal
import org.apache.spark.sql.catalyst.util._
import org.apache.spark.sql.json.JsonRDD.{enforceCorrectType, compatibleType}
import org.apache.spark.sql.{Row, SQLConf, QueryTest}
import java.sql.{Date, Timestamp}

import org.apache.spark.sql.TestData._
import org.apache.spark.sql.catalyst.util._
import org.apache.spark.sql.json.JsonRDD.{compatibleType, enforceCorrectType}
import org.apache.spark.sql.test.TestSQLContext
import org.apache.spark.sql.test.TestSQLContext._

import java.sql.{Date, Timestamp}
import org.apache.spark.sql.types._
import org.apache.spark.sql.types.decimal.Decimal
import org.apache.spark.sql.{QueryTest, Row, SQLConf}

class JsonSuite extends QueryTest {
import TestJsonData._
import org.apache.spark.sql.json.TestJsonData._
TestJsonData

test("Type promotion") {
Expand Down
Loading

0 comments on commit e98a7c0

Please sign in to comment.