Skip to content

Commit

Permalink
address comments.
Browse files Browse the repository at this point in the history
  • Loading branch information
gatorsmile committed Sep 11, 2016
1 parent da7deed commit ae335ae
Show file tree
Hide file tree
Showing 8 changed files with 12 additions and 32 deletions.
Original file line number Diff line number Diff line change
Expand Up @@ -17,9 +17,14 @@

package org.apache.spark.sql.catalyst.plans.logical

import org.apache.spark.sql.catalyst.expressions.Attribute

/**
* A logical node that represents a non-query command to be executed by the system. For example,
* commands can be used by parsers to represent DDL operations. Commands, unlike queries, are
* eagerly executed.
*/
trait Command
trait Command extends LeafNode {
final override def children: Seq[LogicalPlan] = Seq.empty
override def output: Seq[Attribute] = Seq.empty
}
Original file line number Diff line number Diff line change
Expand Up @@ -31,10 +31,7 @@ import org.apache.spark.sql.streaming.OutputMode
import org.apache.spark.sql.types.IntegerType

/** A dummy command for testing unsupported operations. */
case class DummyCommand() extends LogicalPlan with Command {
override def output: Seq[Attribute] = Nil
override def children: Seq[LogicalPlan] = Nil
}
case class DummyCommand() extends Command

class UnsupportedOperationsSuite extends SparkFunSuite {

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -129,6 +129,4 @@ case object ResetCommand extends RunnableCommand with Logging {
sparkSession.sessionState.conf.clear()
Seq.empty[Row]
}

override val output: Seq[Attribute] = Seq.empty
}
Original file line number Diff line number Diff line change
Expand Up @@ -19,7 +19,6 @@ package org.apache.spark.sql.execution.command

import org.apache.spark.sql.{Dataset, Row, SparkSession}
import org.apache.spark.sql.catalyst.TableIdentifier
import org.apache.spark.sql.catalyst.expressions.Attribute
import org.apache.spark.sql.catalyst.plans.QueryPlan
import org.apache.spark.sql.catalyst.plans.logical.LogicalPlan

Expand Down Expand Up @@ -47,8 +46,6 @@ case class CacheTableCommand(

Seq.empty[Row]
}

override def output: Seq[Attribute] = Seq.empty
}


Expand All @@ -58,8 +55,6 @@ case class UncacheTableCommand(tableIdent: TableIdentifier) extends RunnableComm
sparkSession.catalog.uncacheTable(tableIdent.quotedString)
Seq.empty[Row]
}

override def output: Seq[Attribute] = Seq.empty
}

/**
Expand All @@ -71,6 +66,4 @@ case object ClearCacheCommand extends RunnableCommand {
sparkSession.catalog.clearCache()
Seq.empty[Row]
}

override def output: Seq[Attribute] = Seq.empty
}
Original file line number Diff line number Diff line change
Expand Up @@ -35,9 +35,7 @@ import org.apache.spark.sql.types._
* A logical command that is executed for its side-effects. `RunnableCommand`s are
* wrapped in `ExecutedCommand` during execution.
*/
trait RunnableCommand extends LogicalPlan with logical.Command {
override def output: Seq[Attribute] = Seq.empty
final override def children: Seq[LogicalPlan] = Seq.empty
trait RunnableCommand extends logical.Command {
def run(sparkSession: SparkSession): Seq[Row]
}

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -59,6 +59,4 @@ case class SetDatabaseCommand(databaseName: String) extends RunnableCommand {
sparkSession.sessionState.catalog.setCurrentDatabase(databaseName)
Seq.empty[Row]
}

override val output: Seq[Attribute] = Seq.empty
}
Original file line number Diff line number Diff line change
Expand Up @@ -70,8 +70,6 @@ case class CreateDatabaseCommand(
ifNotExists)
Seq.empty[Row]
}

override val output: Seq[Attribute] = Seq.empty
}


Expand Down Expand Up @@ -101,8 +99,6 @@ case class DropDatabaseCommand(
sparkSession.sessionState.catalog.dropDatabase(databaseName, ifExists, cascade)
Seq.empty[Row]
}

override val output: Seq[Attribute] = Seq.empty
}

/**
Expand All @@ -126,8 +122,6 @@ case class AlterDatabasePropertiesCommand(

Seq.empty[Row]
}

override val output: Seq[Attribute] = Seq.empty
}

/**
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -20,14 +20,15 @@ package org.apache.spark.sql.execution.datasources
import org.apache.spark.sql._
import org.apache.spark.sql.catalyst.TableIdentifier
import org.apache.spark.sql.catalyst.catalog.CatalogTable
import org.apache.spark.sql.catalyst.expressions.Attribute
import org.apache.spark.sql.catalyst.plans.QueryPlan
import org.apache.spark.sql.catalyst.plans.logical.{Command, LogicalPlan}
import org.apache.spark.sql.execution.command.RunnableCommand
import org.apache.spark.sql.types._

case class CreateTable(tableDesc: CatalogTable, mode: SaveMode, query: Option[LogicalPlan])
extends LogicalPlan with Command {
case class CreateTable(
tableDesc: CatalogTable,
mode: SaveMode,
query: Option[LogicalPlan]) extends Command {
assert(tableDesc.provider.isDefined, "The table to be created must have a provider.")

if (query.isEmpty) {
Expand All @@ -36,10 +37,6 @@ case class CreateTable(tableDesc: CatalogTable, mode: SaveMode, query: Option[Lo
"create table without data insertion can only use ErrorIfExists or Ignore as SaveMode.")
}

override def output: Seq[Attribute] = Seq.empty[Attribute]

override def children: Seq[LogicalPlan] = Seq.empty[LogicalPlan]

override def innerChildren: Seq[QueryPlan[_]] = query.toSeq
}

Expand Down

0 comments on commit ae335ae

Please sign in to comment.