Return-Path: X-Original-To: archive-asf-public-internal@cust-asf2.ponee.io Delivered-To: archive-asf-public-internal@cust-asf2.ponee.io Received: from cust-asf.ponee.io (cust-asf.ponee.io [163.172.22.183]) by cust-asf2.ponee.io (Postfix) with ESMTP id D1819200D54 for ; Fri, 8 Dec 2017 14:38:00 +0100 (CET) Received: by cust-asf.ponee.io (Postfix) id CFE98160C0D; Fri, 8 Dec 2017 13:38:00 +0000 (UTC) Delivered-To: archive-asf-public@cust-asf.ponee.io Received: from mail.apache.org (hermes.apache.org [140.211.11.3]) by cust-asf.ponee.io (Postfix) with SMTP id 20F33160BF2 for ; Fri, 8 Dec 2017 14:37:59 +0100 (CET) Received: (qmail 65312 invoked by uid 500); 8 Dec 2017 13:37:59 -0000 Mailing-List: contact user-help@flink.apache.org; run by ezmlm Precedence: bulk List-Help: List-Unsubscribe: List-Post: List-Id: Delivered-To: mailing list user@flink.apache.org Received: (qmail 65292 invoked by uid 99); 8 Dec 2017 13:37:59 -0000 Received: from pnap-us-west-generic-nat.apache.org (HELO spamd2-us-west.apache.org) (209.188.14.142) by apache.org (qpsmtpd/0.29) with ESMTP; Fri, 08 Dec 2017 13:37:58 +0000 Received: from localhost (localhost [127.0.0.1]) by spamd2-us-west.apache.org (ASF Mail Server at spamd2-us-west.apache.org) with ESMTP id 8C7241A05B1 for ; Fri, 8 Dec 2017 13:37:58 +0000 (UTC) X-Virus-Scanned: Debian amavisd-new at spamd2-us-west.apache.org X-Spam-Flag: NO X-Spam-Score: 0.379 X-Spam-Level: X-Spam-Status: No, score=0.379 tagged_above=-999 required=6.31 tests=[DKIM_SIGNED=0.1, DKIM_VALID=-0.1, DKIM_VALID_AU=-0.1, RCVD_IN_DNSWL_NONE=-0.0001, RCVD_IN_MSPIKE_H3=-0.01, RCVD_IN_MSPIKE_WL=-0.01, RCVD_IN_SORBS_SPAM=0.5, SPF_PASS=-0.001] autolearn=disabled Authentication-Results: spamd2-us-west.apache.org (amavisd-new); dkim=pass (2048-bit key) header.d=gmail.com Received: from mx1-lw-us.apache.org ([10.40.0.8]) by localhost (spamd2-us-west.apache.org [10.40.0.9]) (amavisd-new, port 10024) with ESMTP id c1ARQnmZt-5P for ; Fri, 8 Dec 2017 13:37:56 +0000 (UTC) Received: from mail-qt0-f177.google.com (mail-qt0-f177.google.com [209.85.216.177]) by mx1-lw-us.apache.org (ASF Mail Server at mx1-lw-us.apache.org) with ESMTPS id ED31F5F256 for ; Fri, 8 Dec 2017 13:37:55 +0000 (UTC) Received: by mail-qt0-f177.google.com with SMTP id g10so25870363qtj.12 for ; Fri, 08 Dec 2017 05:37:55 -0800 (PST) DKIM-Signature: v=1; a=rsa-sha256; c=relaxed/relaxed; d=gmail.com; s=20161025; h=mime-version:in-reply-to:references:from:date:message-id:subject:to :cc:content-transfer-encoding; bh=WCleWza/jAtf31Z0embL22Q+pACXBUp1wUgiGlIinfw=; b=T5Kkf+ULPKdpFuxABR8l9rcO/+CIJPGAqFeCOhd7nM1J0Js+2sDC6AcNVm9RSymog8 k237x615ejuIJk88AcufHZOsjXF27Qom9pRQ/lD98qH005+5Wl87TvP+PhUXTzzxCy0k 3LL0sOh5i1dso8aavgPueA2vgqY37j+0Ua7ZyqtPjx1jz6+kYJIHHzBALdIB3cMvs2k+ 6rk0MUqa+Kfas1KjEz1SrqwCY82/g7IT6vaTiQVf6bGxmbNv+Du4f0MleV/Vj0i9oeaK YVl/uocXPvsoT1WMqRw/ItPYYpODuetXtes743RY9nm2RIuhj1jtluRP49xzq44POG6e HxiA== X-Google-DKIM-Signature: v=1; a=rsa-sha256; c=relaxed/relaxed; d=1e100.net; s=20161025; h=x-gm-message-state:mime-version:in-reply-to:references:from:date :message-id:subject:to:cc:content-transfer-encoding; bh=WCleWza/jAtf31Z0embL22Q+pACXBUp1wUgiGlIinfw=; b=GIndXa7QhXo4A7ArzafY27q08m4/+swDSN0mVD2tCiatdOmqRrUZm/V2IcNPtLJHBz ssHOEIQJTVmRbkYvkd12yEHCwbMvpis7AliE74DK7cW3OekZPfIrEnRacmYr2rM0cNIW NEPjoATYO4tvgCtEUbGYshffPUlo09BEskp7a9Wk5V/b9DS/PjF7Ps0oldQhp23W6awX iHNJ+aqq6n7y4PCwG107+bTHttDjISDS3SPcBhRh0WLW+cag1AlddhMzeWHPseaIfx0Z D9KlJlhlv+WlzL2w9ByjlCYKMOsIqOH2P4Hd2aRLsddX7bA57DnojqUDTHzcl/114wMW 1Ktw== X-Gm-Message-State: AKGB3mKcL+lohZgjODUcex+okIIHRLLER5WwKnLZH5LVKC3elNiX02l6 uKRfQSSSVcfCSrKOX0RXeHPS4kxoJ2vdp/IaF8nW+A== X-Google-Smtp-Source: AGs4zMYacBDXbF8kyt7IgJd7mkEk6tiKcHOfWCZHugamVTVjwpkKoSXIRGAw7BPzyXDLhfkvpw+ysqXADJ8OQeBoJJE= X-Received: by 10.55.139.129 with SMTP id n123mr32699493qkd.177.1512740275611; Fri, 08 Dec 2017 05:37:55 -0800 (PST) MIME-Version: 1.0 Received: by 10.237.62.67 with HTTP; Fri, 8 Dec 2017 05:37:55 -0800 (PST) In-Reply-To: References: From: Jinhua Luo Date: Fri, 8 Dec 2017 21:37:55 +0800 Message-ID: Subject: Re: does the flink sink only support bio? To: Stefan Richter Cc: user@flink.apache.org Content-Type: text/plain; charset="UTF-8" Content-Transfer-Encoding: quoted-printable archived-at: Fri, 08 Dec 2017 13:38:01 -0000 Thank you very much! I have two new questions: 1) the async operator must emit some value to the async collector (even it acts as a sink), right? 2) How could I use CheckpointListener with async operator? Could you give a simple example or doc page? 2017-12-08 18:25 GMT+08:00 Stefan Richter : > Hi, > > Flink currently does not offer async sinks out of the box, but there is n= o fundamental problem against having them and we will probably offer someth= ing is this direction in the future. In the meantime, you can build somethi= ng like this by replacing the sink with an async io operator that acts as s= ink (i.e. does the writes to the db) followed by a discarding sink for comp= liance with the API. > > You need to be a bit careful if your sink needs exactly-once semantics. I= n this case things should either be idempotent or the db must support rolli= ng back changes between checkpoints, e.g. via transactions. Commits should = be triggered for confirmed checkpoints (=E2=80=9EnotifyCheckpointComplete= =E2=80=9C). > > Your assumptions about the blocking behavior of the non-async sinks is co= rrect. > > Best, > Stefan > >> Am 08.12.2017 um 08:11 schrieb Jinhua Luo : >> >> Hi, all. >> >> The invoke method of sink seems no way to make async io? e.g. returns Fu= ture? >> >> For example, the redis connector uses jedis lib to execute redis >> command synchronously: >> >> https://github.com/apache/bahir-flink/blob/master/flink-connector-redis/= src/main/java/org/apache/flink/streaming/connectors/redis/RedisSink.java >> >> Then it will block the task thread of flink waiting the network >> response from redis server per command?! Is it possible for other >> operators running in the same thread with sink? If so, then it would >> block them too? >> >> I know flink has asyncio api, but it seems not for used by sink impl? >> >> https://ci.apache.org/projects/flink/flink-docs-release-1.3/dev/stream/a= syncio.html >> >> Thanks. >