[SPARK-8906][SQL] Move all internal data source classes into execution.datasources.
This way, the sources package contains only public facing interfaces. Author: Reynold Xin <rxin@databricks.com> Closes #7565 from rxin/move-ds and squashes the following commits: 7661aff [Reynold Xin] Mima 9d5196a [Reynold Xin] Rearranged imports. 3dd7174 [Reynold Xin] [SPARK-8906][SQL] Move all internal data source classes into execution.datasources.
This commit is contained in:
parent
9ba7c64dec
commit
60c0ce134d
|
@ -104,6 +104,53 @@ object MimaExcludes {
|
|||
// SPARK-7422 add argmax for sparse vectors
|
||||
ProblemFilters.exclude[MissingMethodProblem](
|
||||
"org.apache.spark.mllib.linalg.Vector.argmax")
|
||||
) ++ Seq(
|
||||
// SPARK-8906 Move all internal data source classes into execution.datasources
|
||||
ProblemFilters.exclude[MissingClassProblem]("org.apache.spark.sql.sources.ResolvedDataSource"),
|
||||
ProblemFilters.exclude[MissingClassProblem]("org.apache.spark.sql.sources.PreInsertCastAndRename$"),
|
||||
ProblemFilters.exclude[MissingClassProblem]("org.apache.spark.sql.sources.CreateTableUsingAsSelect$"),
|
||||
ProblemFilters.exclude[MissingClassProblem]("org.apache.spark.sql.sources.InsertIntoDataSource$"),
|
||||
ProblemFilters.exclude[MissingClassProblem]("org.apache.spark.sql.sources.SqlNewHadoopPartition"),
|
||||
ProblemFilters.exclude[MissingClassProblem]("org.apache.spark.sql.sources.PartitioningUtils$PartitionValues$"),
|
||||
ProblemFilters.exclude[MissingClassProblem]("org.apache.spark.sql.sources.DefaultWriterContainer"),
|
||||
ProblemFilters.exclude[MissingClassProblem]("org.apache.spark.sql.sources.PartitioningUtils$PartitionValues"),
|
||||
ProblemFilters.exclude[MissingClassProblem]("org.apache.spark.sql.sources.RefreshTable$"),
|
||||
ProblemFilters.exclude[MissingClassProblem]("org.apache.spark.sql.sources.CreateTempTableUsing$"),
|
||||
ProblemFilters.exclude[MissingClassProblem]("org.apache.spark.sql.sources.PartitionSpec"),
|
||||
ProblemFilters.exclude[MissingClassProblem]("org.apache.spark.sql.sources.DynamicPartitionWriterContainer"),
|
||||
ProblemFilters.exclude[MissingClassProblem]("org.apache.spark.sql.sources.CreateTableUsingAsSelect"),
|
||||
ProblemFilters.exclude[MissingClassProblem]("org.apache.spark.sql.sources.SqlNewHadoopRDD$"),
|
||||
ProblemFilters.exclude[MissingClassProblem]("org.apache.spark.sql.sources.DescribeCommand$"),
|
||||
ProblemFilters.exclude[MissingClassProblem]("org.apache.spark.sql.sources.PartitioningUtils$"),
|
||||
ProblemFilters.exclude[MissingClassProblem]("org.apache.spark.sql.sources.SqlNewHadoopRDD"),
|
||||
ProblemFilters.exclude[MissingClassProblem]("org.apache.spark.sql.sources.PreInsertCastAndRename"),
|
||||
ProblemFilters.exclude[MissingClassProblem]("org.apache.spark.sql.sources.Partition$"),
|
||||
ProblemFilters.exclude[MissingClassProblem]("org.apache.spark.sql.sources.LogicalRelation$"),
|
||||
ProblemFilters.exclude[MissingClassProblem]("org.apache.spark.sql.sources.PartitioningUtils"),
|
||||
ProblemFilters.exclude[MissingClassProblem]("org.apache.spark.sql.sources.LogicalRelation"),
|
||||
ProblemFilters.exclude[MissingClassProblem]("org.apache.spark.sql.sources.Partition"),
|
||||
ProblemFilters.exclude[MissingClassProblem]("org.apache.spark.sql.sources.BaseWriterContainer"),
|
||||
ProblemFilters.exclude[MissingClassProblem]("org.apache.spark.sql.sources.PreWriteCheck"),
|
||||
ProblemFilters.exclude[MissingClassProblem]("org.apache.spark.sql.sources.CreateTableUsing"),
|
||||
ProblemFilters.exclude[MissingClassProblem]("org.apache.spark.sql.sources.RefreshTable"),
|
||||
ProblemFilters.exclude[MissingClassProblem]("org.apache.spark.sql.sources.SqlNewHadoopRDD$NewHadoopMapPartitionsWithSplitRDD"),
|
||||
ProblemFilters.exclude[MissingClassProblem]("org.apache.spark.sql.sources.DataSourceStrategy$"),
|
||||
ProblemFilters.exclude[MissingClassProblem]("org.apache.spark.sql.sources.CreateTempTableUsing"),
|
||||
ProblemFilters.exclude[MissingClassProblem]("org.apache.spark.sql.sources.CreateTempTableUsingAsSelect$"),
|
||||
ProblemFilters.exclude[MissingClassProblem]("org.apache.spark.sql.sources.CreateTempTableUsingAsSelect"),
|
||||
ProblemFilters.exclude[MissingClassProblem]("org.apache.spark.sql.sources.CreateTableUsing$"),
|
||||
ProblemFilters.exclude[MissingClassProblem]("org.apache.spark.sql.sources.ResolvedDataSource$"),
|
||||
ProblemFilters.exclude[MissingClassProblem]("org.apache.spark.sql.sources.PreWriteCheck$"),
|
||||
ProblemFilters.exclude[MissingClassProblem]("org.apache.spark.sql.sources.InsertIntoDataSource"),
|
||||
ProblemFilters.exclude[MissingClassProblem]("org.apache.spark.sql.sources.InsertIntoHadoopFsRelation"),
|
||||
ProblemFilters.exclude[MissingClassProblem]("org.apache.spark.sql.sources.DDLParser"),
|
||||
ProblemFilters.exclude[MissingClassProblem]("org.apache.spark.sql.sources.CaseInsensitiveMap"),
|
||||
ProblemFilters.exclude[MissingClassProblem]("org.apache.spark.sql.sources.InsertIntoHadoopFsRelation$"),
|
||||
ProblemFilters.exclude[MissingClassProblem]("org.apache.spark.sql.sources.DataSourceStrategy"),
|
||||
ProblemFilters.exclude[MissingClassProblem]("org.apache.spark.sql.sources.SqlNewHadoopRDD$NewHadoopMapPartitionsWithSplitRDD$"),
|
||||
ProblemFilters.exclude[MissingClassProblem]("org.apache.spark.sql.sources.PartitionSpec$"),
|
||||
ProblemFilters.exclude[MissingClassProblem]("org.apache.spark.sql.sources.DescribeCommand"),
|
||||
ProblemFilters.exclude[MissingClassProblem]("org.apache.spark.sql.sources.DDLException")
|
||||
)
|
||||
|
||||
case v if v.startsWith("1.4") =>
|
||||
|
|
|
@ -38,8 +38,8 @@ import org.apache.spark.sql.catalyst.plans.logical.{Filter, _}
|
|||
import org.apache.spark.sql.catalyst.plans.{Inner, JoinType}
|
||||
import org.apache.spark.sql.catalyst.{CatalystTypeConverters, ScalaReflection, SqlParser}
|
||||
import org.apache.spark.sql.execution.{EvaluatePython, ExplainCommand, LogicalRDD}
|
||||
import org.apache.spark.sql.execution.datasources.CreateTableUsingAsSelect
|
||||
import org.apache.spark.sql.json.JacksonGenerator
|
||||
import org.apache.spark.sql.sources.CreateTableUsingAsSelect
|
||||
import org.apache.spark.sql.types._
|
||||
import org.apache.spark.storage.StorageLevel
|
||||
import org.apache.spark.util.Utils
|
||||
|
|
|
@ -20,16 +20,16 @@ package org.apache.spark.sql
|
|||
import java.util.Properties
|
||||
|
||||
import org.apache.hadoop.fs.Path
|
||||
import org.apache.spark.{Logging, Partition}
|
||||
|
||||
import org.apache.spark.{Logging, Partition}
|
||||
import org.apache.spark.annotation.Experimental
|
||||
import org.apache.spark.api.java.JavaRDD
|
||||
import org.apache.spark.deploy.SparkHadoopUtil
|
||||
import org.apache.spark.rdd.RDD
|
||||
import org.apache.spark.sql.execution.datasources.{ResolvedDataSource, LogicalRelation}
|
||||
import org.apache.spark.sql.jdbc.{JDBCPartition, JDBCPartitioningInfo, JDBCRelation}
|
||||
import org.apache.spark.sql.json.JSONRelation
|
||||
import org.apache.spark.sql.parquet.ParquetRelation2
|
||||
import org.apache.spark.sql.sources.{LogicalRelation, ResolvedDataSource}
|
||||
import org.apache.spark.sql.types.StructType
|
||||
|
||||
/**
|
||||
|
|
|
@ -22,8 +22,8 @@ import java.util.Properties
|
|||
import org.apache.spark.annotation.Experimental
|
||||
import org.apache.spark.sql.catalyst.analysis.UnresolvedRelation
|
||||
import org.apache.spark.sql.catalyst.plans.logical.InsertIntoTable
|
||||
import org.apache.spark.sql.execution.datasources.{CreateTableUsingAsSelect, ResolvedDataSource}
|
||||
import org.apache.spark.sql.jdbc.{JDBCWriteDetails, JdbcUtils}
|
||||
import org.apache.spark.sql.sources.{ResolvedDataSource, CreateTableUsingAsSelect}
|
||||
|
||||
|
||||
/**
|
||||
|
|
|
@ -39,8 +39,9 @@ import org.apache.spark.sql.catalyst.optimizer.{DefaultOptimizer, Optimizer}
|
|||
import org.apache.spark.sql.catalyst.plans.logical.{LocalRelation, LogicalPlan}
|
||||
import org.apache.spark.sql.catalyst.rules.RuleExecutor
|
||||
import org.apache.spark.sql.catalyst.{InternalRow, ParserDialect, _}
|
||||
import org.apache.spark.sql.execution.{Filter, _}
|
||||
import org.apache.spark.sql.sources._
|
||||
import org.apache.spark.sql.execution._
|
||||
import org.apache.spark.sql.execution.datasources._
|
||||
import org.apache.spark.sql.sources.BaseRelation
|
||||
import org.apache.spark.sql.types._
|
||||
import org.apache.spark.unsafe.types.UTF8String
|
||||
import org.apache.spark.util.Utils
|
||||
|
@ -146,11 +147,11 @@ class SQLContext(@transient val sparkContext: SparkContext)
|
|||
new Analyzer(catalog, functionRegistry, conf) {
|
||||
override val extendedResolutionRules =
|
||||
ExtractPythonUDFs ::
|
||||
sources.PreInsertCastAndRename ::
|
||||
PreInsertCastAndRename ::
|
||||
Nil
|
||||
|
||||
override val extendedCheckRules = Seq(
|
||||
sources.PreWriteCheck(catalog)
|
||||
datasources.PreWriteCheck(catalog)
|
||||
)
|
||||
}
|
||||
|
||||
|
|
|
@ -17,6 +17,7 @@
|
|||
|
||||
package org.apache.spark.sql.execution
|
||||
|
||||
import org.apache.spark.sql.{SQLContext, Strategy, execution}
|
||||
import org.apache.spark.sql.catalyst.InternalRow
|
||||
import org.apache.spark.sql.catalyst.expressions._
|
||||
import org.apache.spark.sql.catalyst.planning._
|
||||
|
@ -25,10 +26,9 @@ import org.apache.spark.sql.catalyst.plans.logical.{BroadcastHint, LogicalPlan}
|
|||
import org.apache.spark.sql.catalyst.plans.physical._
|
||||
import org.apache.spark.sql.columnar.{InMemoryColumnarTableScan, InMemoryRelation}
|
||||
import org.apache.spark.sql.execution.{DescribeCommand => RunnableDescribeCommand}
|
||||
import org.apache.spark.sql.execution.datasources.{CreateTableUsing, CreateTempTableUsing, DescribeCommand => LogicalDescribeCommand, _}
|
||||
import org.apache.spark.sql.parquet._
|
||||
import org.apache.spark.sql.sources.{CreateTableUsing, CreateTempTableUsing, DescribeCommand => LogicalDescribeCommand, _}
|
||||
import org.apache.spark.sql.types._
|
||||
import org.apache.spark.sql.{SQLContext, Strategy, execution}
|
||||
|
||||
private[sql] abstract class SparkStrategies extends QueryPlanner[SparkPlan] {
|
||||
self: SQLContext#SparkPlanner =>
|
||||
|
|
|
@ -15,24 +15,23 @@
|
|||
* limitations under the License.
|
||||
*/
|
||||
|
||||
package org.apache.spark.sql.sources
|
||||
package org.apache.spark.sql.execution
|
||||
|
||||
import java.text.SimpleDateFormat
|
||||
import java.util.Date
|
||||
|
||||
import org.apache.spark.{Partition => SparkPartition, _}
|
||||
import org.apache.hadoop.conf.{Configurable, Configuration}
|
||||
import org.apache.hadoop.io.Writable
|
||||
import org.apache.hadoop.mapreduce._
|
||||
import org.apache.hadoop.mapreduce.lib.input.{CombineFileSplit, FileSplit}
|
||||
import org.apache.spark.broadcast.Broadcast
|
||||
|
||||
import org.apache.spark.{Partition => SparkPartition, _}
|
||||
import org.apache.spark.annotation.DeveloperApi
|
||||
import org.apache.spark.broadcast.Broadcast
|
||||
import org.apache.spark.deploy.SparkHadoopUtil
|
||||
import org.apache.spark.executor.DataReadMethod
|
||||
import org.apache.spark.mapreduce.SparkHadoopMapReduceUtil
|
||||
import org.apache.spark.rdd.{RDD, HadoopRDD}
|
||||
import org.apache.spark.rdd.NewHadoopRDD.NewHadoopMapPartitionsWithSplitRDD
|
||||
import org.apache.spark.rdd.{HadoopRDD, RDD}
|
||||
import org.apache.spark.storage.StorageLevel
|
||||
import org.apache.spark.util.{SerializableConfiguration, Utils}
|
||||
|
|
@ -15,22 +15,21 @@
|
|||
* limitations under the License.
|
||||
*/
|
||||
|
||||
package org.apache.spark.sql.sources
|
||||
package org.apache.spark.sql.execution.datasources
|
||||
|
||||
import org.apache.spark.{Logging, TaskContext}
|
||||
import org.apache.spark.deploy.SparkHadoopUtil
|
||||
import org.apache.spark.rdd.{MapPartitionsRDD, RDD, UnionRDD}
|
||||
import org.apache.spark.sql._
|
||||
import org.apache.spark.sql.catalyst.InternalRow
|
||||
import org.apache.spark.sql.catalyst.expressions
|
||||
import org.apache.spark.sql.catalyst.{InternalRow, expressions}
|
||||
import org.apache.spark.sql.catalyst.expressions._
|
||||
import org.apache.spark.sql.catalyst.planning.PhysicalOperation
|
||||
import org.apache.spark.sql.catalyst.plans.logical
|
||||
import org.apache.spark.sql.catalyst.plans.logical.LogicalPlan
|
||||
import org.apache.spark.sql.sources._
|
||||
import org.apache.spark.sql.types.{StringType, StructType}
|
||||
import org.apache.spark.sql.{SaveMode, Strategy, execution, sources}
|
||||
import org.apache.spark.util.{SerializableConfiguration, Utils}
|
||||
import org.apache.spark.sql.{SaveMode, Strategy, execution, sources, _}
|
||||
import org.apache.spark.unsafe.types.UTF8String
|
||||
import org.apache.spark.util.{SerializableConfiguration, Utils}
|
||||
|
||||
/**
|
||||
* A Strategy for planning scans over data sources defined using the sources API.
|
|
@ -14,11 +14,12 @@
|
|||
* See the License for the specific language governing permissions and
|
||||
* limitations under the License.
|
||||
*/
|
||||
package org.apache.spark.sql.sources
|
||||
package org.apache.spark.sql.execution.datasources
|
||||
|
||||
import org.apache.spark.sql.catalyst.analysis.MultiInstanceRelation
|
||||
import org.apache.spark.sql.catalyst.expressions.{AttributeReference, AttributeMap}
|
||||
import org.apache.spark.sql.catalyst.plans.logical.{Statistics, LeafNode, LogicalPlan}
|
||||
import org.apache.spark.sql.catalyst.expressions.{AttributeMap, AttributeReference}
|
||||
import org.apache.spark.sql.catalyst.plans.logical.{LeafNode, LogicalPlan, Statistics}
|
||||
import org.apache.spark.sql.sources.BaseRelation
|
||||
|
||||
/**
|
||||
* Used to link a [[BaseRelation]] in to a logical query plan.
|
|
@ -15,9 +15,9 @@
|
|||
* limitations under the License.
|
||||
*/
|
||||
|
||||
package org.apache.spark.sql.sources
|
||||
package org.apache.spark.sql.execution.datasources
|
||||
|
||||
import java.lang.{Double => JDouble, Float => JFloat, Integer => JInteger, Long => JLong}
|
||||
import java.lang.{Double => JDouble, Long => JLong}
|
||||
import java.math.{BigDecimal => JBigDecimal}
|
||||
|
||||
import scala.collection.mutable.ArrayBuffer
|
||||
|
@ -29,6 +29,7 @@ import org.apache.spark.sql.catalyst.InternalRow
|
|||
import org.apache.spark.sql.catalyst.expressions.{Cast, Literal}
|
||||
import org.apache.spark.sql.types._
|
||||
|
||||
|
||||
private[sql] case class Partition(values: InternalRow, path: String)
|
||||
|
||||
private[sql] case class PartitionSpec(partitionColumns: StructType, partitions: Seq[Partition])
|
|
@ -15,7 +15,7 @@
|
|||
* limitations under the License.
|
||||
*/
|
||||
|
||||
package org.apache.spark.sql.sources
|
||||
package org.apache.spark.sql.execution.datasources
|
||||
|
||||
import java.util.{Date, UUID}
|
||||
|
||||
|
@ -24,7 +24,6 @@ import scala.collection.JavaConversions.asScalaIterator
|
|||
import org.apache.hadoop.fs.Path
|
||||
import org.apache.hadoop.mapreduce._
|
||||
import org.apache.hadoop.mapreduce.lib.output.{FileOutputCommitter => MapReduceFileOutputCommitter, FileOutputFormat}
|
||||
|
||||
import org.apache.spark._
|
||||
import org.apache.spark.mapred.SparkHadoopMapRedUtil
|
||||
import org.apache.spark.mapreduce.SparkHadoopMapReduceUtil
|
||||
|
@ -35,9 +34,11 @@ import org.apache.spark.sql.catalyst.expressions.codegen.GenerateProjection
|
|||
import org.apache.spark.sql.catalyst.plans.logical.{LogicalPlan, Project}
|
||||
import org.apache.spark.sql.catalyst.{CatalystTypeConverters, InternalRow}
|
||||
import org.apache.spark.sql.execution.RunnableCommand
|
||||
import org.apache.spark.sql.sources._
|
||||
import org.apache.spark.sql.types.StringType
|
||||
import org.apache.spark.util.SerializableConfiguration
|
||||
|
||||
|
||||
private[sql] case class InsertIntoDataSource(
|
||||
logicalRelation: LogicalRelation,
|
||||
query: LogicalPlan,
|
|
@ -15,23 +15,22 @@
|
|||
* limitations under the License.
|
||||
*/
|
||||
|
||||
package org.apache.spark.sql.sources
|
||||
package org.apache.spark.sql.execution.datasources
|
||||
|
||||
import scala.language.{existentials, implicitConversions}
|
||||
import scala.util.matching.Regex
|
||||
|
||||
import org.apache.hadoop.fs.Path
|
||||
|
||||
import org.apache.spark.Logging
|
||||
import org.apache.spark.deploy.SparkHadoopUtil
|
||||
import org.apache.spark.sql.catalyst.AbstractSparkSQLParser
|
||||
import org.apache.spark.sql.{AnalysisException, DataFrame, SQLContext, SaveMode}
|
||||
import org.apache.spark.sql.catalyst.analysis.UnresolvedRelation
|
||||
import org.apache.spark.sql.catalyst.InternalRow
|
||||
import org.apache.spark.sql.catalyst.expressions.{Attribute, AttributeReference}
|
||||
import org.apache.spark.sql.catalyst.plans.logical._
|
||||
import org.apache.spark.sql.catalyst.{AbstractSparkSQLParser, InternalRow}
|
||||
import org.apache.spark.sql.execution.RunnableCommand
|
||||
import org.apache.spark.sql.sources._
|
||||
import org.apache.spark.sql.types._
|
||||
import org.apache.spark.sql.{AnalysisException, DataFrame, SQLContext, SaveMode}
|
||||
import org.apache.spark.util.Utils
|
||||
|
||||
/**
|
|
@ -15,15 +15,15 @@
|
|||
* limitations under the License.
|
||||
*/
|
||||
|
||||
package org.apache.spark.sql.sources
|
||||
package org.apache.spark.sql.execution.datasources
|
||||
|
||||
import org.apache.spark.sql.{SaveMode, AnalysisException}
|
||||
import org.apache.spark.sql.catalyst.analysis.{EliminateSubQueries, Catalog}
|
||||
import org.apache.spark.sql.catalyst.expressions.{Attribute, Cast, Alias}
|
||||
import org.apache.spark.sql.{AnalysisException, SaveMode}
|
||||
import org.apache.spark.sql.catalyst.analysis.{Catalog, EliminateSubQueries}
|
||||
import org.apache.spark.sql.catalyst.expressions.{Alias, Attribute, Cast}
|
||||
import org.apache.spark.sql.catalyst.plans.logical
|
||||
import org.apache.spark.sql.catalyst.plans.logical._
|
||||
import org.apache.spark.sql.catalyst.rules.Rule
|
||||
import org.apache.spark.sql.types.DataType
|
||||
import org.apache.spark.sql.sources.{BaseRelation, HadoopFsRelation, InsertableRelation}
|
||||
|
||||
/**
|
||||
* A rule to do pre-insert data type casting and field renaming. Before we insert into
|
|
@ -35,15 +35,18 @@ import org.apache.parquet.hadoop.metadata.CompressionCodecName
|
|||
import org.apache.parquet.hadoop.util.ContextUtil
|
||||
import org.apache.parquet.schema.MessageType
|
||||
|
||||
import org.apache.spark.{Logging, Partition => SparkPartition, SparkException}
|
||||
import org.apache.spark.broadcast.Broadcast
|
||||
import org.apache.spark.rdd.RDD
|
||||
import org.apache.spark.rdd.RDD._
|
||||
import org.apache.spark.sql._
|
||||
import org.apache.spark.sql.catalyst.InternalRow
|
||||
import org.apache.spark.sql.execution.{SqlNewHadoopPartition, SqlNewHadoopRDD}
|
||||
import org.apache.spark.sql.execution.datasources.PartitionSpec
|
||||
import org.apache.spark.sql.sources._
|
||||
import org.apache.spark.sql.types.{DataType, StructType}
|
||||
import org.apache.spark.util.{SerializableConfiguration, Utils}
|
||||
import org.apache.spark.{Logging, Partition => SparkPartition, SparkException}
|
||||
|
||||
|
||||
private[sql] class DefaultSource extends HadoopFsRelationProvider {
|
||||
override def createRelation(
|
||||
|
|
|
@ -17,6 +17,10 @@
|
|||
|
||||
package org.apache.spark.sql.sources
|
||||
|
||||
////////////////////////////////////////////////////////////////////////////////////////////////////
|
||||
// This file defines all the filters that we can push down to the data sources.
|
||||
////////////////////////////////////////////////////////////////////////////////////////////////////
|
||||
|
||||
/**
|
||||
* A filter predicate for data sources.
|
||||
*
|
||||
|
|
|
@ -18,7 +18,6 @@
|
|||
package org.apache.spark.sql.sources
|
||||
|
||||
import scala.collection.mutable
|
||||
import scala.collection.mutable.ArrayBuffer
|
||||
import scala.util.Try
|
||||
|
||||
import org.apache.hadoop.conf.Configuration
|
||||
|
@ -33,6 +32,7 @@ import org.apache.spark.sql.catalyst.InternalRow
|
|||
import org.apache.spark.sql.catalyst.expressions._
|
||||
import org.apache.spark.sql.catalyst.expressions.codegen.GenerateMutableProjection
|
||||
import org.apache.spark.sql.execution.RDDConversions
|
||||
import org.apache.spark.sql.execution.datasources.{PartitioningUtils, PartitionSpec, Partition}
|
||||
import org.apache.spark.sql.types.StructType
|
||||
import org.apache.spark.sql._
|
||||
import org.apache.spark.util.SerializableConfiguration
|
||||
|
@ -523,7 +523,7 @@ abstract class HadoopFsRelation private[sql](maybePartitionSpec: Option[Partitio
|
|||
})
|
||||
}
|
||||
|
||||
private[sources] final def buildScan(
|
||||
private[sql] final def buildScan(
|
||||
requiredColumns: Array[String],
|
||||
filters: Array[Filter],
|
||||
inputPaths: Array[String],
|
||||
|
|
|
@ -26,8 +26,8 @@ import org.scalactic.Tolerance._
|
|||
import org.apache.spark.sql.{QueryTest, Row, SQLConf}
|
||||
import org.apache.spark.sql.TestData._
|
||||
import org.apache.spark.sql.catalyst.util.DateTimeUtils
|
||||
import org.apache.spark.sql.execution.datasources.LogicalRelation
|
||||
import org.apache.spark.sql.json.InferSchema.compatibleType
|
||||
import org.apache.spark.sql.sources.LogicalRelation
|
||||
import org.apache.spark.sql.types._
|
||||
import org.apache.spark.util.Utils
|
||||
|
||||
|
|
|
@ -24,7 +24,7 @@ import org.apache.parquet.filter2.predicate.{FilterPredicate, Operators}
|
|||
import org.apache.spark.sql.catalyst.dsl.expressions._
|
||||
import org.apache.spark.sql.catalyst.expressions._
|
||||
import org.apache.spark.sql.catalyst.planning.PhysicalOperation
|
||||
import org.apache.spark.sql.sources.LogicalRelation
|
||||
import org.apache.spark.sql.execution.datasources.LogicalRelation
|
||||
import org.apache.spark.sql.types._
|
||||
import org.apache.spark.sql.{Column, DataFrame, QueryTest, Row, SQLConf}
|
||||
|
||||
|
|
|
@ -28,11 +28,11 @@ import org.apache.hadoop.fs.Path
|
|||
|
||||
import org.apache.spark.sql.catalyst.InternalRow
|
||||
import org.apache.spark.sql.catalyst.expressions.Literal
|
||||
import org.apache.spark.sql.sources.PartitioningUtils._
|
||||
import org.apache.spark.sql.sources.{LogicalRelation, Partition, PartitionSpec}
|
||||
import org.apache.spark.sql.execution.datasources.{LogicalRelation, PartitionSpec, Partition, PartitioningUtils}
|
||||
import org.apache.spark.sql.types._
|
||||
import org.apache.spark.sql._
|
||||
import org.apache.spark.unsafe.types.UTF8String
|
||||
import PartitioningUtils._
|
||||
|
||||
// The data where the partitioning key exists only in the directory structure.
|
||||
case class ParquetData(intField: Int, stringField: String)
|
||||
|
|
|
@ -22,6 +22,7 @@ import java.io.{File, IOException}
|
|||
import org.scalatest.BeforeAndAfterAll
|
||||
|
||||
import org.apache.spark.sql.AnalysisException
|
||||
import org.apache.spark.sql.execution.datasources.DDLException
|
||||
import org.apache.spark.util.Utils
|
||||
|
||||
class CreateTableAsSelectSuite extends DataSourceTest with BeforeAndAfterAll {
|
||||
|
|
|
@ -18,6 +18,7 @@
|
|||
package org.apache.spark.sql.sources
|
||||
|
||||
import org.apache.spark.SparkFunSuite
|
||||
import org.apache.spark.sql.execution.datasources.ResolvedDataSource
|
||||
|
||||
class ResolvedDataSourceSuite extends SparkFunSuite {
|
||||
|
||||
|
|
|
@ -44,9 +44,9 @@ import org.apache.spark.sql.catalyst.ParserDialect
|
|||
import org.apache.spark.sql.catalyst.analysis._
|
||||
import org.apache.spark.sql.catalyst.plans.logical._
|
||||
import org.apache.spark.sql.execution.{ExecutedCommand, ExtractPythonUDFs, SetCommand}
|
||||
import org.apache.spark.sql.execution.datasources.{PreWriteCheck, PreInsertCastAndRename, DataSourceStrategy}
|
||||
import org.apache.spark.sql.hive.client._
|
||||
import org.apache.spark.sql.hive.execution.{DescribeHiveTableCommand, HiveNativeCommand}
|
||||
import org.apache.spark.sql.sources.DataSourceStrategy
|
||||
import org.apache.spark.sql.types._
|
||||
import org.apache.spark.util.Utils
|
||||
|
||||
|
@ -384,11 +384,11 @@ class HiveContext(sc: SparkContext) extends SQLContext(sc) with Logging {
|
|||
catalog.PreInsertionCasts ::
|
||||
ExtractPythonUDFs ::
|
||||
ResolveHiveWindowFunction ::
|
||||
sources.PreInsertCastAndRename ::
|
||||
PreInsertCastAndRename ::
|
||||
Nil
|
||||
|
||||
override val extendedCheckRules = Seq(
|
||||
sources.PreWriteCheck(catalog)
|
||||
PreWriteCheck(catalog)
|
||||
)
|
||||
}
|
||||
|
||||
|
|
|
@ -17,6 +17,8 @@
|
|||
|
||||
package org.apache.spark.sql.hive
|
||||
|
||||
import scala.collection.JavaConversions._
|
||||
|
||||
import com.google.common.base.Objects
|
||||
import com.google.common.cache.{CacheBuilder, CacheLoader, LoadingCache}
|
||||
|
||||
|
@ -28,6 +30,7 @@ import org.apache.hadoop.hive.ql.metadata._
|
|||
import org.apache.hadoop.hive.ql.plan.TableDesc
|
||||
|
||||
import org.apache.spark.Logging
|
||||
import org.apache.spark.sql.{AnalysisException, SQLContext, SaveMode}
|
||||
import org.apache.spark.sql.catalyst.InternalRow
|
||||
import org.apache.spark.sql.catalyst.analysis.{Catalog, MultiInstanceRelation, OverrideCatalog}
|
||||
import org.apache.spark.sql.catalyst.expressions._
|
||||
|
@ -35,14 +38,12 @@ import org.apache.spark.sql.catalyst.planning.PhysicalOperation
|
|||
import org.apache.spark.sql.catalyst.plans.logical
|
||||
import org.apache.spark.sql.catalyst.plans.logical._
|
||||
import org.apache.spark.sql.catalyst.rules._
|
||||
import org.apache.spark.sql.execution.datasources
|
||||
import org.apache.spark.sql.execution.datasources.{Partition => ParquetPartition, PartitionSpec, CreateTableUsingAsSelect, ResolvedDataSource, LogicalRelation}
|
||||
import org.apache.spark.sql.hive.client._
|
||||
import org.apache.spark.sql.parquet.ParquetRelation2
|
||||
import org.apache.spark.sql.sources.{CreateTableUsingAsSelect, LogicalRelation, Partition => ParquetPartition, PartitionSpec, ResolvedDataSource}
|
||||
import org.apache.spark.sql.types._
|
||||
import org.apache.spark.sql.{AnalysisException, SQLContext, SaveMode, sources}
|
||||
|
||||
/* Implicit conversions */
|
||||
import scala.collection.JavaConversions._
|
||||
|
||||
private[hive] class HiveMetastoreCatalog(val client: ClientInterface, hive: HiveContext)
|
||||
extends Catalog with Logging {
|
||||
|
@ -278,7 +279,7 @@ private[hive] class HiveMetastoreCatalog(val client: ClientInterface, hive: Hive
|
|||
parquetRelation.paths.toSet == pathsInMetastore.toSet &&
|
||||
logical.schema.sameType(metastoreSchema) &&
|
||||
parquetRelation.partitionSpec == partitionSpecInMetastore.getOrElse {
|
||||
PartitionSpec(StructType(Nil), Array.empty[sources.Partition])
|
||||
PartitionSpec(StructType(Nil), Array.empty[datasources.Partition])
|
||||
}
|
||||
|
||||
if (useCached) {
|
||||
|
|
|
@ -37,7 +37,7 @@ import org.apache.spark.sql.catalyst.plans.logical
|
|||
import org.apache.spark.sql.catalyst.plans.logical._
|
||||
import org.apache.spark.sql.catalyst.trees.CurrentOrigin
|
||||
import org.apache.spark.sql.execution.ExplainCommand
|
||||
import org.apache.spark.sql.sources.DescribeCommand
|
||||
import org.apache.spark.sql.execution.datasources.DescribeCommand
|
||||
import org.apache.spark.sql.hive.HiveShim._
|
||||
import org.apache.spark.sql.hive.client._
|
||||
import org.apache.spark.sql.hive.execution.{HiveNativeCommand, DropTable, AnalyzeTable, HiveScriptIOSchema}
|
||||
|
|
|
@ -30,9 +30,9 @@ 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.execution.{DescribeCommand => RunnableDescribeCommand, _}
|
||||
import org.apache.spark.sql.execution.datasources.{CreateTableUsing, CreateTableUsingAsSelect, DescribeCommand}
|
||||
import org.apache.spark.sql.hive.execution._
|
||||
import org.apache.spark.sql.parquet.ParquetRelation
|
||||
import org.apache.spark.sql.sources.{CreateTableUsing, CreateTableUsingAsSelect, DescribeCommand}
|
||||
import org.apache.spark.sql.types.StringType
|
||||
|
||||
|
||||
|
|
|
@ -23,6 +23,7 @@ import org.apache.spark.sql.catalyst.expressions.Attribute
|
|||
import org.apache.spark.sql.catalyst.plans.logical.LogicalPlan
|
||||
import org.apache.spark.sql.catalyst.util._
|
||||
import org.apache.spark.sql.execution.RunnableCommand
|
||||
import org.apache.spark.sql.execution.datasources.{ResolvedDataSource, LogicalRelation}
|
||||
import org.apache.spark.sql.hive.HiveContext
|
||||
import org.apache.spark.sql.sources._
|
||||
import org.apache.spark.sql.types._
|
||||
|
|
|
@ -37,6 +37,7 @@ import org.apache.spark.mapred.SparkHadoopMapRedUtil
|
|||
import org.apache.spark.rdd.{HadoopRDD, RDD}
|
||||
import org.apache.spark.sql.catalyst.InternalRow
|
||||
import org.apache.spark.sql.catalyst.expressions._
|
||||
import org.apache.spark.sql.execution.datasources.PartitionSpec
|
||||
import org.apache.spark.sql.hive.{HiveContext, HiveInspectors, HiveMetastoreTypes, HiveShim}
|
||||
import org.apache.spark.sql.sources.{Filter, _}
|
||||
import org.apache.spark.sql.types.StructType
|
||||
|
|
|
@ -28,12 +28,12 @@ import org.apache.hadoop.mapred.InvalidInputException
|
|||
|
||||
import org.apache.spark.Logging
|
||||
import org.apache.spark.sql._
|
||||
import org.apache.spark.sql.execution.datasources.LogicalRelation
|
||||
import org.apache.spark.sql.hive.client.{HiveTable, ManagedTable}
|
||||
import org.apache.spark.sql.hive.test.TestHive
|
||||
import org.apache.spark.sql.hive.test.TestHive._
|
||||
import org.apache.spark.sql.hive.test.TestHive.implicits._
|
||||
import org.apache.spark.sql.parquet.ParquetRelation2
|
||||
import org.apache.spark.sql.sources.LogicalRelation
|
||||
import org.apache.spark.sql.test.SQLTestUtils
|
||||
import org.apache.spark.sql.types._
|
||||
import org.apache.spark.util.Utils
|
||||
|
|
|
@ -22,11 +22,11 @@ import java.io._
|
|||
import org.scalatest.{BeforeAndAfterAll, GivenWhenThen}
|
||||
|
||||
import org.apache.spark.{Logging, SparkFunSuite}
|
||||
import org.apache.spark.sql.sources.DescribeCommand
|
||||
import org.apache.spark.sql.execution.{SetCommand, ExplainCommand}
|
||||
import org.apache.spark.sql.catalyst.planning.PhysicalOperation
|
||||
import org.apache.spark.sql.catalyst.plans.logical._
|
||||
import org.apache.spark.sql.catalyst.util._
|
||||
import org.apache.spark.sql.execution.{SetCommand, ExplainCommand}
|
||||
import org.apache.spark.sql.execution.datasources.DescribeCommand
|
||||
import org.apache.spark.sql.hive.test.TestHive
|
||||
|
||||
/**
|
||||
|
|
|
@ -23,12 +23,12 @@ import org.apache.spark.sql._
|
|||
import org.apache.spark.sql.catalyst.DefaultParserDialect
|
||||
import org.apache.spark.sql.catalyst.analysis.EliminateSubQueries
|
||||
import org.apache.spark.sql.catalyst.errors.DialectException
|
||||
import org.apache.spark.sql.execution.datasources.LogicalRelation
|
||||
import org.apache.spark.sql.hive.test.TestHive
|
||||
import org.apache.spark.sql.hive.test.TestHive._
|
||||
import org.apache.spark.sql.hive.test.TestHive.implicits._
|
||||
import org.apache.spark.sql.hive.{HiveContext, HiveQLDialect, MetastoreRelation}
|
||||
import org.apache.spark.sql.parquet.ParquetRelation2
|
||||
import org.apache.spark.sql.sources.LogicalRelation
|
||||
import org.apache.spark.sql.types._
|
||||
|
||||
case class Nested1(f1: Nested2)
|
||||
|
|
|
@ -23,12 +23,12 @@ import org.scalatest.BeforeAndAfterAll
|
|||
|
||||
import org.apache.spark.sql._
|
||||
import org.apache.spark.sql.execution.{ExecutedCommand, PhysicalRDD}
|
||||
import org.apache.spark.sql.execution.datasources.{InsertIntoDataSource, InsertIntoHadoopFsRelation, LogicalRelation}
|
||||
import org.apache.spark.sql.hive.execution.HiveTableScan
|
||||
import org.apache.spark.sql.hive.test.TestHive
|
||||
import org.apache.spark.sql.hive.test.TestHive._
|
||||
import org.apache.spark.sql.hive.test.TestHive.implicits._
|
||||
import org.apache.spark.sql.parquet.{ParquetRelation2, ParquetTableScan}
|
||||
import org.apache.spark.sql.sources.{InsertIntoDataSource, InsertIntoHadoopFsRelation, LogicalRelation}
|
||||
import org.apache.spark.sql.test.SQLTestUtils
|
||||
import org.apache.spark.sql.types._
|
||||
import org.apache.spark.util.Utils
|
||||
|
|
|
@ -17,10 +17,10 @@
|
|||
|
||||
package org.apache.spark.sql.sources
|
||||
|
||||
import scala.collection.JavaConversions._
|
||||
|
||||
import java.io.File
|
||||
|
||||
import scala.collection.JavaConversions._
|
||||
|
||||
import com.google.common.io.Files
|
||||
import org.apache.hadoop.conf.Configuration
|
||||
import org.apache.hadoop.fs.Path
|
||||
|
@ -31,10 +31,12 @@ import org.apache.parquet.hadoop.ParquetOutputCommitter
|
|||
import org.apache.spark.{SparkException, SparkFunSuite}
|
||||
import org.apache.spark.deploy.SparkHadoopUtil
|
||||
import org.apache.spark.sql._
|
||||
import org.apache.spark.sql.execution.datasources.LogicalRelation
|
||||
import org.apache.spark.sql.hive.test.TestHive
|
||||
import org.apache.spark.sql.test.SQLTestUtils
|
||||
import org.apache.spark.sql.types._
|
||||
|
||||
|
||||
abstract class HadoopFsRelationTest extends QueryTest with SQLTestUtils {
|
||||
override lazy val sqlContext: SQLContext = TestHive
|
||||
|
||||
|
|
Loading…
Reference in a new issue