Skip to content
Closed
Show file tree
Hide file tree
Changes from all commits
Commits
File filter

Filter by extension

Filter by extension

Conversations
Failed to load comments.
Loading
Jump to
Jump to file
Failed to load files.
Loading
Diff view
Diff view
2 changes: 1 addition & 1 deletion scalastyle-config.xml
Original file line number Diff line number Diff line change
Expand Up @@ -218,7 +218,7 @@ This file is divided into 3 sections:
<check level="error" class="org.scalastyle.scalariform.EqualsHashCodeChecker" enabled="false"></check>

<!-- Should turn this on, but we have a few places that need to be fixed first -->
<check level="warning" class="org.scalastyle.scalariform.DisallowSpaceBeforeTokenChecker" enabled="true">
<check customId="whitespacebeforetoken" level="warning" class="org.scalastyle.scalariform.DisallowSpaceBeforeTokenChecker" enabled="true">
<parameters>
<parameter name="tokens">COLON, COMMA</parameter>
</parameters>
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -49,7 +49,7 @@ object ScalaReflection extends ScalaReflection {
* Unlike `schemaFor`, this function doesn't do any massaging of types into the Spark SQL type
* system. As a result, ObjectType will be returned for things like boxed Integers
*/
def dataTypeFor[T : TypeTag]: DataType = dataTypeFor(localTypeOf[T])
def dataTypeFor[T: TypeTag]: DataType = dataTypeFor(localTypeOf[T])

private def dataTypeFor(tpe: `Type`): DataType = ScalaReflectionLock.synchronized {
tpe match {
Expand Down Expand Up @@ -116,7 +116,7 @@ object ScalaReflection extends ScalaReflection {
* from ordinal 0 (since there are no names to map to). The actual location can be moved by
* calling resolve/bind with a new schema.
*/
def constructorFor[T : TypeTag]: Expression = {
def constructorFor[T: TypeTag]: Expression = {
val tpe = localTypeOf[T]
val clsName = getClassNameFromType(tpe)
val walkedTypePath = s"""- root class: "${clsName}"""" :: Nil
Expand Down Expand Up @@ -386,7 +386,7 @@ object ScalaReflection extends ScalaReflection {
* * the element type of [[Array]] or [[Seq]]: `array element class: "abc.xyz.MyClass"`
* * the field of [[Product]]: `field (class: "abc.xyz.MyClass", name: "myField")`
*/
def extractorsFor[T : TypeTag](inputObject: Expression): CreateNamedStruct = {
def extractorsFor[T: TypeTag](inputObject: Expression): CreateNamedStruct = {
val tpe = localTypeOf[T]
val clsName = getClassNameFromType(tpe)
val walkedTypePath = s"""- root class: "${clsName}"""" :: Nil
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -203,7 +203,7 @@ object SqlParser extends AbstractSparkSQLParser with DataTypeParser {
)

protected lazy val ordering: Parser[Seq[SortOrder]] =
( rep1sep(expression ~ direction.? , ",") ^^ {
( rep1sep(expression ~ direction.?, ",") ^^ {
case exps => exps.map(pair => SortOrder(pair._1, pair._2.getOrElse(Ascending)))
}
)
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -84,7 +84,7 @@ class Analyzer(
ResolveAggregateFunctions ::
DistinctAggregationRewriter(conf) ::
HiveTypeCoercion.typeCoercionRules ++
extendedResolutionRules : _*),
extendedResolutionRules: _*),
Batch("Nondeterministic", Once,
PullOutNondeterministic),
Batch("UDF", Once,
Expand All @@ -110,7 +110,7 @@ class Analyzer(
// Taking into account the reasonableness and the implementation complexity,
// here use the CTE definition first, check table name only and ignore database name
// see https://github.com/apache/spark/pull/4929#discussion_r27186638 for more info
case u : UnresolvedRelation =>
case u: UnresolvedRelation =>
val substituted = cteRelations.get(u.tableIdentifier.table).map { relation =>
val withAlias = u.alias.map(Subquery(_, relation))
withAlias.getOrElse(relation)
Expand Down Expand Up @@ -889,7 +889,7 @@ class Analyzer(
_.transform {
// Extracts children expressions of a WindowFunction (input parameters of
// a WindowFunction).
case wf : WindowFunction =>
case wf: WindowFunction =>
val newChildren = wf.children.map(extractExpr)
wf.withNewChildren(newChildren)

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -323,13 +323,13 @@ object FunctionRegistry {
} else {
// Otherwise, find an ctor method that matches the number of arguments, and use that.
val params = Seq.fill(expressions.size)(classOf[Expression])
val f = Try(tag.runtimeClass.getDeclaredConstructor(params : _*)) match {
val f = Try(tag.runtimeClass.getDeclaredConstructor(params: _*)) match {
case Success(e) =>
e
case Failure(e) =>
throw new AnalysisException(s"Invalid number of arguments for function $name")
}
Try(f.newInstance(expressions : _*).asInstanceOf[Expression]) match {
Try(f.newInstance(expressions: _*).asInstanceOf[Expression]) match {
case Success(e) => e
case Failure(e) => throw new AnalysisException(e.getMessage)
}
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -529,7 +529,7 @@ object HiveTypeCoercion {
if falseValues.contains(value) => And(IsNotNull(bool), Not(bool))

case EqualTo(left @ BooleanType(), right @ NumericType()) =>
transform(left , right)
transform(left, right)
case EqualTo(left @ NumericType(), right @ BooleanType()) =>
transform(right, left)
case EqualNullSafe(left @ BooleanType(), right @ NumericType()) =>
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -61,9 +61,11 @@ package object dsl {
trait ImplicitOperators {
def expr: Expression

// scalastyle:off whitespacebeforetoken
def unary_- : Expression = UnaryMinus(expr)
def unary_! : Predicate = Not(expr)
def unary_~ : Expression = BitwiseNot(expr)
// scalastyle:on whitespacebeforetoken

def + (other: Expression): Expression = Add(expr, other)
def - (other: Expression): Expression = Subtract(expr, other)
Expand Down Expand Up @@ -141,7 +143,7 @@ package object dsl {
// Note that if we make ExpressionConversions an object rather than a trait, we can
// then make this a value class to avoid the small penalty of runtime instantiation.
def $(args: Any*): analysis.UnresolvedAttribute = {
analysis.UnresolvedAttribute(sc.s(args : _*))
analysis.UnresolvedAttribute(sc.s(args: _*))
}
}

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -44,7 +44,7 @@ import org.apache.spark.util.Utils
* to the name `value`.
*/
object ExpressionEncoder {
def apply[T : TypeTag](): ExpressionEncoder[T] = {
def apply[T: TypeTag](): ExpressionEncoder[T] = {
// We convert the not-serializable TypeTag into StructType and ClassTag.
val mirror = typeTag[T].mirror
val cls = mirror.runtimeClass(typeTag[T].tpe)
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -27,7 +27,7 @@ package object encoders {
* references from a specific schema.) This requirement allows us to preserve whether a given
* object type is being bound by name or by ordinal when doing resolution.
*/
private[sql] def encoderFor[A : Encoder]: ExpressionEncoder[A] = implicitly[Encoder[A]] match {
private[sql] def encoderFor[A: Encoder]: ExpressionEncoder[A] = implicitly[Encoder[A]] match {
case e: ExpressionEncoder[A] =>
e.assertUnresolved()
e
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -164,7 +164,7 @@ abstract class Expression extends TreeNode[Expression] {
* Returns the hash for this expression. Expressions that compute the same result, even if
* they differ cosmetically should return the same hash.
*/
def semanticHash() : Int = {
def semanticHash(): Int = {
def computeHash(e: Seq[Any]): Int = {
// See http://stackoverflow.com/questions/113511/hash-code-implementation
var hash: Int = 17
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -46,7 +46,7 @@ case class Concat(children: Seq[Expression]) extends Expression with ImplicitCas

override def eval(input: InternalRow): Any = {
val inputs = children.map(_.eval(input).asInstanceOf[UTF8String])
UTF8String.concat(inputs : _*)
UTF8String.concat(inputs: _*)
}

override protected def genCode(ctx: CodeGenContext, ev: GeneratedExpressionCode): String = {
Expand Down Expand Up @@ -99,7 +99,7 @@ case class ConcatWs(children: Seq[Expression])
case null => Iterator(null.asInstanceOf[UTF8String])
}
}
UTF8String.concatWs(flatInputs.head, flatInputs.tail : _*)
UTF8String.concatWs(flatInputs.head, flatInputs.tail: _*)
}

override protected def genCode(ctx: CodeGenContext, ev: GeneratedExpressionCode): String = {
Expand Down Expand Up @@ -990,7 +990,7 @@ case class FormatNumber(x: Expression, d: Expression)

def typeHelper(p: String): String = {
x.dataType match {
case _ : DecimalType => s"""$p.toJavaBigDecimal()"""
case _: DecimalType => s"""$p.toJavaBigDecimal()"""
case _ => s"$p"
}
}
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -496,7 +496,7 @@ case class MapPartitions[T, U](

/** Factory for constructing new `AppendColumn` nodes. */
object AppendColumns {
def apply[T, U : Encoder](
def apply[T, U: Encoder](
func: T => U,
tEncoder: ExpressionEncoder[T],
child: LogicalPlan): AppendColumns[T, U] = {
Expand All @@ -522,7 +522,7 @@ case class AppendColumns[T, U](

/** Factory for constructing new `MapGroups` nodes. */
object MapGroups {
def apply[K, T, U : Encoder](
def apply[K, T, U: Encoder](
func: (K, Iterator[T]) => TraversableOnce[U],
kEncoder: ExpressionEncoder[K],
tEncoder: ExpressionEncoder[T],
Expand Down Expand Up @@ -557,7 +557,7 @@ case class MapGroups[K, T, U](

/** Factory for constructing new `CoGroup` nodes. */
object CoGroup {
def apply[Key, Left, Right, Result : Encoder](
def apply[Key, Left, Right, Result: Encoder](
func: (Key, Iterator[Left], Iterator[Right]) => TraversableOnce[Result],
keyEnc: ExpressionEncoder[Key],
leftEnc: ExpressionEncoder[Left],
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -122,7 +122,7 @@ object NumberConverter {
* unsigned, otherwise it is signed.
* NB: This logic is borrowed from org.apache.hadoop.hive.ql.ud.UDFConv
*/
def convert(n: Array[Byte] , fromBase: Int, toBase: Int ): UTF8String = {
def convert(n: Array[Byte], fromBase: Int, toBase: Int ): UTF8String = {
if (fromBase < Character.MIN_RADIX || fromBase > Character.MAX_RADIX
|| Math.abs(toBase) < Character.MIN_RADIX
|| Math.abs(toBase) > Character.MAX_RADIX) {
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -90,7 +90,7 @@ case class ArrayType(elementType: DataType, containsNull: Boolean) extends DataT
private[sql] lazy val interpretedOrdering: Ordering[ArrayData] = new Ordering[ArrayData] {
private[this] val elementOrdering: Ordering[Any] = elementType match {
case dt: AtomicType => dt.ordering.asInstanceOf[Ordering[Any]]
case a : ArrayType => a.interpretedOrdering.asInstanceOf[Ordering[Any]]
case a: ArrayType => a.interpretedOrdering.asInstanceOf[Ordering[Any]]
case s: StructType => s.interpretedOrdering.asInstanceOf[Ordering[Any]]
case other =>
throw new IllegalArgumentException(s"Type $other does not support ordered operations")
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -310,13 +310,15 @@ final class Decimal extends Ordered[Decimal] with Serializable {

def remainder(that: Decimal): Decimal = this % that

// scalastyle:off whitespacebeforetoken
def unary_- : Decimal = {
if (decimalVal.ne(null)) {
Decimal(-decimalVal, precision, scale)
} else {
Decimal(-longVal, precision, scale)
}
}
// scalastyle:on whitespacebeforetoken

def abs: Decimal = if (this.compare(Decimal.ZERO) < 0) this.unary_- else this

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -98,5 +98,5 @@ class EncoderErrorMessageSuite extends SparkFunSuite {
s"""array element class: "${clsName[NonEncodable]}""""))
}

private def clsName[T : ClassTag]: String = implicitly[ClassTag[T]].runtimeClass.getName
private def clsName[T: ClassTag]: String = implicitly[ClassTag[T]].runtimeClass.getName
}
Original file line number Diff line number Diff line change
Expand Up @@ -80,7 +80,7 @@ class JavaSerializable(val value: Int) extends Serializable {
class ExpressionEncoderSuite extends SparkFunSuite {
OuterScopes.outerScopes.put(getClass.getName, this)

implicit def encoder[T : TypeTag]: ExpressionEncoder[T] = ExpressionEncoder()
implicit def encoder[T: TypeTag]: ExpressionEncoder[T] = ExpressionEncoder()

// test flat encoders
encodeDecodeTest(false, "primitive boolean")
Expand Down Expand Up @@ -145,7 +145,7 @@ class ExpressionEncoderSuite extends SparkFunSuite {
encoderFor(Encoders.javaSerialization[JavaSerializable]))

// test product encoders
private def productTest[T <: Product : ExpressionEncoder](input: T): Unit = {
private def productTest[T <: Product: ExpressionEncoder](input: T): Unit = {
encodeDecodeTest(input, input.getClass.getSimpleName)
}

Expand Down Expand Up @@ -286,7 +286,7 @@ class ExpressionEncoderSuite extends SparkFunSuite {
}
}

private def encodeDecodeTest[T : ExpressionEncoder](
private def encodeDecodeTest[T: ExpressionEncoder](
input: T,
testName: String): Unit = {
test(s"encode/decode for $testName: $input") {
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -80,7 +80,7 @@ class BooleanSimplificationSuite extends PlanTest with PredicateHelper {

checkCondition(('a < 2 || 'a > 3 || 'b > 5) && 'a < 2, 'a < 2)

checkCondition('a < 2 && ('a < 2 || 'a > 3 || 'b > 5) , 'a < 2)
checkCondition('a < 2 && ('a < 2 || 'a > 3 || 'b > 5), 'a < 2)

checkCondition(('a < 2 || 'b > 3) && ('a < 2 || 'c > 5), 'a < 2 || ('b > 3 && 'c > 5))

Expand Down
4 changes: 3 additions & 1 deletion sql/core/src/main/scala/org/apache/spark/sql/Column.scala
Original file line number Diff line number Diff line change
Expand Up @@ -152,7 +152,7 @@ class Column(protected[sql] val expr: Expression) extends Logging {
* results into the correct JVM types.
* @since 1.6.0
*/
def as[U : Encoder]: TypedColumn[Any, U] = new TypedColumn[Any, U](expr, encoderFor[U])
def as[U: Encoder]: TypedColumn[Any, U] = new TypedColumn[Any, U](expr, encoderFor[U])

/**
* Extracts a value or values from a complex type.
Expand All @@ -171,6 +171,7 @@ class Column(protected[sql] val expr: Expression) extends Logging {
UnresolvedExtractValue(expr, lit(extraction).expr)
}

// scalastyle:off whitespacebeforetoken
/**
* Unary minus, i.e. negate the expression.
* {{{
Expand Down Expand Up @@ -202,6 +203,7 @@ class Column(protected[sql] val expr: Expression) extends Logging {
* @since 1.3.0
*/
def unary_! : Column = withExpr { Not(expr) }
// scalastyle:on whitespacebeforetoken

/**
* Equality test.
Expand Down
Loading