[
https://issues.apache.org/jira/browse/BEAM-9449?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=17423817#comment-17423817
]
Jan Lukavský commented on BEAM-9449:
------------------------------------
I cannot speak for PubSubIO, but regarding Kafka, you should be able to achieve
what you want by:
a) compile FlinkRunner with patch for BEAM-12704 (will be part of 2.33.0
release)
b) you can then start own exapansion service in ReadFromKafka via passing
expansion_service parameter and using JavaJarExpansionService and passing both
default environment and use_deprecated_read experiment (expansion service
accepts command-line arguments via BEAM-12538):
{code:java}
def get_expansion_service(jar="<expansion_service_jar>", args=None):
if args == None:
args = [
"--defaultEnvironmentType=PROCESS",
"--defaultEnvironmentConfig={\"command\": \"/opt/apache/beam/boot\"}",
"--experiments=use_deprecated_read"]
return JavaJarExpansionService(jar, ['{{PORT}}'] + args) {code}
The above uses PROCESS default environment with the SDK harness installed in
/opt/apache/beam inside Flink's TaskManager docker image. This setup works for
me for Kafka+Flink+Python. We should probably enable to pass parameters to the
expansion service without the need to ship the expansion service jar,
[~robertwb] I'm not sure if this can be done today?
> Consider passing pipeline options for expansion service.
> --------------------------------------------------------
>
> Key: BEAM-9449
> URL: https://issues.apache.org/jira/browse/BEAM-9449
> Project: Beam
> Issue Type: New Feature
> Components: beam-model, cross-language
> Reporter: Robert Bradshaw
> Priority: P2
> Labels: stale-P2
> Time Spent: 2h 40m
> Remaining Estimate: 0h
>
--
This message was sent by Atlassian Jira
(v8.3.4#803005)