Hi Gurus,
I need to understand the checkpoint logic of the Dataflow Runner, like when
and how will the runner trigger a finalize on a checkpoint, is the finalize
thread same as the reader thread?
Could you share me the information, or point me to the related source code,
thanks in advance!
Islan
f lot
> of higher priority work gets scheduled (e.g. when a large window fires).
>
> If you want to force a reader to close and resume in your test, throw an
> IOException. Dataflow will restart the reader in streaming. Obviously this
> is only for testing.
>
> On Thu, Oct 18, 2018 at
Hi gurus,
I've added some debug output in my UnboundedReader and Checkpoint classes,
and noticed that the Dataflow Runner keeps encoding the checkpoint objects,
but never decode it, and never invoke the UnboundedReader.close() method in
my testing.
However, the Direct Runner will decode the check
s
>> needed.
>>
>> On Wed, Sep 26, 2018 at 7:11 PM flyisland wrote:
>>
>>> Hi, My jira id is "flyisland", thanks!
>>>
>>> On Thu, Sep 27, 2018 at 3:25 AM Jean-Baptiste Onofré
>>> wrote:
>>>
>>>> Yes please.
b) closing the client connection, these are two different
> actions which do not have to depend on each other.
>
> On 05.10.18 16:06, flyisland wrote:
> > I've checked the PubsubUnboundedSource, it just throws an exception if
> > it's a "restored checkpoint".
the Reader.
>
> Perhaps you want to check out out PubsubUnboundedSource for an example
> of how to do that?
>
> Cheers,
> Max
>
> On 05.10.18 14:47, flyisland wrote:
> > Hi Gurus,
> >
> > I'm building a new IO connector now, and I try to ack messages i
Hi Gurus,
I'm building a new IO connector now, and I try to ack messages in the
"UnboundedSource.CheckpointMark.finalizeCheckpoint()" method as MqttIO and
JmsIO did, but I found in the javadoc it said
> It is NOT safe to assume the UnboundedSource.UnboundedReader from which
this checkpoint was c
gt; connection that gets closed after some idle time.
>
> Assuming you need to ack on the same connection that served the records,
> finalize() functionality in UnboundedSource API is important case. You can
> use UnboundeSource API for now.
>
> On Thu, Sep 20, 2018 at 8:25 PM
Hi, My jira id is "flyisland", thanks!
On Thu, Sep 27, 2018 at 3:25 AM Jean-Baptiste Onofré
wrote:
> Yes please. Create a jira, I will tackle that. Thanks.
>
> Regards
> JB
> Le 26 sept. 2018, à 15:12, Lukasz Cwik a écrit:
>>
>> Yes, please create a JIRA ac
in submitting a patch with a test that exercises
> the bug?
>
> On Tue, Sep 25, 2018 at 1:21 AM flyisland wrote:
>
>> Hi
>>
>> There is a bug of the built-in MqttIO, please check the <
>> https://github.com/apache/beam/blob/master/sdks/java/io/mqtt/src/mai
Hi
There is a bug of the built-in MqttIO, please check the <
https://github.com/apache/beam/blob/master/sdks/java/io/mqtt/src/main/java/org/apache/beam/sdk/io/mqtt/MqttIO.java#L336>,
this readObject() method forget to invoke the "stream.defaultReadObject()"
method.
// set an empty list to message
s important case. You can
> use UnboundeSource API for now.
>
> On Thu, Sep 20, 2018 at 8:25 PM flyisland wrote:
>
>> Hi Reuven,
>>
>> There is no explicit ID in the message itself, and if there is
>> information can be used as an ID is depend on use cases.
>>
>
> On Thu, Sep 20, 2018 at 6:36 PM flyisland wrote:
>
>> Hi Lukasz,
>>
>> With the current API we provided, messages cannot be acked from a
>> different client.
>>
>> The server will re-send messages to the reconnected client if those
>> messages we
b0f6827195a262932e1267c4d4bfba/sdks/java/core/src/main/java/org/apache/beam/sdk/io/UnboundedSource.java#L93
>
>
> On Wed, Sep 19, 2018 at 8:31 PM flyisland wrote:
>
>> Hi Lukasz,
>>
>> This socket server is like an MQTT server, it has queues inside it and
>> the
n X to replay past messages)?
>
>
>
>
> On Tue, Sep 18, 2018 at 5:00 PM flyisland wrote:
>
>>
>> Hi Gurus,
>>
>> I'm trying to create an IO connector to fetch data from a socket server
>> from Beam, I'm new to Beam, but according to th
Hi Gurus,
I'm trying to create an IO connector to fetch data from a socket server
from Beam, I'm new to Beam, but according to this blog <
https://beam.apache.org/blog/2017/08/16/splittable-do-fn.html>, it seems
that SDF is the recommended way to implement an IO connector now.
This in-house built
16 matches
Mail list logo