umehrot2 commented on a change in pull request #2651:
URL: https://github.com/apache/hudi/pull/2651#discussion_r601854740
##########
File path: hudi-common/src/main/java/org/apache/hudi/common/fs/FSUtils.java
##########
@@ -276,6 +276,16 @@ public static void processFiles(FileSystem fs, String
basePathStr, Function<File
}
}
+ public static FileStatus[] getFilesInPartition(HoodieEngineContext
engineContext, HoodieMetadataConfig metadataConfig,
+ String basePathStr, Path
partitionPath) {
+ try (HoodieTableMetadata tableMetadata =
HoodieTableMetadata.create(engineContext,
+ metadataConfig, basePathStr,
FileSystemViewStorageConfig.DEFAULT_VIEW_SPILLABLE_DIR)) {
+ return tableMetadata.getAllFilesInPartition(partitionPath);
+ } catch (Exception e) {
+ throw new HoodieException("Error get files in partition from metadata
table", e);
Review comment:
This internally will fallback to the filesystem in-case metadata based
listing is not enabled. So in the exception message just mention `Unable to get
list of files in partition.`
##########
File path:
hudi-spark-datasource/hudi-spark/src/main/scala/org/apache/hudi/HoodieFileIndex.scala
##########
@@ -0,0 +1,349 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements. See the NOTICE file distributed with
+ * this work for additional information regarding copyright ownership.
+ * The ASF licenses this file to You under the Apache License, Version 2.0
+ * (the "License"); you may not use this file except in compliance with
+ * the License. You may obtain a copy of the License at
+ *
+ * http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+
+package org.apache.hudi
+
+import java.util.Properties
+
+import scala.collection.JavaConverters._
+import org.apache.hadoop.fs.{FileStatus, Path}
+import org.apache.hudi.client.common.HoodieSparkEngineContext
+import org.apache.hudi.common.config.{HoodieMetadataConfig,
SerializableConfiguration}
+import org.apache.hudi.common.engine.HoodieLocalEngineContext
+import org.apache.hudi.common.fs.FSUtils
+import org.apache.hudi.common.model.HoodieBaseFile
+import org.apache.hudi.common.table.{HoodieTableMetaClient,
TableSchemaResolver}
+import org.apache.hudi.common.table.view.HoodieTableFileSystemView
+import org.apache.hudi.config.HoodieWriteConfig
+import org.apache.spark.api.java.JavaSparkContext
+import org.apache.spark.internal.Logging
+import org.apache.spark.sql.catalyst.{InternalRow, expressions}
+import org.apache.spark.sql.SparkSession
+import org.apache.spark.sql.avro.SchemaConverters
+import org.apache.spark.sql.catalyst.expressions.{AttributeReference,
BoundReference, Expression, InterpretedPredicate}
+import org.apache.spark.sql.catalyst.util.{CaseInsensitiveMap, DateTimeUtils}
+import org.apache.spark.sql.execution.datasources.{FileIndex, FileStatusCache,
NoopCache, PartitionDirectory, PartitionUtils}
+import org.apache.spark.sql.internal.SQLConf
+import org.apache.spark.sql.types.StructType
+import org.apache.spark.unsafe.types.UTF8String
+
+import scala.collection.mutable
+
+/**
+ * A File Index which support partition prune for hoodie snapshot and
read-optimized
+ * query.
+ * Main steps to get the file list for query:
+ * 1、Load all files and partition values from the table path.
+ * 2、Do the partition prune by the partition filter condition.
+ *
+ * There are 3 cases for this:
+ * 1、If the partition columns size is equal to the actually partition path
level, we
+ * read it as partitioned table.(e.g partition column is "dt", the partition
path is "2021-03-10")
+ *
+ * 2、If the partition columns size is not equal to the partition path level,
but the partition
+ * column size is "1" (e.g. partition column is "dt", but the partition path
is "2021/03/10"
+ * who'es directory level is 3).We can still read it as a partitioned table.
We will mapping the
+ * partition path (e.g. 2021/03/10) to the only partition column (e.g. "dt").
+ *
+ * 3、Else the the partition columns size is not equal to the partition
directory level and the
+ * size is great than "1" (e.g. partition column is "dt,hh", the partition
path is "2021/03/10/12")
+ * , we read it as a None Partitioned table because we cannot know how to
mapping the partition
+ * path with the partition columns in this case.
+ */
+case class HoodieFileIndex(
+ spark: SparkSession,
+ metaClient: HoodieTableMetaClient,
+ schemaSpec: Option[StructType],
+ options: Map[String, String],
+ @transient fileStatusCache: FileStatusCache = NoopCache)
+ extends FileIndex with Logging {
+
+ private val basePath = metaClient.getBasePath
+
+ @transient private val queryPath = new Path(options.getOrElse("path",
"'path' option required"))
+ /**
+ * Get the schema of the table.
+ */
+ lazy val schema: StructType = schemaSpec.getOrElse({
+ val schemaUtil = new TableSchemaResolver(metaClient)
+ SchemaConverters.toSqlType(schemaUtil.getTableAvroSchema)
+ .dataType.asInstanceOf[StructType]
+ })
+
+ /**
+ * Get the partition schema from the hoodie.properties.
+ */
+ private lazy val _partitionSchemaFromProperties: StructType = {
+ val tableConfig = metaClient.getTableConfig
+ val partitionColumns = tableConfig.getPartitionColumns
+ val nameFieldMap = schema.fields.map(filed => filed.name -> filed).toMap
+
+ if (partitionColumns.isPresent) {
+ val partitionFields = partitionColumns.get().map(column =>
+ nameFieldMap.getOrElse(column, throw new
IllegalArgumentException(s"Cannot find column: '" +
+ s"$column' in the schema[${schema.fields.mkString(",")}]")))
+ new StructType(partitionFields)
+ } else { // If the partition columns have not stored in
hoodie.properites(the table that was
+ // created earlier), we trait it as a none-partitioned table.
+ new StructType()
+ }
+ }
+
+ @transient @volatile private var fileSystemView: HoodieTableFileSystemView =
_
+ @transient @volatile private var cachedAllInputFiles: Array[HoodieBaseFile]
= _
+ @transient @volatile private var cachedFileSize: Long = 0L
+ @transient @volatile private var cachedAllPartitionPaths:
Seq[PartitionRowPath] = _
+
+ @volatile private var queryAsNonePartitionedTable: Boolean = _
+
+ refresh0()
+
+ override def rootPaths: Seq[Path] = queryPath :: Nil
+
+ override def listFiles(partitionFilters: Seq[Expression],
+ dataFilters: Seq[Expression]):
Seq[PartitionDirectory] = {
+ if (queryAsNonePartitionedTable) { // Read as None Partitioned table.
+ Seq(PartitionDirectory(InternalRow.empty, allFiles))
+ } else {
+ // Prune the partition path by the partition filters
+ val prunedPartitions = prunePartition(cachedAllPartitionPaths,
partitionFilters)
+ prunedPartitions.map { partition =>
+ val fileStatues =
fileSystemView.getLatestBaseFiles(partition.partitionPath).iterator()
+ .asScala.toSeq
+ .map(_.getFileStatus)
+ PartitionDirectory(partition.values, fileStatues)
+ }
+ }
+ }
+
+ override def inputFiles: Array[String] = {
+ cachedAllInputFiles.map(_.getFileStatus.getPath.toString)
+ }
+
+ override def refresh(): Unit = {
+ fileStatusCache.invalidateAll()
+ refresh0()
+ }
+
+ private def refresh0(): Unit = {
+ val startTime = System.currentTimeMillis()
+ val partitionFiles = loadPartitionPathFiles()
+ val allFiles = partitionFiles.values.reduceOption(_ ++ _)
+ .getOrElse(Array.empty[FileStatus])
+
+ metaClient.reloadActiveTimeline()
+ val activeInstants =
metaClient.getActiveTimeline.getCommitsTimeline.filterCompletedInstants
+ fileSystemView = new HoodieTableFileSystemView(metaClient, activeInstants,
allFiles)
+ cachedAllInputFiles =
fileSystemView.getLatestBaseFiles.iterator().asScala.toArray
+ cachedAllPartitionPaths = partitionFiles.keys.toSeq
+ cachedFileSize = cachedAllInputFiles.map(_.getFileLen).sum
+
+ // If the partition value contains InternalRow.empty, we query it as a
none partitioned table.
+ queryAsNonePartitionedTable = cachedAllPartitionPaths
+ .exists(p => p.values == InternalRow.empty)
+ val flushSpend = System.currentTimeMillis() - startTime
+ logInfo(s"Refresh for table ${metaClient.getTableConfig.getTableName}," +
+ s" spend: $flushSpend ms")
+ }
+
+ override def sizeInBytes: Long = {
+ cachedFileSize
+ }
+
+ override def partitionSchema: StructType = {
+ if (queryAsNonePartitionedTable) {
+ // If we read it as None Partitioned table, we should not
+ // return the partition schema.
+ new StructType()
+ } else {
+ _partitionSchemaFromProperties
+ }
+ }
+
+ /**
+ * Get the data schema of the table.
+ * @return
+ */
+ def dataSchema: StructType = {
+ val partitionColumns = partitionSchema.fields.map(_.name).toSet
+ StructType(schema.fields.filterNot(f => partitionColumns.contains(f.name)))
+ }
+
+ def allFiles: Seq[FileStatus] = cachedAllInputFiles.map(_.getFileStatus)
+
+ private def prunePartition(partitionPaths: Seq[PartitionRowPath],
+ predicates: Seq[Expression]):
Seq[PartitionRowPath] = {
+
+ val partitionColumnNames = partitionSchema.fields.map(_.name).toSet
+ val partitionPruningPredicates = predicates.filter {
+ _.references.map(_.name).toSet.subsetOf(partitionColumnNames)
+ }
+ if (partitionPruningPredicates.nonEmpty) {
+ val predicate = partitionPruningPredicates.reduce(expressions.And)
+
+ val boundPredicate = InterpretedPredicate(predicate.transform {
+ case a: AttributeReference =>
+ val index = partitionSchema.indexWhere(a.name == _.name)
+ BoundReference(index, partitionSchema(index).dataType, nullable =
true)
+ })
+
+ val partitionPruned = partitionPaths.filter {
Review comment:
Rename to `prunedPartitionPaths`
##########
File path:
hudi-spark-datasource/hudi-spark/src/test/scala/org/apache/hudi/functional/TestCOWDataSource.scala
##########
@@ -619,4 +622,69 @@ class TestCOWDataSource extends HoodieClientTestBase {
.load(basePath + "/*")
assertTrue(recordsReadDF.filter(col("_hoodie_partition_path") =!=
lit("")).count() == 0)
}
+
+ @Test def testQueryCowWithoutStar(): Unit = {
Review comment:
- Rename to `testQueryCOWWithBasePathAndFileIndex`
- Can we make this parameterized as well ?
##########
File path:
hudi-spark-datasource/hudi-spark/src/main/scala/org/apache/hudi/DefaultSource.scala
##########
@@ -79,39 +81,52 @@ class DefaultSource extends RelationProvider
val allPaths = path.map(p => Seq(p)).getOrElse(Seq()) ++ readPaths
val fs = FSUtils.getFs(allPaths.head,
sqlContext.sparkContext.hadoopConfiguration)
- val globPaths = HoodieSparkUtils.checkAndGlobPathIfNecessary(allPaths, fs)
-
- val tablePath = DataSourceUtils.getTablePath(fs, globPaths.toArray)
+ // Use the HoodieFileIndex only if the 'path' has specified with no "*"
contains.
+ // And READ_PATHS_OPT_KEY has not specified.
+ // Or else we use the original way to read hoodie table.
+ val useHoodieFileIndex = path.isDefined && !path.get.contains("*") &&
Review comment:
@pengzhiwei2018 I agree with Vinoth that we should guard this against a
config to start with. Non-star paths do work currently for non-partitioned
tables. We can enable this by default I feel, but just in-case we notice any
issues, atleast customers would have the option to disable it for
non-partitioned tables.
##########
File path:
hudi-spark-datasource/hudi-spark/src/main/scala/org/apache/hudi/HoodieFileIndex.scala
##########
@@ -0,0 +1,349 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements. See the NOTICE file distributed with
+ * this work for additional information regarding copyright ownership.
+ * The ASF licenses this file to You under the Apache License, Version 2.0
+ * (the "License"); you may not use this file except in compliance with
+ * the License. You may obtain a copy of the License at
+ *
+ * http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+
+package org.apache.hudi
+
+import java.util.Properties
+
+import scala.collection.JavaConverters._
+import org.apache.hadoop.fs.{FileStatus, Path}
+import org.apache.hudi.client.common.HoodieSparkEngineContext
+import org.apache.hudi.common.config.{HoodieMetadataConfig,
SerializableConfiguration}
+import org.apache.hudi.common.engine.HoodieLocalEngineContext
+import org.apache.hudi.common.fs.FSUtils
+import org.apache.hudi.common.model.HoodieBaseFile
+import org.apache.hudi.common.table.{HoodieTableMetaClient,
TableSchemaResolver}
+import org.apache.hudi.common.table.view.HoodieTableFileSystemView
+import org.apache.hudi.config.HoodieWriteConfig
+import org.apache.spark.api.java.JavaSparkContext
+import org.apache.spark.internal.Logging
+import org.apache.spark.sql.catalyst.{InternalRow, expressions}
+import org.apache.spark.sql.SparkSession
+import org.apache.spark.sql.avro.SchemaConverters
+import org.apache.spark.sql.catalyst.expressions.{AttributeReference,
BoundReference, Expression, InterpretedPredicate}
+import org.apache.spark.sql.catalyst.util.{CaseInsensitiveMap, DateTimeUtils}
+import org.apache.spark.sql.execution.datasources.{FileIndex, FileStatusCache,
NoopCache, PartitionDirectory, PartitionUtils}
+import org.apache.spark.sql.internal.SQLConf
+import org.apache.spark.sql.types.StructType
+import org.apache.spark.unsafe.types.UTF8String
+
+import scala.collection.mutable
+
+/**
+ * A File Index which support partition prune for hoodie snapshot and
read-optimized
+ * query.
+ * Main steps to get the file list for query:
+ * 1、Load all files and partition values from the table path.
+ * 2、Do the partition prune by the partition filter condition.
+ *
+ * There are 3 cases for this:
+ * 1、If the partition columns size is equal to the actually partition path
level, we
+ * read it as partitioned table.(e.g partition column is "dt", the partition
path is "2021-03-10")
+ *
+ * 2、If the partition columns size is not equal to the partition path level,
but the partition
+ * column size is "1" (e.g. partition column is "dt", but the partition path
is "2021/03/10"
+ * who'es directory level is 3).We can still read it as a partitioned table.
We will mapping the
+ * partition path (e.g. 2021/03/10) to the only partition column (e.g. "dt").
+ *
+ * 3、Else the the partition columns size is not equal to the partition
directory level and the
+ * size is great than "1" (e.g. partition column is "dt,hh", the partition
path is "2021/03/10/12")
+ * , we read it as a None Partitioned table because we cannot know how to
mapping the partition
+ * path with the partition columns in this case.
+ */
+case class HoodieFileIndex(
+ spark: SparkSession,
+ metaClient: HoodieTableMetaClient,
+ schemaSpec: Option[StructType],
+ options: Map[String, String],
+ @transient fileStatusCache: FileStatusCache = NoopCache)
+ extends FileIndex with Logging {
+
+ private val basePath = metaClient.getBasePath
+
+ @transient private val queryPath = new Path(options.getOrElse("path",
"'path' option required"))
+ /**
+ * Get the schema of the table.
+ */
+ lazy val schema: StructType = schemaSpec.getOrElse({
+ val schemaUtil = new TableSchemaResolver(metaClient)
+ SchemaConverters.toSqlType(schemaUtil.getTableAvroSchema)
+ .dataType.asInstanceOf[StructType]
+ })
+
+ /**
+ * Get the partition schema from the hoodie.properties.
+ */
+ private lazy val _partitionSchemaFromProperties: StructType = {
+ val tableConfig = metaClient.getTableConfig
+ val partitionColumns = tableConfig.getPartitionColumns
+ val nameFieldMap = schema.fields.map(filed => filed.name -> filed).toMap
+
+ if (partitionColumns.isPresent) {
+ val partitionFields = partitionColumns.get().map(column =>
+ nameFieldMap.getOrElse(column, throw new
IllegalArgumentException(s"Cannot find column: '" +
+ s"$column' in the schema[${schema.fields.mkString(",")}]")))
+ new StructType(partitionFields)
+ } else { // If the partition columns have not stored in
hoodie.properites(the table that was
+ // created earlier), we trait it as a none-partitioned table.
Review comment:
- Change ` we trait it as a none-partitioned table.` => `we treat it as
a non-partitioned table.`
- I see at several places in the code you have used the term `none
partitioned`. Please change it to `non partitioned`
- Lets do an info log here that `No partition columns available from
hoodie.properties. Partition pruning will not work.`
##########
File path:
hudi-spark-datasource/hudi-spark/src/main/scala/org/apache/hudi/HoodieFileIndex.scala
##########
@@ -0,0 +1,349 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements. See the NOTICE file distributed with
+ * this work for additional information regarding copyright ownership.
+ * The ASF licenses this file to You under the Apache License, Version 2.0
+ * (the "License"); you may not use this file except in compliance with
+ * the License. You may obtain a copy of the License at
+ *
+ * http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+
+package org.apache.hudi
+
+import java.util.Properties
+
+import scala.collection.JavaConverters._
+import org.apache.hadoop.fs.{FileStatus, Path}
+import org.apache.hudi.client.common.HoodieSparkEngineContext
+import org.apache.hudi.common.config.{HoodieMetadataConfig,
SerializableConfiguration}
+import org.apache.hudi.common.engine.HoodieLocalEngineContext
+import org.apache.hudi.common.fs.FSUtils
+import org.apache.hudi.common.model.HoodieBaseFile
+import org.apache.hudi.common.table.{HoodieTableMetaClient,
TableSchemaResolver}
+import org.apache.hudi.common.table.view.HoodieTableFileSystemView
+import org.apache.hudi.config.HoodieWriteConfig
+import org.apache.spark.api.java.JavaSparkContext
+import org.apache.spark.internal.Logging
+import org.apache.spark.sql.catalyst.{InternalRow, expressions}
+import org.apache.spark.sql.SparkSession
+import org.apache.spark.sql.avro.SchemaConverters
+import org.apache.spark.sql.catalyst.expressions.{AttributeReference,
BoundReference, Expression, InterpretedPredicate}
+import org.apache.spark.sql.catalyst.util.{CaseInsensitiveMap, DateTimeUtils}
+import org.apache.spark.sql.execution.datasources.{FileIndex, FileStatusCache,
NoopCache, PartitionDirectory, PartitionUtils}
+import org.apache.spark.sql.internal.SQLConf
+import org.apache.spark.sql.types.StructType
+import org.apache.spark.unsafe.types.UTF8String
+
+import scala.collection.mutable
+
+/**
+ * A File Index which support partition prune for hoodie snapshot and
read-optimized
Review comment:
Your javadoc formatting is incorrect at all places. Can you please fix
that ?
##########
File path:
hudi-spark-datasource/hudi-spark/src/test/scala/org/apache/hudi/functional/TestMORDataSource.scala
##########
@@ -562,4 +564,25 @@ class TestMORDataSource extends HoodieClientTestBase {
df.show(1)
df.select("_hoodie_commit_seqno", "fare.amount", "fare.currency",
"tip_history").show(1)
}
+
+ @Test def testQueryMorWithoutStar(): Unit = {
Review comment:
- Rename to `testQueryMORWithBasePathAndFileIndex`
- Can we make this parameterized and more elaborate like the test you have
written in COW testsuite ?
##########
File path:
hudi-spark-datasource/hudi-spark/src/main/scala/org/apache/hudi/HoodieFileIndex.scala
##########
@@ -0,0 +1,349 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements. See the NOTICE file distributed with
+ * this work for additional information regarding copyright ownership.
+ * The ASF licenses this file to You under the Apache License, Version 2.0
+ * (the "License"); you may not use this file except in compliance with
+ * the License. You may obtain a copy of the License at
+ *
+ * http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+
+package org.apache.hudi
+
+import java.util.Properties
+
+import scala.collection.JavaConverters._
+import org.apache.hadoop.fs.{FileStatus, Path}
+import org.apache.hudi.client.common.HoodieSparkEngineContext
+import org.apache.hudi.common.config.{HoodieMetadataConfig,
SerializableConfiguration}
+import org.apache.hudi.common.engine.HoodieLocalEngineContext
+import org.apache.hudi.common.fs.FSUtils
+import org.apache.hudi.common.model.HoodieBaseFile
+import org.apache.hudi.common.table.{HoodieTableMetaClient,
TableSchemaResolver}
+import org.apache.hudi.common.table.view.HoodieTableFileSystemView
+import org.apache.hudi.config.HoodieWriteConfig
+import org.apache.spark.api.java.JavaSparkContext
+import org.apache.spark.internal.Logging
+import org.apache.spark.sql.catalyst.{InternalRow, expressions}
+import org.apache.spark.sql.SparkSession
+import org.apache.spark.sql.avro.SchemaConverters
+import org.apache.spark.sql.catalyst.expressions.{AttributeReference,
BoundReference, Expression, InterpretedPredicate}
+import org.apache.spark.sql.catalyst.util.{CaseInsensitiveMap, DateTimeUtils}
+import org.apache.spark.sql.execution.datasources.{FileIndex, FileStatusCache,
NoopCache, PartitionDirectory, PartitionUtils}
+import org.apache.spark.sql.internal.SQLConf
+import org.apache.spark.sql.types.StructType
+import org.apache.spark.unsafe.types.UTF8String
+
+import scala.collection.mutable
+
+/**
+ * A File Index which support partition prune for hoodie snapshot and
read-optimized
+ * query.
+ * Main steps to get the file list for query:
+ * 1、Load all files and partition values from the table path.
+ * 2、Do the partition prune by the partition filter condition.
+ *
+ * There are 3 cases for this:
+ * 1、If the partition columns size is equal to the actually partition path
level, we
+ * read it as partitioned table.(e.g partition column is "dt", the partition
path is "2021-03-10")
+ *
+ * 2、If the partition columns size is not equal to the partition path level,
but the partition
+ * column size is "1" (e.g. partition column is "dt", but the partition path
is "2021/03/10"
+ * who'es directory level is 3).We can still read it as a partitioned table.
We will mapping the
+ * partition path (e.g. 2021/03/10) to the only partition column (e.g. "dt").
+ *
+ * 3、Else the the partition columns size is not equal to the partition
directory level and the
+ * size is great than "1" (e.g. partition column is "dt,hh", the partition
path is "2021/03/10/12")
+ * , we read it as a None Partitioned table because we cannot know how to
mapping the partition
+ * path with the partition columns in this case.
Review comment:
In this javadoc, we need to mention that this contains copied code from
`prunePartition` function in `PartitioningAwareFileIndex` of Spark. Possibly
reference the javadoc to the Spark class.
##########
File path:
hudi-spark-datasource/hudi-spark/src/test/scala/org/apache/hudi/functional/TestDataSourceForBootstrap.scala
##########
@@ -149,7 +153,8 @@ class TestDataSourceForBootstrap {
.save(srcPath)
// Perform bootstrap
- val commitInstantTime1 =
runMetadataBootstrapAndVerifyCommit(DataSourceWriteOptions.COW_TABLE_TYPE_OPT_VAL)
+ val commitInstantTime1 = runMetadataBootstrapAndVerifyCommit(
+ DataSourceWriteOptions.COW_TABLE_TYPE_OPT_VAL, Some("datestr"))
Review comment:
For this test as well we should test with base path.
##########
File path:
hudi-spark-datasource/hudi-spark/src/test/scala/org/apache/hudi/functional/TestDataSourceForBootstrap.scala
##########
@@ -130,6 +130,10 @@ class TestDataSourceForBootstrap {
hoodieROViewDF1 = spark.read.format("hudi").load(basePath + "/*")
assertEquals(numRecords, hoodieROViewDF1.count())
assertEquals(numRecordsUpdate, hoodieROViewDF1.filter(s"timestamp ==
$updateTimestamp").count())
+ // Read without *
+ val hoodieROViewDF1WithoutStar = spark.read.format("hudi").load(basePath)
Review comment:
Rename to `hoodieROViewDF1WithBasePath`
##########
File path:
hudi-spark-datasource/hudi-spark/src/main/scala/org/apache/hudi/HoodieFileIndex.scala
##########
@@ -0,0 +1,349 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements. See the NOTICE file distributed with
+ * this work for additional information regarding copyright ownership.
+ * The ASF licenses this file to You under the Apache License, Version 2.0
+ * (the "License"); you may not use this file except in compliance with
+ * the License. You may obtain a copy of the License at
+ *
+ * http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+
+package org.apache.hudi
+
+import java.util.Properties
+
+import scala.collection.JavaConverters._
+import org.apache.hadoop.fs.{FileStatus, Path}
+import org.apache.hudi.client.common.HoodieSparkEngineContext
+import org.apache.hudi.common.config.{HoodieMetadataConfig,
SerializableConfiguration}
+import org.apache.hudi.common.engine.HoodieLocalEngineContext
+import org.apache.hudi.common.fs.FSUtils
+import org.apache.hudi.common.model.HoodieBaseFile
+import org.apache.hudi.common.table.{HoodieTableMetaClient,
TableSchemaResolver}
+import org.apache.hudi.common.table.view.HoodieTableFileSystemView
+import org.apache.hudi.config.HoodieWriteConfig
+import org.apache.spark.api.java.JavaSparkContext
+import org.apache.spark.internal.Logging
+import org.apache.spark.sql.catalyst.{InternalRow, expressions}
+import org.apache.spark.sql.SparkSession
+import org.apache.spark.sql.avro.SchemaConverters
+import org.apache.spark.sql.catalyst.expressions.{AttributeReference,
BoundReference, Expression, InterpretedPredicate}
+import org.apache.spark.sql.catalyst.util.{CaseInsensitiveMap, DateTimeUtils}
+import org.apache.spark.sql.execution.datasources.{FileIndex, FileStatusCache,
NoopCache, PartitionDirectory, PartitionUtils}
+import org.apache.spark.sql.internal.SQLConf
+import org.apache.spark.sql.types.StructType
+import org.apache.spark.unsafe.types.UTF8String
+
+import scala.collection.mutable
+
+/**
+ * A File Index which support partition prune for hoodie snapshot and
read-optimized
+ * query.
+ * Main steps to get the file list for query:
+ * 1、Load all files and partition values from the table path.
+ * 2、Do the partition prune by the partition filter condition.
+ *
+ * There are 3 cases for this:
+ * 1、If the partition columns size is equal to the actually partition path
level, we
+ * read it as partitioned table.(e.g partition column is "dt", the partition
path is "2021-03-10")
+ *
+ * 2、If the partition columns size is not equal to the partition path level,
but the partition
+ * column size is "1" (e.g. partition column is "dt", but the partition path
is "2021/03/10"
+ * who'es directory level is 3).We can still read it as a partitioned table.
We will mapping the
+ * partition path (e.g. 2021/03/10) to the only partition column (e.g. "dt").
+ *
+ * 3、Else the the partition columns size is not equal to the partition
directory level and the
+ * size is great than "1" (e.g. partition column is "dt,hh", the partition
path is "2021/03/10/12")
+ * , we read it as a None Partitioned table because we cannot know how to
mapping the partition
+ * path with the partition columns in this case.
+ */
+case class HoodieFileIndex(
+ spark: SparkSession,
+ metaClient: HoodieTableMetaClient,
+ schemaSpec: Option[StructType],
+ options: Map[String, String],
+ @transient fileStatusCache: FileStatusCache = NoopCache)
+ extends FileIndex with Logging {
+
+ private val basePath = metaClient.getBasePath
+
+ @transient private val queryPath = new Path(options.getOrElse("path",
"'path' option required"))
+ /**
+ * Get the schema of the table.
+ */
+ lazy val schema: StructType = schemaSpec.getOrElse({
+ val schemaUtil = new TableSchemaResolver(metaClient)
+ SchemaConverters.toSqlType(schemaUtil.getTableAvroSchema)
+ .dataType.asInstanceOf[StructType]
+ })
+
+ /**
+ * Get the partition schema from the hoodie.properties.
+ */
+ private lazy val _partitionSchemaFromProperties: StructType = {
+ val tableConfig = metaClient.getTableConfig
+ val partitionColumns = tableConfig.getPartitionColumns
+ val nameFieldMap = schema.fields.map(filed => filed.name -> filed).toMap
+
+ if (partitionColumns.isPresent) {
+ val partitionFields = partitionColumns.get().map(column =>
+ nameFieldMap.getOrElse(column, throw new
IllegalArgumentException(s"Cannot find column: '" +
+ s"$column' in the schema[${schema.fields.mkString(",")}]")))
+ new StructType(partitionFields)
+ } else { // If the partition columns have not stored in
hoodie.properites(the table that was
+ // created earlier), we trait it as a none-partitioned table.
+ new StructType()
+ }
+ }
+
+ @transient @volatile private var fileSystemView: HoodieTableFileSystemView =
_
+ @transient @volatile private var cachedAllInputFiles: Array[HoodieBaseFile]
= _
+ @transient @volatile private var cachedFileSize: Long = 0L
+ @transient @volatile private var cachedAllPartitionPaths:
Seq[PartitionRowPath] = _
+
+ @volatile private var queryAsNonePartitionedTable: Boolean = _
+
+ refresh0()
+
+ override def rootPaths: Seq[Path] = queryPath :: Nil
+
+ override def listFiles(partitionFilters: Seq[Expression],
+ dataFilters: Seq[Expression]):
Seq[PartitionDirectory] = {
+ if (queryAsNonePartitionedTable) { // Read as None Partitioned table.
+ Seq(PartitionDirectory(InternalRow.empty, allFiles))
+ } else {
+ // Prune the partition path by the partition filters
+ val prunedPartitions = prunePartition(cachedAllPartitionPaths,
partitionFilters)
+ prunedPartitions.map { partition =>
+ val fileStatues =
fileSystemView.getLatestBaseFiles(partition.partitionPath).iterator()
+ .asScala.toSeq
+ .map(_.getFileStatus)
+ PartitionDirectory(partition.values, fileStatues)
+ }
+ }
+ }
+
+ override def inputFiles: Array[String] = {
+ cachedAllInputFiles.map(_.getFileStatus.getPath.toString)
+ }
+
+ override def refresh(): Unit = {
+ fileStatusCache.invalidateAll()
+ refresh0()
+ }
+
+ private def refresh0(): Unit = {
+ val startTime = System.currentTimeMillis()
+ val partitionFiles = loadPartitionPathFiles()
+ val allFiles = partitionFiles.values.reduceOption(_ ++ _)
+ .getOrElse(Array.empty[FileStatus])
+
+ metaClient.reloadActiveTimeline()
+ val activeInstants =
metaClient.getActiveTimeline.getCommitsTimeline.filterCompletedInstants
+ fileSystemView = new HoodieTableFileSystemView(metaClient, activeInstants,
allFiles)
+ cachedAllInputFiles =
fileSystemView.getLatestBaseFiles.iterator().asScala.toArray
+ cachedAllPartitionPaths = partitionFiles.keys.toSeq
+ cachedFileSize = cachedAllInputFiles.map(_.getFileLen).sum
+
+ // If the partition value contains InternalRow.empty, we query it as a
none partitioned table.
+ queryAsNonePartitionedTable = cachedAllPartitionPaths
+ .exists(p => p.values == InternalRow.empty)
+ val flushSpend = System.currentTimeMillis() - startTime
+ logInfo(s"Refresh for table ${metaClient.getTableConfig.getTableName}," +
+ s" spend: $flushSpend ms")
+ }
+
+ override def sizeInBytes: Long = {
+ cachedFileSize
+ }
+
+ override def partitionSchema: StructType = {
+ if (queryAsNonePartitionedTable) {
+ // If we read it as None Partitioned table, we should not
+ // return the partition schema.
+ new StructType()
+ } else {
+ _partitionSchemaFromProperties
+ }
+ }
+
+ /**
+ * Get the data schema of the table.
+ * @return
+ */
+ def dataSchema: StructType = {
+ val partitionColumns = partitionSchema.fields.map(_.name).toSet
+ StructType(schema.fields.filterNot(f => partitionColumns.contains(f.name)))
+ }
+
+ def allFiles: Seq[FileStatus] = cachedAllInputFiles.map(_.getFileStatus)
+
+ private def prunePartition(partitionPaths: Seq[PartitionRowPath],
+ predicates: Seq[Expression]):
Seq[PartitionRowPath] = {
+
+ val partitionColumnNames = partitionSchema.fields.map(_.name).toSet
+ val partitionPruningPredicates = predicates.filter {
+ _.references.map(_.name).toSet.subsetOf(partitionColumnNames)
+ }
+ if (partitionPruningPredicates.nonEmpty) {
+ val predicate = partitionPruningPredicates.reduce(expressions.And)
+
+ val boundPredicate = InterpretedPredicate(predicate.transform {
+ case a: AttributeReference =>
+ val index = partitionSchema.indexWhere(a.name == _.name)
+ BoundReference(index, partitionSchema(index).dataType, nullable =
true)
+ })
+
+ val partitionPruned = partitionPaths.filter {
+ case PartitionRowPath(values, _) => boundPredicate.eval(values)
+ }
+ logInfo(s"Total partition size is: ${partitionPaths.size}," +
+ s" after partition prune size is: ${partitionPruned.size}")
+ partitionPruned
+ } else {
+ partitionPaths
+ }
+ }
+
+ /**
+ * Load all partition paths and it's files under the query table path.
+ */
+ private def loadPartitionPathFiles(): Map[PartitionRowPath,
Array[FileStatus]] = {
+ val sparkEngine = new HoodieSparkEngineContext(new
JavaSparkContext(spark.sparkContext))
+ val properties = new Properties()
+ properties.putAll(options.asJava)
+ val metadataConfig =
HoodieMetadataConfig.newBuilder.fromProperties(properties).build()
+
+ val queryPartitionPath = FSUtils.getRelativePartitionPath(new
Path(basePath), queryPath)
+ // Load all the partition path from the basePath, and filter by the query
partition path.
+ val partitionPaths = FSUtils.getAllPartitionPaths(sparkEngine,
metadataConfig, basePath).asScala
+ .filter(_.startsWith(queryPartitionPath))
Review comment:
Can you add a `TODO` here that we should be able to directly get
partition paths under the `querypath` instead of listing all partitions under
the `basepath` and then filtering. Will be a good optimization for tables with
large number of partitions.
##########
File path:
hudi-spark-datasource/hudi-spark/src/test/scala/org/apache/hudi/TestHoodieFileIndex.scala
##########
@@ -0,0 +1,219 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements. See the NOTICE file distributed with
+ * this work for additional information regarding copyright ownership.
+ * The ASF licenses this file to You under the Apache License, Version 2.0
+ * (the "License"); you may not use this file except in compliance with
+ * the License. You may obtain a copy of the License at
+ *
+ * http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+
+package org.apache.hudi
+
+import java.net.URLEncoder
+
+import org.apache.hudi.DataSourceWriteOptions.{KEYGENERATOR_CLASS_OPT_KEY,
PARTITIONPATH_FIELD_OPT_KEY, PRECOMBINE_FIELD_OPT_KEY, RECORDKEY_FIELD_OPT_KEY}
+import org.apache.hudi.common.config.HoodieMetadataConfig
+import org.apache.hudi.common.table.HoodieTableMetaClient
+import org.apache.hudi.common.table.view.HoodieTableFileSystemView
+import org.apache.hudi.common.testutils.HoodieTestDataGenerator
+import org.apache.hudi.common.testutils.RawTripTestPayload.recordsToStrings
+import org.apache.hudi.config.HoodieWriteConfig
+import org.apache.hudi.keygen.ComplexKeyGenerator
+import org.apache.hudi.testutils.HoodieClientTestBase
+import org.apache.spark.sql.catalyst.InternalRow
+import org.apache.spark.sql.{SaveMode, SparkSession}
+import org.apache.spark.sql.catalyst.expressions.{And, AttributeReference,
EqualTo, GreaterThanOrEqual, LessThan, Literal}
+import org.apache.spark.sql.execution.datasources.PartitionDirectory
+import org.apache.spark.sql.types.StringType
+import org.junit.jupiter.api.Assertions.assertEquals
+import org.junit.jupiter.api.{BeforeEach, Test}
+import org.junit.jupiter.params.ParameterizedTest
+import org.junit.jupiter.params.provider.ValueSource
+
+import scala.collection.JavaConversions._
+import scala.collection.JavaConverters._
+
+class TestHoodieFileIndex extends HoodieClientTestBase {
+
+ var spark: SparkSession = _
+ val commonOpts = Map(
+ "hoodie.insert.shuffle.parallelism" -> "4",
+ "hoodie.upsert.shuffle.parallelism" -> "4",
+ DataSourceWriteOptions.RECORDKEY_FIELD_OPT_KEY -> "_row_key",
+ DataSourceWriteOptions.PARTITIONPATH_FIELD_OPT_KEY -> "partition",
+ DataSourceWriteOptions.PRECOMBINE_FIELD_OPT_KEY -> "timestamp",
+ HoodieWriteConfig.TABLE_NAME -> "hoodie_test"
+ )
+
+ @BeforeEach override def setUp() {
+ initPath()
+ initSparkContexts()
+ spark = sqlContext.sparkSession
+ initTestDataGenerator()
+ initFileSystem()
+ initMetaClient()
+ }
+
+ @Test def testPartitionSchema(): Unit = {
Review comment:
Please turn this into `ParameterizedTest` as well with
`DataSourceWriteOptions.URL_ENCODE_PARTITIONING_OPT_KEY` as `true,false`
##########
File path:
hudi-spark-datasource/hudi-spark/src/main/scala/org/apache/hudi/DefaultSource.scala
##########
@@ -79,39 +81,52 @@ class DefaultSource extends RelationProvider
val allPaths = path.map(p => Seq(p)).getOrElse(Seq()) ++ readPaths
val fs = FSUtils.getFs(allPaths.head,
sqlContext.sparkContext.hadoopConfiguration)
- val globPaths = HoodieSparkUtils.checkAndGlobPathIfNecessary(allPaths, fs)
-
- val tablePath = DataSourceUtils.getTablePath(fs, globPaths.toArray)
+ // Use the HoodieFileIndex only if the 'path' has specified with no "*"
contains.
Review comment:
Please change this to: `Use the HoodieFileIndex only if the 'path' is
not globbed`
--
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
For queries about this service, please contact Infrastructure at:
[email protected]