You are viewing a plain text version of this content. The canonical link for it is here.
Posted to issues@iceberg.apache.org by GitBox <gi...@apache.org> on 2021/12/15 17:47:21 UTC

[GitHub] [iceberg] kbendick commented on a change in pull request #3749: Spark: We should probably say why we cannot process the snapshot in SparkMicroBatchStream

kbendick commented on a change in pull request #3749:
URL: https://github.com/apache/iceberg/pull/3749#discussion_r769837962



##########
File path: site/docs/spark-structured-streaming.md
##########
@@ -26,6 +26,28 @@ As of Spark 3.0, DataFrame reads and writes are supported.
 |--------------------------------------------------|----------|------------|------------------------------------------------|
 | [DataFrame write](#writing-with-streaming-query) | ✔        | ✔          |                                                |
 
+## Streaming Reads
+
+Iceberg supports processing incremental data in spark structured streaming  jobs which starts from a historical timestamp:
+
+```scala
+val spark:SparkSession = ...
+val tableIdentifier: String = ...
+
+val df = spark.readStream
+    .format("iceberg")
+    .option(SparkReadOptions.STREAM_FROM_TIMESTAMP, Long.toString(streamStartTimestamp))
+    .load(tableIdentifier)
+```
+
+The `tableIdentifier` can be:
+
+* The fully-qualified path to a HDFS table, like `hdfs://nn:8020/path/to/table`
+* A table name if the table is tracked by a catalog, like `database.table_name`
+
+!!! Note
+    Iceberg only supports read data from snapshot whose type of Data Operations is APPEND\REPLACE\DELETE. In particular if some of your snapshots are of DELETE type, you need to add 'streaming-skip-delete-snapshots' option to skip it, otherwise the task will fail.

Review comment:
       Nit: In the rich diff, this note isn't coming up formatted. Have you verified using `mkdocs` that this formats like the other parts that use `!!!`?
   
   Also, we might want to just format this as any other config box vs using the `!!! Note` statement.

##########
File path: site/docs/spark-structured-streaming.md
##########
@@ -26,6 +26,28 @@ As of Spark 3.0, DataFrame reads and writes are supported.
 |--------------------------------------------------|----------|------------|------------------------------------------------|
 | [DataFrame write](#writing-with-streaming-query) | ✔        | ✔          |                                                |
 
+## Streaming Reads
+
+Iceberg supports processing incremental data in spark structured streaming  jobs which starts from a historical timestamp:
+
+```scala
+val spark:SparkSession = ...
+val tableIdentifier: String = ...
+
+val df = spark.readStream
+    .format("iceberg")
+    .option(SparkReadOptions.STREAM_FROM_TIMESTAMP, Long.toString(streamStartTimestamp))
+    .load(tableIdentifier)
+```
+
+The `tableIdentifier` can be:
+
+* The fully-qualified path to a HDFS table, like `hdfs://nn:8020/path/to/table`
+* A table name if the table is tracked by a catalog, like `database.table_name`

Review comment:
       Question / Comment: It might be better to just say that the table identifier can be any valid table identifier or table path and link to any existing docs we have on that., instead of repeating the definition here or hiding the definition within Spark streaming reads section (if we don't have it defined somewhere else).
   
   Is there a place we can link to already?

##########
File path: spark/v3.2/spark/src/main/java/org/apache/iceberg/spark/source/SparkMicroBatchStream.java
##########
@@ -204,7 +204,9 @@ private boolean shouldProcess(Snapshot snapshot) {
         "Cannot process delete snapshot: %s", snapshot.snapshotId());
     Preconditions.checkState(
         op.equals(DataOperations.DELETE) || op.equals(DataOperations.APPEND) || op.equals(DataOperations.REPLACE),
-        "Cannot process %s snapshot: %s", op.toLowerCase(Locale.ROOT), snapshot.snapshotId());
+        "Cannot process snapshot: %s, Structured Streaming does not support snapshots of type %s",

Review comment:
       Nit: Can we say `.... does not currently support snapshots of type %s`? In the future, we will support reading more of them, like we do in Flink.
   
   Also, can we mention the config `streaming-skip-delete-snapshots` in the Preconditions check? That way, if users get this exception, they know the option to get passed it if they'd like.
   
   Maybe like 
   ```java
       Preconditions.checkState(
           op.equals(DataOperations.DELETE) || op.equals(DataOperations.APPEND) || op.equals(DataOperations.REPLACE),
           "Cannot process snapshot: %s. Structured Streaming does not support snapshots of type %s. To ignore snapshots of type delete, set the config %s to true.",
          snapshot.snapshotId(),
          op.toLowerCase(Locale.ROOT),
          SparkReadOptions.STREAMING_SKIP_DELETE_SNAPSHOTS);
   ````




-- 
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: issues-unsubscribe@iceberg.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: issues-unsubscribe@iceberg.apache.org
For additional commands, e-mail: issues-help@iceberg.apache.org