@@ -31,13 +31,12 @@ import org.apache.spark._
31
31
import org .apache .spark .internal .Logging
32
32
import org .apache .spark .internal .io .{FileCommitProtocol , SparkHadoopWriterUtils }
33
33
import org .apache .spark .internal .io .FileCommitProtocol .TaskCommitMessage
34
- import org .apache .spark .sql .{ Dataset , SparkSession }
34
+ import org .apache .spark .sql .SparkSession
35
35
import org .apache .spark .sql .catalyst .catalog .{BucketSpec , ExternalCatalogUtils }
36
36
import org .apache .spark .sql .catalyst .catalog .CatalogTypes .TablePartitionSpec
37
37
import org .apache .spark .sql .catalyst .expressions ._
38
38
import org .apache .spark .sql .catalyst .plans .physical .HashPartitioning
39
39
import org .apache .spark .sql .catalyst .InternalRow
40
- import org .apache .spark .sql .catalyst .plans .logical .LogicalPlan
41
40
import org .apache .spark .sql .execution .{QueryExecution , SQLExecution , UnsafeKVExternalSorter }
42
41
import org .apache .spark .sql .types .{IntegerType , StringType , StructField , StructType }
43
42
import org .apache .spark .util .{SerializableConfiguration , Utils }
@@ -47,6 +46,13 @@ import org.apache.spark.util.collection.unsafe.sort.UnsafeExternalSorter
47
46
/** A helper object for writing FileFormat data out to a location. */
48
47
object FileFormatWriter extends Logging {
49
48
49
+ /**
50
+ * Max number of files a single task writes out due to file size. In most cases the number of
51
+ * files written should be very small. This is just a safe guard to protect some really bad
52
+ * settings, e.g. maxRecordsPerFile = 1.
53
+ */
54
+ private val MAX_FILE_COUNTER = 1000 * 1000
55
+
50
56
/** Describes how output files should be placed in the filesystem. */
51
57
case class OutputSpec (
52
58
outputPath : String , customPartitionLocations : Map [TablePartitionSpec , String ])
@@ -61,7 +67,8 @@ object FileFormatWriter extends Logging {
61
67
val nonPartitionColumns : Seq [Attribute ],
62
68
val bucketSpec : Option [BucketSpec ],
63
69
val path : String ,
64
- val customPartitionLocations : Map [TablePartitionSpec , String ])
70
+ val customPartitionLocations : Map [TablePartitionSpec , String ],
71
+ val maxRecordsPerFile : Long )
65
72
extends Serializable {
66
73
67
74
assert(AttributeSet (allColumns) == AttributeSet (partitionColumns ++ nonPartitionColumns),
@@ -116,7 +123,10 @@ object FileFormatWriter extends Logging {
116
123
nonPartitionColumns = dataColumns,
117
124
bucketSpec = bucketSpec,
118
125
path = outputSpec.outputPath,
119
- customPartitionLocations = outputSpec.customPartitionLocations)
126
+ customPartitionLocations = outputSpec.customPartitionLocations,
127
+ maxRecordsPerFile = options.get(" maxRecordsPerFile" ).map(_.toLong)
128
+ .getOrElse(sparkSession.sessionState.conf.maxRecordsPerFile)
129
+ )
120
130
121
131
SQLExecution .withNewExecutionId(sparkSession, queryExecution) {
122
132
// This call shouldn't be put into the `try` block below because it only initializes and
@@ -225,32 +235,49 @@ object FileFormatWriter extends Logging {
225
235
taskAttemptContext : TaskAttemptContext ,
226
236
committer : FileCommitProtocol ) extends ExecuteWriteTask {
227
237
228
- private [this ] var outputWriter : OutputWriter = {
238
+ private [this ] var currentWriter : OutputWriter = _
239
+
240
+ private def newOutputWriter (fileCounter : Int ): Unit = {
241
+ val ext = description.outputWriterFactory.getFileExtension(taskAttemptContext)
229
242
val tmpFilePath = committer.newTaskTempFile(
230
243
taskAttemptContext,
231
244
None ,
232
- description.outputWriterFactory.getFileExtension(taskAttemptContext) )
245
+ f " -c $fileCounter %03d " + ext )
233
246
234
- val outputWriter = description.outputWriterFactory.newInstance(
247
+ currentWriter = description.outputWriterFactory.newInstance(
235
248
path = tmpFilePath,
236
249
dataSchema = description.nonPartitionColumns.toStructType,
237
250
context = taskAttemptContext)
238
- outputWriter.initConverter(dataSchema = description.nonPartitionColumns.toStructType)
239
- outputWriter
251
+ currentWriter.initConverter(dataSchema = description.nonPartitionColumns.toStructType)
240
252
}
241
253
242
254
override def execute (iter : Iterator [InternalRow ]): Set [String ] = {
255
+ var fileCounter = 0
256
+ var recordsInFile : Long = 0L
257
+ newOutputWriter(fileCounter)
243
258
while (iter.hasNext) {
259
+ if (description.maxRecordsPerFile > 0 && recordsInFile >= description.maxRecordsPerFile) {
260
+ fileCounter += 1
261
+ assert(fileCounter < MAX_FILE_COUNTER ,
262
+ s " File counter $fileCounter is beyond max value $MAX_FILE_COUNTER" )
263
+
264
+ recordsInFile = 0
265
+ releaseResources()
266
+ newOutputWriter(fileCounter)
267
+ }
268
+
244
269
val internalRow = iter.next()
245
- outputWriter.writeInternal(internalRow)
270
+ currentWriter.writeInternal(internalRow)
271
+ recordsInFile += 1
246
272
}
273
+ releaseResources()
247
274
Set .empty
248
275
}
249
276
250
277
override def releaseResources (): Unit = {
251
- if (outputWriter != null ) {
252
- outputWriter .close()
253
- outputWriter = null
278
+ if (currentWriter != null ) {
279
+ currentWriter .close()
280
+ currentWriter = null
254
281
}
255
282
}
256
283
}
@@ -300,8 +327,15 @@ object FileFormatWriter extends Logging {
300
327
* Open and returns a new OutputWriter given a partition key and optional bucket id.
301
328
* If bucket id is specified, we will append it to the end of the file name, but before the
302
329
* file extension, e.g. part-r-00009-ea518ad4-455a-4431-b471-d24e03814677-00002.gz.parquet
330
+ *
331
+ * @param key vaues for fields consisting of partition keys for the current row
332
+ * @param partString a function that projects the partition values into a string
333
+ * @param fileCounter the number of files that have been written in the past for this specific
334
+ * partition. This is used to limit the max number of records written for a
335
+ * single file. The value should start from 0.
303
336
*/
304
- private def newOutputWriter (key : InternalRow , partString : UnsafeProjection ): OutputWriter = {
337
+ private def newOutputWriter (
338
+ key : InternalRow , partString : UnsafeProjection , fileCounter : Int ): Unit = {
305
339
val partDir =
306
340
if (description.partitionColumns.isEmpty) None else Option (partString(key).getString(0 ))
307
341
@@ -311,7 +345,10 @@ object FileFormatWriter extends Logging {
311
345
} else {
312
346
" "
313
347
}
314
- val ext = bucketId + description.outputWriterFactory.getFileExtension(taskAttemptContext)
348
+
349
+ // This must be in a form that matches our bucketing format. See BucketingUtils.
350
+ val ext = f " $bucketId.c $fileCounter%03d " +
351
+ description.outputWriterFactory.getFileExtension(taskAttemptContext)
315
352
316
353
val customPath = partDir match {
317
354
case Some (dir) =>
@@ -324,12 +361,12 @@ object FileFormatWriter extends Logging {
324
361
} else {
325
362
committer.newTaskTempFile(taskAttemptContext, partDir, ext)
326
363
}
327
- val newWriter = description.outputWriterFactory.newInstance(
364
+
365
+ currentWriter = description.outputWriterFactory.newInstance(
328
366
path = path,
329
367
dataSchema = description.nonPartitionColumns.toStructType,
330
368
context = taskAttemptContext)
331
- newWriter.initConverter(description.nonPartitionColumns.toStructType)
332
- newWriter
369
+ currentWriter.initConverter(description.nonPartitionColumns.toStructType)
333
370
}
334
371
335
372
override def execute (iter : Iterator [InternalRow ]): Set [String ] = {
@@ -349,7 +386,7 @@ object FileFormatWriter extends Logging {
349
386
description.nonPartitionColumns, description.allColumns)
350
387
351
388
// Returns the partition path given a partition key.
352
- val getPartitionString = UnsafeProjection .create(
389
+ val getPartitionStringFunc = UnsafeProjection .create(
353
390
Seq (Concat (partitionStringExpression)), description.partitionColumns)
354
391
355
392
// Sorts the data before write, so that we only need one writer at the same time.
@@ -366,7 +403,6 @@ object FileFormatWriter extends Logging {
366
403
val currentRow = iter.next()
367
404
sorter.insertKV(getSortingKey(currentRow), getOutputRow(currentRow))
368
405
}
369
- logInfo(s " Sorting complete. Writing out partition files one at a time. " )
370
406
371
407
val getBucketingKey : InternalRow => InternalRow = if (sortColumns.isEmpty) {
372
408
identity
@@ -379,30 +415,43 @@ object FileFormatWriter extends Logging {
379
415
val sortedIterator = sorter.sortedIterator()
380
416
381
417
// If anything below fails, we should abort the task.
418
+ var recordsInFile : Long = 0L
419
+ var fileCounter = 0
382
420
var currentKey : UnsafeRow = null
383
421
val updatedPartitions = mutable.Set [String ]()
384
422
while (sortedIterator.next()) {
385
423
val nextKey = getBucketingKey(sortedIterator.getKey).asInstanceOf [UnsafeRow ]
386
424
if (currentKey != nextKey) {
387
- if (currentWriter != null ) {
388
- currentWriter.close()
389
- currentWriter = null
390
- }
425
+ // See a new key - write to a new partition (new file).
391
426
currentKey = nextKey.copy()
392
427
logDebug(s " Writing partition: $currentKey" )
393
428
394
- currentWriter = newOutputWriter(currentKey, getPartitionString)
395
- val partitionPath = getPartitionString(currentKey).getString(0 )
429
+ recordsInFile = 0
430
+ fileCounter = 0
431
+
432
+ releaseResources()
433
+ newOutputWriter(currentKey, getPartitionStringFunc, fileCounter)
434
+ val partitionPath = getPartitionStringFunc(currentKey).getString(0 )
396
435
if (partitionPath.nonEmpty) {
397
436
updatedPartitions.add(partitionPath)
398
437
}
438
+ } else if (description.maxRecordsPerFile > 0 &&
439
+ recordsInFile >= description.maxRecordsPerFile) {
440
+ // Exceeded the threshold in terms of the number of records per file.
441
+ // Create a new file by increasing the file counter.
442
+ recordsInFile = 0
443
+ fileCounter += 1
444
+ assert(fileCounter < MAX_FILE_COUNTER ,
445
+ s " File counter $fileCounter is beyond max value $MAX_FILE_COUNTER" )
446
+
447
+ releaseResources()
448
+ newOutputWriter(currentKey, getPartitionStringFunc, fileCounter)
399
449
}
450
+
400
451
currentWriter.writeInternal(sortedIterator.getValue)
452
+ recordsInFile += 1
401
453
}
402
- if (currentWriter != null ) {
403
- currentWriter.close()
404
- currentWriter = null
405
- }
454
+ releaseResources()
406
455
updatedPartitions.toSet
407
456
}
408
457
0 commit comments