You are viewing a plain text version of this content. The canonical link for it is here.
Posted to dev@flink.apache.org by 123052475 <re...@foxmail.com> on 2021/06/30 02:41:52 UTC

回复: [DISCUSS] FLIP-171: Async Sink

退订




------------------&nbsp;原始邮件&nbsp;------------------
发件人: "Hausmann, Steffen"<shausma@amazon.de.INVALID&gt;; 
发送时间: 2021年6月29日(星期二) 晚上11:58
收件人: "dev"<dev@flink.apache.org&gt;; "Piotr Nowojski"<pnowojski@apache.org&gt;; "Till Rohrmann"<trohrmann@apache.org&gt;; 
抄送: "Arvid Heise"<arvid@apache.org&gt;; 
主题: Re: [DISCUSS] FLIP-171: Async Sink



Hey Poitr,

I've just adapted the FLIP and changed the signature for the `submitRequestEntries` method:

protected abstract void submitRequestEntries(List<RequestEntryT&gt; requestEntries, ResultFuture<?&gt; requestResult);

In addition, we are likely to use an AtomicLong to track the number of outstanding requests, as you have proposed in 2b). I've already indicated this in the FLIP, but it's not fully fleshed out. But as you have said, that seems to be an implementation detail and the important part is the change of the `submitRequestEntries` signature.

Thanks for your feedback!

Cheers, Steffen


On 25.06.21, 17:05, "Hausmann, Steffen" <shausma@amazon.de.INVALID&gt; wrote:

&nbsp;&nbsp;&nbsp; CAUTION: This email originated from outside of the organization. Do not click links or open attachments unless you can confirm the sender and know the content is safe.



&nbsp;&nbsp;&nbsp; Hi Piotr,

&nbsp;&nbsp;&nbsp; I’m happy to take your guidance on this. I need to think through your proposals and I’ll follow-up on Monday with some more context so that we can close the discussion on these details. But for now, I’ll close the vote.

&nbsp;&nbsp;&nbsp; Thanks, Steffen

&nbsp;&nbsp;&nbsp; From: Piotr Nowojski <pnowojski@apache.org&gt;
&nbsp;&nbsp;&nbsp; Date: Friday, 25. June 2021 at 14:48
&nbsp;&nbsp;&nbsp; To: Till Rohrmann <trohrmann@apache.org&gt;
&nbsp;&nbsp;&nbsp; Cc: Steffen Hausmann <shausma@amazon.de&gt;, "dev@flink.apache.org" <dev@flink.apache.org&gt;, Arvid Heise <arvid@apache.org&gt;
&nbsp;&nbsp;&nbsp; Subject: RE: [EXTERNAL] [DISCUSS] FLIP-171: Async Sink


&nbsp;&nbsp;&nbsp; CAUTION: This email originated from outside of the organization. Do not click links or open attachments unless you can confirm the sender and know the content is safe.


&nbsp;&nbsp;&nbsp; Hey,

&nbsp;&nbsp;&nbsp; I've just synced with Arvid about a couple of more remarks from my side and he shared mine concerns.

&nbsp;&nbsp;&nbsp; 1. I would very strongly recommend ditching `CompletableFuture<?&gt; ` from the&nbsp; `protected abstract CompletableFuture<?&gt; submitRequestEntries(List<RequestEntryT&gt; requestEntries);`&nbsp; in favor of something like `org.apache.flink.streaming.api.functions.async.ResultFuture` interface. `CompletableFuture<?&gt;` would partially make the threading model of the `AsyncSincWriter` part of the public API and it would tie our hands. Regardless how `CompletableFuture<?&gt;` is used, it imposes performance overhead because it's synchronisation/volatile inside of it. On the other hand something like:

&nbsp;&nbsp;&nbsp; protected abstract void submitRequestEntries(List<RequestEntryT&gt; requestEntries, ResultFuture<?&gt; requestResult);

&nbsp;&nbsp;&nbsp; Would allow us to implement the threading model as we wish. `ResultFuture` could be backed via `CompletableFuture<?&gt;` underneath, but it could also be something more efficient.&nbsp; I will explain what I have in mind in a second.

&nbsp;&nbsp;&nbsp; 2. It looks to me that proposed `AsyncSinkWriter` Internals are not very efficient and maybe the threading model hasn't been thought through? Especially private fields:

&nbsp;&nbsp;&nbsp; private final BlockingDeque<RequestEntryT&gt; bufferedRequestEntries;
&nbsp;&nbsp;&nbsp; private BlockingDeque<CompletableFuture<?&gt;&gt; inFlightRequests;

&nbsp;&nbsp;&nbsp; are a bit strange to me. Why do we need two separate thread safe collections? Why do we need a `BlockingDeque` of `CompletableFuture<?&gt;`s? If we are already using a fully synchronised collection, there should be no need for another layer of thread safe `CompletableFuture<?&gt;`.

&nbsp;&nbsp;&nbsp; As I understand, the threading model of the `AsyncSinkWriter` is very similar to that of the `AsyncWaitOperator`, with very similar requirements for inducing backpressure. How I would see it implemented is for example:

&nbsp;&nbsp;&nbsp; a) Having a single lock, that would encompass the whole `AsyncSinkWriter#flush()` method. `flush()` would be called from the task thread (mailbox). To induce backpressure, `#flush()` would just call `lock.wait()`. `ResultFuture#complete(...)` called from an async thread, would also synchronize on the same lock, and mark some of the inflight requests as completed and call `lock.notify()`.

&nbsp;&nbsp;&nbsp; b) More efficient solution. On the hot path we would have for example only `AtomicLong numberOfInFlightRequests`. Task thread would be bumping it, `ResultFuture#complete()` would be decreasing it. If the task thread when bumping `numberOfInFlightRequests` exceeds a threshold, he goes to sleep/wait on a lock or some `CompletableFuture`. If `ResultFuture#complete()` when decreasing the count goes below the threshold, it would wake up the task thread.&nbsp; Compared to the option a),&nbsp; on the hot path, option b) would have only AtomicLong.increment overhead

&nbsp;&nbsp;&nbsp; c) We could use mailbox, the same way as AsyncWaitOperator is doing. In this case `ResultFuture#complete()` would be enquing mailbox action, which is thread safe on it's own.

&nbsp;&nbsp;&nbsp; Either of those options would be more efficient and simpler (from the threading model perspective) than having two `BlockingQueues` and `CompletableFuture<?&gt;`. Also as you can see, neither of those solutions require the overhead of ` CompletableFuture<?&gt; submitRequestEntries(List<RequestEntryT&gt; requestEntries)`. Each one of those could use a more efficient and custom implementation of `ResultFuture.complete(...)`.


&nbsp;&nbsp;&nbsp; Whether we use a), b) or c) I think should be an implementation detail. But to allow this to truly be an implementation detail, we would need to agree on 1. Nevertheless I think that the change I proposed in 1. is small enough that I think there is no need to cancel the current vote on the FLIP.

&nbsp;&nbsp;&nbsp; WDYT?

&nbsp;&nbsp;&nbsp; Piotrek


&nbsp;&nbsp;&nbsp; wt., 22 cze 2021 o 11:42 Till Rohrmann <trohrmann@apache.org<mailto:trohrmann@apache.org&gt;&gt; napisał(a):
&nbsp;&nbsp;&nbsp; Adding the InterruptException to the write method would make it explicit that the write call can block but must react to interruptions (e.g. when Flink wants to cancel the operation). I think this makes the contract a bit clearer.

&nbsp;&nbsp;&nbsp; I think starting simple and then extending the API as we see the need is a good idea.

&nbsp;&nbsp;&nbsp; Cheers,
&nbsp;&nbsp;&nbsp; Till

&nbsp;&nbsp;&nbsp; On Tue, Jun 22, 2021 at 11:20 AM Hausmann, Steffen <shausma@amazon.de<mailto:shausma@amazon.de&gt;&gt; wrote:
&nbsp;&nbsp;&nbsp; Hey,

&nbsp;&nbsp;&nbsp; Agreed on starting with a blocking `write`. I've adapted the FLIP accordingly.

&nbsp;&nbsp;&nbsp; For now I've chosen to add the `InterruptedException` to the `write` method signature as I'm not fully understanding the implications of swallowing the exception. Depending on the details of&nbsp; the code that is calling the write method, it may cause event loss. But this seems more of an implementation detail, that we can revisit once we are actually implementing the sink.

&nbsp;&nbsp;&nbsp; Unless there are additional comments, does it make sense to start the voting process in the next day or two?

&nbsp;&nbsp;&nbsp; Cheers, Steffen


&nbsp;&nbsp;&nbsp; On 21.06.21, 14:51, "Piotr Nowojski" <pnowojski@apache.org<mailto:pnowojski@apache.org&gt;&gt; wrote:

&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; CAUTION: This email originated from outside of the organization. Do not click links or open attachments unless you can confirm the sender and know the content is safe.



&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; Hi,

&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; Thanks Steffen for the explanations. I think it makes sense to me.

&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; Re Arvid/Steffen:

&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; - Keep in mind that even if we choose to provide a non blocking API using
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; the `isAvailable()`/`getAvailableFuture()` method, we would still need to
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; support blocking inside the sinks. For example at the very least, emitting
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; many records at once (`flatMap`) or firing timers are scenarios when output
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; availability would be ignored at the moment by the runtime. Also I would
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; imagine writing very large (like 1GB) records would be blocking on
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; something as well.
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; - Secondly, exposing availability to the API level might not be that
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; easy/trivial. The availability pattern as defined in `AvailabilityProvider`
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; class is quite complicated and not that easy to implement by a user.

&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; Both of those combined with lack of a clear motivation for adding
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; `AvailabilityProvider` to the sinks/operators/functions,&nbsp; I would vote on
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; just starting with blocking `write` calls. This can always be extended in
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; the future with availability if needed/motivated properly.

&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; That would be aligned with either Arvid's option 1 or 2. I don't know what
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; are the best practices with `InterruptedException`, but I'm always afraid
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; of it, so I would feel personally safer with option 2.

&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; I'm not sure what problem option 3 is helping to solve? Adding `wakeUp()`
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; would sound strange to me.

&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; Best,
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; Piotrek

&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; pon., 21 cze 2021 o 12:15 Arvid Heise <arvid@apache.org<mailto:arvid@apache.org&gt;&gt; napisał(a):

&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt; Hi Piotr,
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt; to pick up this discussion thread again:
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt; - This FLIP is about providing some base implementation for FLIP-143 sinks
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt; that make adding new implementations easier, similar to the
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt; SourceReaderBase.
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt; - The whole availability topic will most likely be a separate FLIP. The
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt; basic issue just popped up here because we currently have no way to signal
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt; backpressure in sinks except by blocking `write`. This feels quite natural
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt; in sinks with sync communication but quite unnatural in async sinks.
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt; Now we have a couple of options. In all cases, we would have some WIP
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt; limit on the number of records/requests being able to be processed in
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt; parallel asynchronously (similar to asyncIO).
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt; 1. We use some blocking queue in `write`, then we need to handle
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt; interruptions. In the easiest case, we extend `write` to throw the
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt; `InterruptedException`, which is a small API change.
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt; 2. We use a blocking queue, but handle interrupts and swallow/translate
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt; them. No API change.
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt; Both solutions block the task thread, so any RPC message / unaligned
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt; checkpoint would be processed only after the backpressure is temporarily
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt; lifted. That's similar to the discussions that you linked. Cancellation may
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt; also be a tad harder on 2.
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt; 3. We could also add some `wakeUp` to the `SinkWriter` similar to
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt; `SplitFetcher` [1]. Basically, you use a normal queue with a completeable
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt; future on which you block. Wakeup would be a clean way to complete it next
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt; to the natural completion through finished requests.
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt; 4. We add availability to the sink. However, this API change also requires
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt; that we allow operators to be available so it may be a bigger change with
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt; undesired side-effects. On the other hand, we could also use the same
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt; mechanism for asyncIO.
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt; For users of FLIP-171, none of the options are exposed. So we could also
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt; start with a simple solution (add `InterruptedException`) and later try to
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt; add availability. Option 1+2 would also not require an additional FLIP; we
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt; could add it as part of this FLIP.
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt; Best,
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt; Arvid
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt; [1]
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt; https://github.com/apache/flink/blob/master/flink-connectors/flink-connector-base/src/main/java/org/apache/flink/connector/base/source/reader/fetcher/SplitFetcher.java#L258-L258
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt; On Thu, Jun 10, 2021 at 10:09 AM Hausmann, Steffen
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt; <shausma@amazon.de.invalid&gt; wrote:
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; Hey Piotrek,
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; Thanks for your comments on the FLIP. I'll address your second question
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; first, as I think it's more central to this FLIP. Just looking at the AWS
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; ecosystem, there are several sinks with overlapping functionality. I've
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; chosen AWS sinks here because I'm most familiar with those, but a similar
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; argument applies more generically for destination that support async ingest.
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; There is, for instance, a sink for Amazon Kinesis Data Streams that is
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; part of Apache Flink [1], a sink for Amazon Kinesis Data Firehose [2], a
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; sink for Amazon DynamoDB [3], and a sink for Amazon Timestream [4]. All
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; these sinks have implemented their own mechanisms for batching, persisting,
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; and retrying events. And I'm not sure if all of them properly participate
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; in checkpointing. [3] even seems to closely mirror [1] as it contains
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; references to the Kinesis Producer Library, which is unrelated to Amazon
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; DynamoDB.
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; These sinks predate FLIP-143. But as batching, persisting, and retrying
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; capabilities do not seem to be part of FLIP-143, I'd argue that we would
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; end up with similar duplication, even if these sinks were rewritten today
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; based on FLIP-143. And that's the idea of FLIP-171: abstract away these
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; commonly required capabilities so that it becomes easy to create support
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; for a wide range of destination without having to think about batching,
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; retries, checkpointing, etc. I've included an example in the FLIP [5] that
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; shows that it only takes a couple of lines of code to implement a sink with
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; exactly-once semantics. To be fair, the example is lacking robust failure
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; handling and some more advanced capabilities of [1], but I think it still
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; supports this point.
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; Regarding your point on the isAvailable pattern. We need some way for the
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; sink to propagate backpressure and we would also like to support time based
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; buffering hints. There are two options I currently see and would need
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; additional input on which one is the better or more desirable one. The
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; first option is to use the non-blocking isAvailable pattern. Internally,
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; the sink persists buffered events in the snapshot state which avoids having
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; to flush buffered record on a checkpoint. This seems to align well with the
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; non-blocking isAvailable pattern. The second option is to make calls to
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; `write` blocking and leverage an internal thread to trigger flushes based
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; on time based buffering hints. We've discussed these options with Arvid and
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; suggested to assumed that the `isAvailable` pattern will become available
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; for sinks through and additional FLIP.
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; I think it is an important discussion to have. My understanding of the
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; implications for Flink in general are very naïve, so I'd be happy to get
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; further guidance. However, I don't want to make this discussion part of
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; FLIP-171. For FLIP-171 we'll use whatever is available.
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; Does that make sense?
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; Cheers, Steffen
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; [1]
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; https://github.com/apache/flink/tree/master/flink-connectors/flink-connector-kinesis
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; [2] https://github.com/aws/aws-kinesisanalytics-flink-connectors
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; [3] https://github.com/klarna-incubator/flink-connector-dynamodb
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; [4] https://github.com/awslabs/amazon-timestream-tools/
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; [5]
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; https://cwiki.apache.org/confluence/display/FLINK/FLIP-171%3A+Async+Sink#FLIP171:AsyncSink-SimplifiedAsyncSinkWriterforKinesisDataStreams
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; On 09.06.21, 19:44, "Piotr Nowojski" <pnowojski@apache.org<mailto:pnowojski@apache.org&gt;&gt; wrote:
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;&nbsp;&nbsp;&nbsp;&nbsp; CAUTION: This email originated from outside of the organization. Do
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; not click links or open attachments unless you can confirm the sender and
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; know the content is safe.
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;&nbsp;&nbsp;&nbsp;&nbsp; Hi Steffen,
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;&nbsp;&nbsp;&nbsp;&nbsp; Thanks for writing down the proposal. Back when the new Sink API was
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; being
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;&nbsp;&nbsp;&nbsp;&nbsp; discussed, I was proposing to add our usual `CompletableFuture<Void&gt;
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;&nbsp;&nbsp;&nbsp;&nbsp; isAvailable()` pattern to make sinks non-blocking. You can see the
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;&nbsp;&nbsp;&nbsp;&nbsp; discussion starting here [1], and continuing for a couple of more
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; posts
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;&nbsp;&nbsp;&nbsp;&nbsp; until here [2]. Back then, the outcome was that it would give very
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; little
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;&nbsp;&nbsp;&nbsp;&nbsp; benefit, at the expense of making the API more complicated. Could you
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; maybe
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;&nbsp;&nbsp;&nbsp;&nbsp; relate your proposal to that discussion from last year?
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;&nbsp;&nbsp;&nbsp;&nbsp; I see that your proposal is going much further than just adding the
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;&nbsp;&nbsp;&nbsp;&nbsp; availability method, could you also motivate this a bit further?
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; Could you
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;&nbsp;&nbsp;&nbsp;&nbsp; maybe reference/show some sinks that:
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;&nbsp;&nbsp;&nbsp;&nbsp; 1. are already implemented using FLIP-143
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;&nbsp;&nbsp;&nbsp;&nbsp; 2. that have some code duplication...
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;&nbsp;&nbsp;&nbsp;&nbsp; 3. ...this duplication would be solved by FLIP-171
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;&nbsp;&nbsp;&nbsp;&nbsp; Best,
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;&nbsp;&nbsp;&nbsp;&nbsp; Piotrek
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;&nbsp;&nbsp;&nbsp;&nbsp; [1]
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; http://apache-flink-mailing-list-archive.1008284.n3.nabble.com/DISCUSS-FLIP-143-Unified-Sink-API-tp44602p44872.html
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;&nbsp;&nbsp;&nbsp;&nbsp; [2]
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; http://apache-flink-mailing-list-archive.1008284.n3.nabble.com/DISCUSS-FLIP-143-Unified-Sink-API-tp44602p44930.html
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;&nbsp;&nbsp;&nbsp;&nbsp; śr., 9 cze 2021 o 09:49 Hausmann, Steffen <shausma@amazon.de.invalid&gt;
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;&nbsp;&nbsp;&nbsp;&nbsp; napisał(a):
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;&nbsp;&nbsp;&nbsp;&nbsp; &gt; Hi there,
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;&nbsp;&nbsp;&nbsp;&nbsp; &gt;
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;&nbsp;&nbsp;&nbsp;&nbsp; &gt; We would like to start a discussion thread on "FLIP-171: Async
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; Sink" [1],
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;&nbsp;&nbsp;&nbsp;&nbsp; &gt; where we propose to create a common abstraction for destinations
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; that
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;&nbsp;&nbsp;&nbsp;&nbsp; &gt; support async requests. This abstraction will make it easier to add
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;&nbsp;&nbsp;&nbsp;&nbsp; &gt; destinations to Flink by implementing a lightweight shim, while it
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; avoids
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;&nbsp;&nbsp;&nbsp;&nbsp; &gt; maintaining dozens of independent sinks.
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;&nbsp;&nbsp;&nbsp;&nbsp; &gt;
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;&nbsp;&nbsp;&nbsp;&nbsp; &gt; Looking forward to your feedback.
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;&nbsp;&nbsp;&nbsp;&nbsp; &gt;
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;&nbsp;&nbsp;&nbsp;&nbsp; &gt; Cheers, Steffen
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;&nbsp;&nbsp;&nbsp;&nbsp; &gt;
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;&nbsp;&nbsp;&nbsp;&nbsp; &gt; [1]
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;&nbsp;&nbsp;&nbsp;&nbsp; &gt;
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; https://cwiki.apache.org/confluence/display/FLINK/FLIP-171%3A+Async+Sink
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;&nbsp;&nbsp;&nbsp;&nbsp; &gt;
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;&nbsp;&nbsp;&nbsp;&nbsp; &gt;
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;&nbsp;&nbsp;&nbsp;&nbsp; &gt;
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;&nbsp;&nbsp;&nbsp;&nbsp; &gt; Amazon Web Services EMEA SARL
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;&nbsp;&nbsp;&nbsp;&nbsp; &gt; 38 avenue John F. Kennedy, L-1855 Luxembourg
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;&nbsp;&nbsp;&nbsp;&nbsp; &gt; Sitz der Gesellschaft: L-1855 Luxemburg
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;&nbsp;&nbsp;&nbsp;&nbsp; &gt; eingetragen im Luxemburgischen Handelsregister unter R.C.S. B186284
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;&nbsp;&nbsp;&nbsp;&nbsp; &gt;
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;&nbsp;&nbsp;&nbsp;&nbsp; &gt; Amazon Web Services EMEA SARL, Niederlassung Deutschland
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;&nbsp;&nbsp;&nbsp;&nbsp; &gt; Marcel-Breuer-Str. 12, D-80807 Muenchen
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;&nbsp;&nbsp;&nbsp;&nbsp; &gt; Sitz der Zweigniederlassung: Muenchen
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;&nbsp;&nbsp;&nbsp;&nbsp; &gt; eingetragen im Handelsregister des Amtsgerichts Muenchen unter HRB
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; 242240,
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;&nbsp;&nbsp;&nbsp;&nbsp; &gt; USt-ID DE317013094
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;&nbsp;&nbsp;&nbsp;&nbsp; &gt;
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;&nbsp;&nbsp;&nbsp;&nbsp; &gt;
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;&nbsp;&nbsp;&nbsp;&nbsp; &gt;
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;&nbsp;&nbsp;&nbsp;&nbsp; &gt;
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; Amazon Web Services EMEA SARL
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; 38 avenue John F. Kennedy, L-1855 Luxembourg
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; Sitz der Gesellschaft: L-1855 Luxemburg
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; eingetragen im Luxemburgischen Handelsregister unter R.C.S. B186284
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; Amazon Web Services EMEA SARL, Niederlassung Deutschland
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; Marcel-Breuer-Str. 12, D-80807 Muenchen
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; Sitz der Zweigniederlassung: Muenchen
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; eingetragen im Handelsregister des Amtsgerichts Muenchen unter HRB
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt; 242240, USt-ID DE317013094
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;
&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp;&nbsp; &gt;&gt;




&nbsp;&nbsp;&nbsp; Amazon Web Services EMEA SARL
&nbsp;&nbsp;&nbsp; 38 avenue John F. Kennedy, L-1855 Luxembourg
&nbsp;&nbsp;&nbsp; Sitz der Gesellschaft: L-1855 Luxemburg
&nbsp;&nbsp;&nbsp; eingetragen im Luxemburgischen Handelsregister unter R.C.S. B186284

&nbsp;&nbsp;&nbsp; Amazon Web Services EMEA SARL, Niederlassung Deutschland
&nbsp;&nbsp;&nbsp; Marcel-Breuer-Str. 12, D-80807 Muenchen
&nbsp;&nbsp;&nbsp; Sitz der Zweigniederlassung: Muenchen
&nbsp;&nbsp;&nbsp; eingetragen im Handelsregister des Amtsgerichts Muenchen unter HRB 242240, USt-ID DE317013094





&nbsp;&nbsp;&nbsp; Amazon Web Services EMEA SARL
&nbsp;&nbsp;&nbsp; 38 avenue John F. Kennedy, L-1855 Luxembourg
&nbsp;&nbsp;&nbsp; Sitz der Gesellschaft: L-1855 Luxemburg
&nbsp;&nbsp;&nbsp; eingetragen im Luxemburgischen Handelsregister unter R.C.S. B186284

&nbsp;&nbsp;&nbsp; Amazon Web Services EMEA SARL, Niederlassung Deutschland
&nbsp;&nbsp;&nbsp; Marcel-Breuer-Str. 12, D-80807 Muenchen
&nbsp;&nbsp;&nbsp; Sitz der Zweigniederlassung: Muenchen
&nbsp;&nbsp;&nbsp; eingetragen im Handelsregister des Amtsgerichts Muenchen unter HRB 242240, USt-ID DE317013094







Amazon Web Services EMEA SARL
38 avenue John F. Kennedy, L-1855 Luxembourg
Sitz der Gesellschaft: L-1855 Luxemburg
eingetragen im Luxemburgischen Handelsregister unter R.C.S. B186284

Amazon Web Services EMEA SARL, Niederlassung Deutschland
Marcel-Breuer-Str. 12, D-80807 Muenchen
Sitz der Zweigniederlassung: Muenchen
eingetragen im Handelsregister des Amtsgerichts Muenchen unter HRB 242240, USt-ID DE317013094