[
https://issues.apache.org/jira/browse/HUDI-269?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=16938163#comment-16938163
]
Xing Pan commented on HUDI-269:
-------------------------------
I tried to run the same hudi app via hudi spark datasource writer:
{code:java}
spark
.readStream
.format("kafka")
.option("kafka.bootstrap.servers", KAFKA_SERVER)
.option("subscribe", DEMO_11_TOPIC)
.load()
.select(from_confluent_avro(col("value"), SCHEMA_REGISTRY_CONF) as
'data).select("data.*")
.writeStream.format("org.apache.hudi")
.option(DataSourceWriteOptions.STORAGE_TYPE_OPT_KEY, tableType)
.option(DataSourceWriteOptions.RECORDKEY_FIELD_OPT_KEY, "id")
.option(DataSourceWriteOptions.PARTITIONPATH_FIELD_OPT_KEY, "dateStr")
.option(DataSourceWriteOptions.PRECOMBINE_FIELD_OPT_KEY, "ts")
.option(HoodieWriteConfig.TABLE_NAME, DEMO_11_TABLE_NAME)
.option("checkpointLocation", checkpointPath)
.option(DataSourceWriteOptions.HIVE_TABLE_OPT_KEY, DEMO_11_TABLE_NAME)
.option(DataSourceWriteOptions.HIVE_DATABASE_OPT_KEY, "default")
.option(DataSourceWriteOptions.HIVE_URL_OPT_KEY, HIVE_URL)
.option(DataSourceWriteOptions.HIVE_PARTITION_FIELDS_OPT_KEY, "dateStr")
.option(DataSourceWriteOptions.HIVE_PARTITION_EXTRACTOR_CLASS_OPT_KEY,
classOf[SlashEncodedDayPartitionValueExtractor].getCanonicalName)
.option(DataSourceWriteOptions.HIVE_SYNC_ENABLED_OPT_KEY, "true")
.outputMode(OutputMode.Append)
.trigger(Trigger.ProcessingTime(5000))
.start(outputPath)
.awaitTermination()
{code}
{code:java}
spark-submit --class xxx.HudiSpark \
--jars \
xxx/hudi-spark-bundle-0.5.1-SNAPSHOT.jar,\
xxx/abris_2.11-3.0.1.jar,\
xxx/common-utils-5.3.0.jar,xxx/kafka-schema-registry-client-5.3.0.jar,xxx/kafka-avro-serializer-5.3.0.jar,xxx/common-config-5.3.0.jar
\
--packages
org.apache.spark:spark-sql-kafka-0-10_2.11:2.4.3,org.apache.spark:spark-avro_2.11:2.4.3
\
--conf spark.hadoop.fs.s3a.endpoint=s3-ap-east-1.amazonaws.com \
--conf spark.dynamicAllocation.executorIdleTimeout=10s \
--conf hoodie.embed.timeline.server=true \
--conf hoodie.filesystem.view.incr.timeline.sync.enable=true \
--conf hoodie.upsert.shuffle.parallelism=2 \
--executor-memory 1g \
my_test.jar
{code}
and push 300 records for every second, and the S3 request count is fairly low:
!image-2019-09-26-09-02-24-761.png!
I am not quite sure about the difference between datasource writer and delta
streamer, as far as I know, when there is no data coming, request count is
about the same, but if I push some record every second, *datasource writer*
costs about 10 times lower request count than delta streamer.
[~vinoth]
> Provide ability to throttle DeltaStreamer sync runs
> ---------------------------------------------------
>
> Key: HUDI-269
> URL: https://issues.apache.org/jira/browse/HUDI-269
> Project: Apache Hudi (incubating)
> Issue Type: Improvement
> Components: deltastreamer
> Reporter: Balaji Varadarajan
> Assignee: Xing Pan
> Priority: Major
> Labels: pull-request-available
> Fix For: 0.5.0
>
> Attachments: hudi_request_test.tar.gz,
> image-2019-09-25-08-51-19-686.png, image-2019-09-26-09-02-24-761.png
>
> Time Spent: 10m
> Remaining Estimate: 0h
>
> Copied from [https://github.com/apache/incubator-hudi/issues/922]
> In some scenario in our cluster, we may want delta streamer to slow down a
> bit.
> so it's nice to have a parameter to control the min sync interval of each
> sync in continuous mode.
> this param is default to 0, so this should not affect current logic.
> minor pr: [#921|https://github.com/apache/incubator-hudi/pull/921]
> the main reason we want to slow it down is that aws s3 is charged by s3
> get/put/list requests. we don't want to pay for too many requests for a
> really slow change table.
--
This message was sent by Atlassian Jira
(v8.3.4#803005)