aboutsummaryrefslogtreecommitdiff
path: root/sql/catalyst/src/main
diff options
context:
space:
mode:
authorShixiong Zhu <shixiong@databricks.com>2017-01-10 17:58:11 -0800
committerShixiong Zhu <shixiong@databricks.com>2017-01-10 17:58:11 -0800
commitbc6c56e940fe93591a1e5ba45751f1b243b57e28 (patch)
treea1cdea8a10912863fbdd6c8b7fe50951e57bf78d /sql/catalyst/src/main
parent856bae6af64982ae0221948c58ff564887e54a70 (diff)
downloadspark-bc6c56e940fe93591a1e5ba45751f1b243b57e28.tar.gz
spark-bc6c56e940fe93591a1e5ba45751f1b243b57e28.tar.bz2
spark-bc6c56e940fe93591a1e5ba45751f1b243b57e28.zip
[SPARK-19140][SS] Allow update mode for non-aggregation streaming queries
## What changes were proposed in this pull request? This PR allow update mode for non-aggregation streaming queries. It will be same as the append mode if a query has no aggregations. ## How was this patch tested? Jenkins Author: Shixiong Zhu <shixiong@databricks.com> Closes #16520 from zsxwing/update-without-agg.
Diffstat (limited to 'sql/catalyst/src/main')
-rw-r--r--sql/catalyst/src/main/java/org/apache/spark/sql/streaming/OutputMode.java3
-rw-r--r--sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/analysis/UnsupportedOperationChecker.scala2
-rw-r--r--sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/streaming/InternalOutputModes.scala4
3 files changed, 5 insertions, 4 deletions
diff --git a/sql/catalyst/src/main/java/org/apache/spark/sql/streaming/OutputMode.java b/sql/catalyst/src/main/java/org/apache/spark/sql/streaming/OutputMode.java
index cf0579fd36..3f7cdb293e 100644
--- a/sql/catalyst/src/main/java/org/apache/spark/sql/streaming/OutputMode.java
+++ b/sql/catalyst/src/main/java/org/apache/spark/sql/streaming/OutputMode.java
@@ -57,7 +57,8 @@ public class OutputMode {
/**
* OutputMode in which only the rows that were updated in the streaming DataFrame/Dataset will
- * be written to the sink every time there are some updates.
+ * be written to the sink every time there are some updates. If the query doesn't contain
+ * aggregations, it will be equivalent to `Append` mode.
*
* @since 2.1.1
*/
diff --git a/sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/analysis/UnsupportedOperationChecker.scala b/sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/analysis/UnsupportedOperationChecker.scala
index 053c8eb617..c2666b2ab9 100644
--- a/sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/analysis/UnsupportedOperationChecker.scala
+++ b/sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/analysis/UnsupportedOperationChecker.scala
@@ -73,7 +73,7 @@ object UnsupportedOperationChecker {
s"streaming DataFrames/DataSets")(plan)
}
- case InternalOutputModes.Complete | InternalOutputModes.Update if aggregates.isEmpty =>
+ case InternalOutputModes.Complete if aggregates.isEmpty =>
throwError(
s"$outputMode output mode not supported when there are no streaming aggregations on " +
s"streaming DataFrames/Datasets")(plan)
diff --git a/sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/streaming/InternalOutputModes.scala b/sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/streaming/InternalOutputModes.scala
index 915f4a9e25..351bd6fff4 100644
--- a/sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/streaming/InternalOutputModes.scala
+++ b/sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/streaming/InternalOutputModes.scala
@@ -40,8 +40,8 @@ private[sql] object InternalOutputModes {
/**
* OutputMode in which only the rows in the streaming DataFrame/Dataset that were updated will be
- * written to the sink every time these is some updates. This output mode can only be used in
- * queries that contain aggregations.
+ * written to the sink every time these is some updates. If the query doesn't contain
+ * aggregations, it will be equivalent to `Append` mode.
*/
case object Update extends OutputMode
}