lokeshj1703 commented on code in PR #11710: URL: https://github.com/apache/hudi/pull/11710#discussion_r1745284081
########## hudi-spark-datasource/hudi-spark-common/src/main/scala/org/apache/hudi/HoodieReaderFileIndex.scala: ########## @@ -0,0 +1,118 @@ +/* + * 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 org.apache.hudi.common.table.{HoodieTableConfig, HoodieTableMetaClient} +import org.apache.hudi.keygen.CustomAvroKeyGenerator.PartitionKeyType +import org.apache.hudi.keygen.constant.KeyGeneratorType +import org.apache.hudi.keygen.{CustomAvroKeyGenerator, KeyGenUtils} +import org.apache.spark.sql.SparkSession +import org.apache.spark.sql.catalyst.InternalRow +import org.apache.spark.sql.catalyst.expressions.Expression +import org.apache.spark.sql.execution.datasources.{FileStatusCache, NoopCache, PartitionDirectory} +import org.apache.spark.sql.types.StructType + +class HoodieReaderFileIndex(override val spark: SparkSession, + override val metaClient: HoodieTableMetaClient, + override val schemaSpec: Option[StructType], + override val options: Map[String, String], + @transient override val fileStatusCache: FileStatusCache = NoopCache, + override val includeLogFiles: Boolean = false, + override val shouldEmbedFileSlices: Boolean = false) + extends HoodieFileIndex( + spark = spark, + metaClient = metaClient, + schemaSpec = schemaSpec, + options = options, + fileStatusCache = fileStatusCache, + includeLogFiles = includeLogFiles, + shouldEmbedFileSlices = shouldEmbedFileSlices) { + + /** + * + * Returns set of indices with timestamp partition type. For Timestamp based keygen, there is only one + * partition so index is 0. For custom keygen, it is the partition indices for which partition type is + * timestamp. + */ + private def getTimestampPartitionIndex(): Set[Int] = { + val tableConfig = metaClient.getTableConfig + val keyGeneratorClassNameOpt = Option.apply(tableConfig.getKeyGeneratorClassName) + val recordKeyFieldOpt = common.util.Option.ofNullable(tableConfig.getRawRecordKeyFieldProp) + val keyGeneratorClassName = keyGeneratorClassNameOpt.getOrElse(KeyGenUtils.inferKeyGeneratorType(recordKeyFieldOpt, tableConfig.getPartitionFieldProp).getClassName) + if (keyGeneratorClassName.equals(KeyGeneratorType.TIMESTAMP.getClassName) + || keyGeneratorClassName.equals(KeyGeneratorType.TIMESTAMP_AVRO.getClassName)) { + Set(0) + } else if (keyGeneratorClassName.equals(KeyGeneratorType.CUSTOM.getClassName) + || keyGeneratorClassName.equals(KeyGeneratorType.CUSTOM_AVRO.getClassName)) { + val partitionFields = HoodieTableConfig.getPartitionFieldsForKeyGenerator(tableConfig).orElse(java.util.Collections.emptyList[String]()) + val partitionTypes = CustomAvroKeyGenerator.getPartitionTypes(partitionFields) + var partitionIndexes: Set[Int] = Set.empty + for (i <- 0 until partitionTypes.size()) { + if (partitionTypes.get(i).equals(PartitionKeyType.TIMESTAMP)) { + partitionIndexes = partitionIndexes + i + } + } + partitionIndexes + } else { + Set.empty + } + } + + /** + * Invoked by Spark to fetch list of latest base files per partition. + * + * @param partitionFilters partition column filters + * @param dataFilters data columns filters + * @return list of PartitionDirectory containing partition to base files mapping + */ + override def listFiles(partitionFilters: Seq[Expression], dataFilters: Seq[Expression]): Seq[PartitionDirectory] = { + val partitionDirectories = super.listFiles(partitionFilters, dataFilters) Review Comment: Fixed this code and removed the post processing overhead. File index does the transformation of partition values if flag `useStringTypeForCustomTimestampPartition` is enabled. -- 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. To unsubscribe, e-mail: [email protected] For queries about this service, please contact Infrastructure at: [email protected]
