Skip to content
Closed
Show file tree
Hide file tree
Changes from all commits
Commits
File filter

Filter by extension

Filter by extension

Conversations
Failed to load comments.
Loading
Jump to
Jump to file
Failed to load files.
Loading
Diff view
Diff view
Original file line number Diff line number Diff line change
Expand Up @@ -628,7 +628,8 @@ class StreamExecution(
// Rewire the plan to use the new attributes that were returned by the source.
val replacementMap = AttributeMap(replacements)
val triggerLogicalPlan = withNewSources transformAllExpressions {
case a: Attribute if replacementMap.contains(a) => replacementMap(a)
case a: Attribute if replacementMap.contains(a) =>
replacementMap(a).withMetadata(a.metadata)
case ct: CurrentTimestamp =>
CurrentBatchTimestamp(offsetSeqMetadata.batchTimestampMs,
ct.dataType)
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -391,6 +391,34 @@ class EventTimeWatermarkSuite extends StreamTest with BeforeAndAfter with Matche
checkDataset[Long](df, 1L to 100L: _*)
}

test("SPARK-21565: watermark operator accepts attributes from replacement") {
withTempDir { dir =>
dir.delete()

val df = Seq(("a", 100.0, new java.sql.Timestamp(100L)))
.toDF("symbol", "price", "eventTime")
df.write.json(dir.getCanonicalPath)

val input = spark.readStream.schema(df.schema)
.json(dir.getCanonicalPath)

val groupEvents = input
.withWatermark("eventTime", "2 seconds")
.groupBy("symbol", "eventTime")
.agg(count("price") as 'count)
.select("symbol", "eventTime", "count")
val q = groupEvents.writeStream
.outputMode("append")
.format("console")
.start()
try {
q.processAllAvailable()
} finally {
q.stop()
}
}
}

private def assertNumStateRows(numTotalRows: Long): AssertOnQuery = AssertOnQuery { q =>
val progressWithData = q.recentProgress.filter(_.numInputRows > 0).lastOption.get
assert(progressWithData.stateOperators(0).numRowsTotal === numTotalRows)
Expand Down