1818package org .apache .spark .sql .streaming
1919
2020import java .io .{File , IOException }
21- import java .nio .file .Files
21+ import java .nio .file .{ Files , Paths }
2222import java .util .Locale
2323
2424import scala .collection .mutable .ArrayBuffer
@@ -27,7 +27,7 @@ import scala.jdk.CollectionConverters._
2727import org .apache .hadoop .fs .{FileStatus , Path , RawLocalFileSystem }
2828import org .apache .hadoop .mapreduce .JobContext
2929
30- import org .apache .spark .SparkConf
30+ import org .apache .spark .{ SparkConf , SparkException }
3131import org .apache .spark .deploy .SparkHadoopUtil
3232import org .apache .spark .internal .io .FileCommitProtocol
3333import org .apache .spark .paths .SparkPath
@@ -36,6 +36,7 @@ import org.apache.spark.sql.{AnalysisException, DataFrame}
3636import org .apache .spark .sql .catalyst .util .stringToFile
3737import org .apache .spark .sql .execution .DataSourceScanExec
3838import org .apache .spark .sql .execution .datasources ._
39+ import org .apache .spark .sql .execution .datasources .parquet .ParquetFileFormat
3940import org .apache .spark .sql .execution .datasources .v2 .{BatchScanExec , DataSourceV2Relation , FileScan , FileTable }
4041import org .apache .spark .sql .execution .streaming ._
4142import org .apache .spark .sql .functions ._
@@ -292,35 +293,48 @@ abstract class FileStreamSinkSuite extends StreamTest {
292293 test(" parquet" ) {
293294 testFormat(None ) // should not throw error as default format parquet when not specified
294295 testFormat(Some (" parquet" ))
296+ testFormat(None , relativizeOutputPath = true )
297+ testFormat(Some (" parquet" ), relativizeOutputPath = true )
295298 }
296299
297300 test(" orc" ) {
298301 testFormat(Some (" orc" ))
302+ testFormat(Some (" orc" ), relativizeOutputPath = true )
299303 }
300304
301305 test(" text" ) {
302306 testFormat(Some (" text" ))
307+ testFormat(Some (" text" ), relativizeOutputPath = true )
303308 }
304309
305310 test(" json" ) {
306311 testFormat(Some (" json" ))
312+ testFormat(Some (" json" ), relativizeOutputPath = true )
307313 }
308314
309- def testFormat (format : Option [String ]): Unit = {
310- val inputData = MemoryStream [Int ]
315+ def testFormat (format : Option [String ], relativizeOutputPath : Boolean = false ): Unit = {
316+ val inputData = MemoryStream [String ] // text format only supports String
311317 val ds = inputData.toDS()
312318
313- val outputDir = Utils .createTempDir(namePrefix = " stream.output" ).getCanonicalPath
319+ val tempDir = Utils .createTempDir(namePrefix = " stream.output" )
320+ val outputPath = if (relativizeOutputPath) {
321+ Paths .get(" " ).toAbsolutePath.relativize(tempDir.toPath).toString
322+ } else {
323+ tempDir.getCanonicalPath
324+ }
314325 val checkpointDir = Utils .createTempDir(namePrefix = " stream.checkpoint" ).getCanonicalPath
315326
316- var query : StreamingQuery = null
327+ val writer = ds.toDF(" value" ).writeStream
328+ .option(" checkpointLocation" , checkpointDir)
329+ if (format.nonEmpty) {
330+ writer.format(format.get)
331+ }
317332
333+ var query : StreamingQuery = null
318334 try {
319- val writer = ds.map(i => (i, i * 1000 )).toDF(" id" , " value" ).writeStream
320- if (format.nonEmpty) {
321- writer.format(format.get)
322- }
323- query = writer.option(" checkpointLocation" , checkpointDir).start(outputDir)
335+ query = writer.start(outputPath)
336+ inputData.addData(" data" )
337+ query.processAllAvailable()
324338 } finally {
325339 if (query != null ) {
326340 query.stop()
@@ -664,6 +678,16 @@ abstract class FileStreamSinkSuite extends StreamTest {
664678 s " $path. " ))
665679 }
666680 }
681+
682+ test(" SPARK-50854: Make path fully qualified before passing it to FileStreamSink" ) {
683+ val fileFormat = new ParquetFileFormat () // any valid FileFormat
684+ val partitionColumnNames = Seq .empty[String ]
685+ val options = Map .empty[String , String ]
686+ val exception = intercept[SparkException ] {
687+ new FileStreamSink (spark, " test.parquet" , fileFormat, partitionColumnNames, options)
688+ }
689+ assert(exception.getMessage.contains(" is not absolute path." ))
690+ }
667691}
668692
669693object PendingCommitFilesTrackingManifestFileCommitProtocol {
0 commit comments