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 44AAD200C2B for ; Thu, 16 Feb 2017 01:05:24 +0100 (CET) Received: by cust-asf.ponee.io (Postfix) id 4326C160B70; Thu, 16 Feb 2017 00:05:24 +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 60462160B5E for ; Thu, 16 Feb 2017 01:05:23 +0100 (CET) Received: (qmail 42136 invoked by uid 500); 16 Feb 2017 00:05:17 -0000 Mailing-List: contact dev-help@kafka.apache.org; run by ezmlm Precedence: bulk List-Help: List-Unsubscribe: List-Post: List-Id: Reply-To: dev@kafka.apache.org Delivered-To: mailing list dev@kafka.apache.org Received: (qmail 42114 invoked by uid 99); 16 Feb 2017 00:05:17 -0000 Received: from pnap-us-west-generic-nat.apache.org (HELO spamd1-us-west.apache.org) (209.188.14.142) by apache.org (qpsmtpd/0.29) with ESMTP; Thu, 16 Feb 2017 00:05:17 +0000 Received: from localhost (localhost [127.0.0.1]) by spamd1-us-west.apache.org (ASF Mail Server at spamd1-us-west.apache.org) with ESMTP id C1F18C69DC for ; Thu, 16 Feb 2017 00:05:16 +0000 (UTC) X-Virus-Scanned: Debian amavisd-new at spamd1-us-west.apache.org X-Spam-Flag: NO X-Spam-Score: -0.002 X-Spam-Level: X-Spam-Status: No, score=-0.002 tagged_above=-999 required=6.31 tests=[DKIM_SIGNED=0.1, DKIM_VALID=-0.1, RCVD_IN_DNSWL_NONE=-0.0001, RCVD_IN_MSPIKE_H2=-0.001, SPF_PASS=-0.001] autolearn=disabled Authentication-Results: spamd1-us-west.apache.org (amavisd-new); dkim=pass (2048-bit key) header.d=confluent-io.20150623.gappssmtp.com Received: from mx1-lw-us.apache.org ([10.40.0.8]) by localhost (spamd1-us-west.apache.org [10.40.0.7]) (amavisd-new, port 10024) with ESMTP id dNN7AdM_drLt for ; Thu, 16 Feb 2017 00:05:14 +0000 (UTC) Received: from mail-pf0-f179.google.com (mail-pf0-f179.google.com [209.85.192.179]) by mx1-lw-us.apache.org (ASF Mail Server at mx1-lw-us.apache.org) with ESMTPS id 1DD575FE16 for ; Thu, 16 Feb 2017 00:05:14 +0000 (UTC) Received: by mail-pf0-f179.google.com with SMTP id 189so585841pfu.3 for ; Wed, 15 Feb 2017 16:05:14 -0800 (PST) DKIM-Signature: v=1; a=rsa-sha256; c=relaxed/relaxed; d=confluent-io.20150623.gappssmtp.com; s=20150623; h=subject:to:references:from:organization:message-id:date:user-agent :mime-version:in-reply-to; bh=Nj/zwdeOLEW7DHtgTPH0xE36N2GlPpyTPK8HNkscQMM=; b=AQnRfz4e+fvQ5n3n1RKA6gVqYBreZ1zSNVz76/5Ahlz2t8JM7jljNfGGIlqzP9Nt42 I732nK6eq0WXi9wX05ctsDOr1x3+RbBzPATIO35Y15dy3+tPt7nhgJY8NvKSTq4Ef6K/ IqUhzIut12ei+7RKgJ+0JlczK1DoB66sWY18p1JqUsV8oqHaE625PDSr/sS1CPxVi87t 6FKM4DWtiYPwvtED3Y+9NrnjqN67PZGbvnGYXdomihZBX2ZrChBw2UTQQoE+gcuxowqB MuKdF1afQSwEqTsvroo7XTdyyWZoNYrII8J7zXWPetGTZ22B6LUGbPUM2KbrldELXdgC x1Iw== X-Google-DKIM-Signature: v=1; a=rsa-sha256; c=relaxed/relaxed; d=1e100.net; s=20161025; h=x-gm-message-state:subject:to:references:from:organization :message-id:date:user-agent:mime-version:in-reply-to; bh=Nj/zwdeOLEW7DHtgTPH0xE36N2GlPpyTPK8HNkscQMM=; b=BLb0xF/7Lsg2r99UrH2c8l8FHjWp6E8Ikv2ScSBd29qp1ysOp2QWFx3ptKAGThem/W To3U4tCD8/Makk/YX62UscCu2d00VA8NQHunUeV4F0DZoGV7XGKzO2WSZ4aYn+omZcnj 5BM+5tGQK+aK7ktgrZATJq8UTiKRqv2qu2P0UOL4INXDNSCgoZtyQtc26Cm1eKUuXVCD b2YMq/fPFKgbDCeMfJqG1ED9pYAp0MunVq7tA6fnjMIcAIti3UAGmoRO+dYsGj7umJOI ng8SuqEunsVaONpYzh8RczTGmxKvTzCHpbhAHS54UMtP+C3YGhRX5EsTdNvbAyyeQg5b N/IA== X-Gm-Message-State: AMke39mh5YF1BytD+v/PZo4WRw91GQ0U7EoM6niJfmFqWDpQ/Q4dDcFT2ZykWknwOezgFRev X-Received: by 10.99.168.2 with SMTP id o2mr41303078pgf.159.1487203496380; Wed, 15 Feb 2017 16:04:56 -0800 (PST) Received: from Matthias-Sax-Macbook-Pro.local ([216.3.4.180]) by smtp.gmail.com with ESMTPSA id x30sm9470977pgc.67.2017.02.15.16.04.55 (version=TLS1_2 cipher=ECDHE-RSA-AES128-GCM-SHA256 bits=128/128); Wed, 15 Feb 2017 16:04:55 -0800 (PST) Subject: Re: [DISCUSS] KIP-120: Cleanup Kafka Streams builder API To: users@kafka.apache.org, "dev@kafka.apache.org" References: <21df6e5c-3aed-ac03-339a-0c7526c4392a@confluent.io> <87c6103a-08b9-b895-d8bd-5fbd5885b550@confluent.io> <2798abbf-02da-6f1c-f4d9-3ff4e238fab0@confluent.io> From: "Matthias J. Sax" Organization: Confluent Inc Message-ID: <2ba66790-e3c9-a3d1-a801-b3835a3a3559@confluent.io> Date: Wed, 15 Feb 2017 16:04:54 -0800 User-Agent: Mozilla/5.0 (Macintosh; Intel Mac OS X 10.12; rv:45.0) Gecko/20100101 Thunderbird/45.7.1 MIME-Version: 1.0 In-Reply-To: <2798abbf-02da-6f1c-f4d9-3ff4e238fab0@confluent.io> Content-Type: multipart/signed; micalg=pgp-sha512; protocol="application/pgp-signature"; boundary="VAgHLaTue8eT4AdgEKgkBLMt2EwOd6Pq1" archived-at: Thu, 16 Feb 2017 00:05:24 -0000 --VAgHLaTue8eT4AdgEKgkBLMt2EwOd6Pq1 Content-Type: multipart/mixed; boundary="213eSAc1ouI6GdJmoQnwRkaIWJXSHNltX"; protected-headers="v1" From: "Matthias J. Sax" To: users@kafka.apache.org, "dev@kafka.apache.org" Message-ID: <2ba66790-e3c9-a3d1-a801-b3835a3a3559@confluent.io> Subject: Re: [DISCUSS] KIP-120: Cleanup Kafka Streams builder API References: <21df6e5c-3aed-ac03-339a-0c7526c4392a@confluent.io> <87c6103a-08b9-b895-d8bd-5fbd5885b550@confluent.io> <2798abbf-02da-6f1c-f4d9-3ff4e238fab0@confluent.io> In-Reply-To: <2798abbf-02da-6f1c-f4d9-3ff4e238fab0@confluent.io> --213eSAc1ouI6GdJmoQnwRkaIWJXSHNltX Content-Type: text/plain; charset=utf-8 Content-Transfer-Encoding: quoted-printable Hi, according to the feedback, I updated the KIP, and limited its scope to some extend: - Instead of changing the creation of KafkaStreams instances, we keep the current pattern (we might do a follow up KIP on this though). - We also added a new method #describe() that returns a TopologyDescription that can be used for any "pretty print" functionality or similar. - We also removed method #topologyBuilder() from KStreamBuilder because we think #transform() should provide all functionality you need to mix-an-match Processor API and DSL. If there is any further concern about this, please let us know. -Matthias On 2/14/17 9:59 AM, Matthias J. Sax wrote: > You can already output any number of record within .transform() using > the provided Context object from init()... >=20 >=20 > -Matthias >=20 > On 2/14/17 9:16 AM, Guozhang Wang wrote: >>> and you can't output multiple records or branching logic from a >> transform(); >> >> For output multiple records in transform, we are currently working on >> https://issues.apache.org/jira/browse/KAFKA-4217, I think that should = cover >> this use case. >> >> For branching the output in transform, I agree this is not perfect but= I >> think users can follow some patterns like "stream.transform().branch()= ", >> would that work for you? >> >> >> Guozhang >> >> >> On Tue, Feb 14, 2017 at 8:29 AM, Mathieu Fenniak < >> mathieu.fenniak@replicon.com> wrote: >> >>> On Tue, Feb 14, 2017 at 1:14 AM, Guozhang Wang w= rote: >>> >>>> Some thoughts on the mixture usage of DSL / PAPI: >>>> >>>> There were some suggestions on mixing the usage of DSL and PAPI: >>>> https://issues.apache.org/jira/browse/KAFKA-3455, and after thinking= it >>> a >>>> bit more carefully, I'd rather not recommend users following this >>> pattern, >>>> since in DSL this can always be achieved in process() / transform().= >>> Hence >>>> I think it is okay to prevent such patterns in the new APIs. And for= the >>>> same reasons, I think we can remove KStreamBuilder#newName() from th= e >>>> public APIs. >>>> >>> >>> I'm not sure that things can always be achieved by process() / >>> transform()... there are some limitations to these APIs. You can't o= utput >>> from a process(), and you can't output multiple records or branching = logic >>> from a transform(); these are things that can be done in the PAPI qui= te >>> easily. >>> >>> I definitely understand a preference for using process()/transform() = where >>> possible, but, they don't seem to replace the PAPI. >>> >>> I would love to be operating in a world that was entirely DSL. But t= he DSL >>> is limited, and it isn't extensible (... by any stable API). I don't= mind >>> reaching into internals today and making my own life difficult to ext= end >>> it, and I'd continue to find a way to do that if you made the APIs di= stinct >>> and split, but I'm just expressing my preference that you not do that= =2E :-) >>> >>> And about printing the topology for debuggability: I agrees this is a= >>>> potential drawback, and I'd suggest maintain some functionality to b= uild >>> a >>>> "dry topology" as Mathieu suggested; the difficulty is that, interna= lly >>> we >>>> need a different "copy" of the topology for each thread so that they= will >>>> not share any states, so we cannot directly pass in the topology int= o >>>> KafkaStreams instead of the topology builder. So how about adding a >>>> `ToplogyBuilder#toString` function which calls `build()` internally = then >>>> prints the built dry topology? >>>> >>> >>> Well, this sounds better than KafkaStreams#toString() in that it does= n't >>> require a running processor. But I'd really love to have a simple ob= ject >>> model for the topology, not a string output, so that I can output my = own >>> debug format. I currently have that in the form of >>> TopologyBuilder#nodeGroups() & TopologyBuilder#build(Integer). >>> >>> Mathieu >>> >> >> >> >=20 --213eSAc1ouI6GdJmoQnwRkaIWJXSHNltX-- --VAgHLaTue8eT4AdgEKgkBLMt2EwOd6Pq1 Content-Type: application/pgp-signature; name="signature.asc" Content-Description: OpenPGP digital signature Content-Disposition: attachment; filename="signature.asc" -----BEGIN PGP SIGNATURE----- Comment: GPGTools - https://gpgtools.org iQIcBAEBCgAGBQJYpOynAAoJECnhiMLycopPjl4P/2qgbwN003PP6zdbjWdadJrO tzwIVj7H5cjQQwgzEeqlbHGTXTL80Z1Slt1fIqR+VqssYGCPOWiKp3lix7wBAcud JmnvhKZl3iBLqWm3qFJaz9nA6ckszY5ciQ/TGeJpjejfLs9rJfny/6LYRioHE079 o7upzZKUECoU4VSf6lDuNoHQD79VCD3oKZP1vs/1GsZLYrOvUDuIZtW80WIAiPfw 1VAK2pfIkmC+9arNyDMSJyc//n9+2lFPZWlAkrclQcPBcA/YOcdfl7tZ3O2IdDaG S09ggpnuNgroavs62vs/QTuBNoYiZF98hLLyETBG1cDc3v/BORY5CkIuK8NK+rJ5 kPVYbE9mEKpAi5VXkvfVEs9rmImqlRLxK8/WXEqcyqB/enVyHD2yOTBwqybDzUAH NJQD8OXpLmt1BFCVg6iukOGD72XFR8GqH6lqFYHHTC6A/WbzuaQJ6zDrbbsuvtfB BWxqgDCFqoNbOVcdx3S/+etQ2sKgpqXdDC6CdUmLDJA6yAhxK4Z98qA7ZkBkH7NK lA8neLXxetwWpSiu4Nld0uxudqztf0r4RHAP9QGbuK83gaaW0nLm2UxNaDd0tLCn lc4zI+Ai+56nAbfVCZqPRGlDshNioVObc1v5xdALvTQ20z8Jgx0qwCk6b3qtiy3q y/sJdWVPxh2QHGWnjLZ/ =OYXp -----END PGP SIGNATURE----- --VAgHLaTue8eT4AdgEKgkBLMt2EwOd6Pq1--