1 /*
2  * librdkafka - Apache Kafka C library
3  *
4  * Copyright (c) 2012-2018, Magnus Edenhill
5  * All rights reserved.
6  *
7  * Redistribution and use in source and binary forms, with or without
8  * modification, are permitted provided that the following conditions are met:
9  *
10  * 1. Redistributions of source code must retain the above copyright notice,
11  *    this list of conditions and the following disclaimer.
12  * 2. Redistributions in binary form must reproduce the above copyright notice,
13  *    this list of conditions and the following disclaimer in the documentation
14  *    and/or other materials provided with the distribution.
15  *
16  * THIS SOFTWARE IS PROVIDED BY THE COPYRIGHT HOLDERS AND CONTRIBUTORS "AS IS"
17  * AND ANY EXPRESS OR IMPLIED WARRANTIES, INCLUDING, BUT NOT LIMITED TO, THE
18  * IMPLIED WARRANTIES OF MERCHANTABILITY AND FITNESS FOR A PARTICULAR PURPOSE
19  * ARE DISCLAIMED. IN NO EVENT SHALL THE COPYRIGHT OWNER OR CONTRIBUTORS BE
20  * LIABLE FOR ANY DIRECT, INDIRECT, INCIDENTAL, SPECIAL, EXEMPLARY, OR
21  * CONSEQUENTIAL DAMAGES (INCLUDING, BUT NOT LIMITED TO, PROCUREMENT OF
22  * SUBSTITUTE GOODS OR SERVICES; LOSS OF USE, DATA, OR PROFITS; OR BUSINESS
23  * INTERRUPTION) HOWEVER CAUSED AND ON ANY THEORY OF LIABILITY, WHETHER IN
24  * CONTRACT, STRICT LIABILITY, OR TORT (INCLUDING NEGLIGENCE OR OTHERWISE)
25  * ARISING IN ANY WAY OUT OF THE USE OF THIS SOFTWARE, EVEN IF ADVISED OF THE
26  * POSSIBILITY OF SUCH DAMAGE.
27  */
28 
29 #include "test.h"
30 #include "rdkafka.h"
31 
32 
33 /**
34  * @brief Attempt to verify head-of-line-blocking behaviour.
35  *
36  * - Create two high-level consumers with socket.timeout.ms=low,
37  *   and max.poll.interval.ms=high, metadata refresh interval=low.
38  * - Have first consumer join the group (subscribe()), should finish quickly.
39  * - Have second consumer join the group, but don't call poll on
40  *   the first consumer for some time to have the second consumer
41  *   block on JoinGroup.
42  * - Verify that errors were raised due to timed out (Metadata) requests.
43  */
44 
45 struct _consumer {
46         rd_kafka_t *rk;
47         int64_t last;
48         int cnt;
49         int rebalance_cnt;
50         int max_rebalance_cnt;
51 };
52 
do_consume(struct _consumer * cons,int timeout_s)53 static void do_consume (struct _consumer *cons, int timeout_s) {
54         rd_kafka_message_t *rkm;
55 
56         rkm = rd_kafka_consumer_poll(cons->rk, 100+(timeout_s*1000));
57         if (!rkm)
58                 return;
59 
60         TEST_ASSERT(!rkm->err,
61                     "%s consumer error: %s (last poll was %dms ago)",
62                     rd_kafka_name(cons->rk),
63                     rd_kafka_message_errstr(rkm),
64                     (int)((test_clock() - cons->last)/1000));
65 
66         rd_kafka_message_destroy(rkm);
67 
68         cons->cnt++;
69         cons->last = test_clock();
70 
71         if (timeout_s > 0) {
72                 TEST_SAY("%s: simulate processing by sleeping for %ds\n",
73                          rd_kafka_name(cons->rk), timeout_s);
74                 rd_sleep(timeout_s);
75         }
76 }
77 
78 
rebalance_cb(rd_kafka_t * rk,rd_kafka_resp_err_t err,rd_kafka_topic_partition_list_t * parts,void * opaque)79 static void rebalance_cb (rd_kafka_t *rk,
80                           rd_kafka_resp_err_t err,
81                           rd_kafka_topic_partition_list_t *parts,
82                           void *opaque) {
83         struct _consumer *cons = opaque;
84 
85         cons->rebalance_cnt++;
86 
87         TEST_SAY(_C_BLU "%s rebalance #%d/%d: %s: %d partition(s)\n",
88                  rd_kafka_name(cons->rk),
89                  cons->rebalance_cnt, cons->max_rebalance_cnt,
90                  rd_kafka_err2name(err),
91                  parts->cnt);
92 
93         TEST_ASSERT(cons->rebalance_cnt <= cons->max_rebalance_cnt,
94                     "%s rebalanced %d times, max was %d",
95                     rd_kafka_name(cons->rk),
96                     cons->rebalance_cnt, cons->max_rebalance_cnt);
97 
98         if (err == RD_KAFKA_RESP_ERR__ASSIGN_PARTITIONS)
99                 rd_kafka_assign(rk, parts);
100         else
101                 rd_kafka_assign(rk, NULL);
102 }
103 
104 
105 #define _CONSUMER_CNT 2
main_0093_holb_consumer(int argc,char ** argv)106 int main_0093_holb_consumer (int argc, char **argv) {
107         const char *topic = test_mk_topic_name("0093_holb_consumer", 1);
108         int64_t testid;
109         const int msgcnt = 100;
110         struct _consumer c[_CONSUMER_CNT] = RD_ZERO_INIT;
111         rd_kafka_conf_t *conf;
112 
113         testid = test_id_generate();
114 
115         test_conf_init(&conf, NULL, 60);
116 
117         test_create_topic(NULL, topic, 1, 1);
118 
119         test_produce_msgs_easy(topic, testid, 0, msgcnt);
120 
121         test_conf_set(conf, "session.timeout.ms", "6000");
122         test_conf_set(conf, "max.poll.interval.ms", "20000");
123         test_conf_set(conf, "socket.timeout.ms", "3000");
124         test_conf_set(conf, "auto.offset.reset", "earliest");
125         /* Trigger other requests often */
126         test_conf_set(conf, "topic.metadata.refresh.interval.ms", "500");
127         rd_kafka_conf_set_rebalance_cb(conf, rebalance_cb);
128 
129         rd_kafka_conf_set_opaque(conf, &c[0]);
130         c[0].rk = test_create_consumer(topic, NULL,
131                                        rd_kafka_conf_dup(conf), NULL);
132 
133         rd_kafka_conf_set_opaque(conf, &c[1]);
134         c[1].rk = test_create_consumer(topic, NULL, conf, NULL);
135 
136         test_consumer_subscribe(c[0].rk, topic);
137 
138         /* c0: assign */
139         c[0].max_rebalance_cnt = 1;
140 
141         /* c1: none, hasn't joined yet */
142         c[1].max_rebalance_cnt = 0;
143 
144         TEST_SAY("Waiting for c[0] assignment\n");
145         while (1) {
146                 rd_kafka_topic_partition_list_t *parts = NULL;
147 
148                 do_consume(&c[0], 1/*1s*/);
149 
150                 if (rd_kafka_assignment(c[0].rk, &parts) !=
151                     RD_KAFKA_RESP_ERR_NO_ERROR ||
152                     !parts || parts->cnt == 0) {
153                         if (parts)
154                                 rd_kafka_topic_partition_list_destroy(parts);
155                         continue;
156                 }
157 
158                 TEST_SAY("%s got assignment of %d partition(s)\n",
159                          rd_kafka_name(c[0].rk), parts->cnt);
160                 rd_kafka_topic_partition_list_destroy(parts);
161                 break;
162         }
163 
164         TEST_SAY("c[0] got assignment, consuming..\n");
165         do_consume(&c[0], 5/*5s*/);
166 
167         TEST_SAY("Joining second consumer\n");
168         test_consumer_subscribe(c[1].rk, topic);
169 
170         /* Just poll second consumer for 10s, the rebalance will not
171          * finish until the first consumer polls */
172         do_consume(&c[1], 10/*10s*/);
173 
174         /* c0: the next call to do_consume/poll will trigger
175          *     its rebalance callback, first revoke then assign. */
176         c[0].max_rebalance_cnt += 2;
177         /* c1: first rebalance */
178         c[1].max_rebalance_cnt++;
179 
180         TEST_SAY("Expected rebalances: c[0]: %d/%d, c[1]: %d/%d\n",
181                  c[0].rebalance_cnt, c[0].max_rebalance_cnt,
182                  c[1].rebalance_cnt, c[1].max_rebalance_cnt);
183 
184         /* Let rebalances kick in, then consume messages. */
185         while (c[0].cnt + c[1].cnt < msgcnt) {
186                 do_consume(&c[0], 0);
187                 do_consume(&c[1], 0);
188         }
189 
190         /* Allow the extra revoke rebalance on close() */
191         c[0].max_rebalance_cnt++;
192         c[1].max_rebalance_cnt++;
193 
194         test_consumer_close(c[0].rk);
195         test_consumer_close(c[1].rk);
196 
197         rd_kafka_destroy(c[0].rk);
198         rd_kafka_destroy(c[1].rk);
199 
200         return 0;
201 }
202