-
Notifications
You must be signed in to change notification settings - Fork 3.2k
/
rdkafka_partition.c
4801 lines (3948 loc) · 174 KB
/
rdkafka_partition.c
1
2
3
4
5
6
7
8
9
10
11
12
13
14
15
16
17
18
19
20
21
22
23
24
25
26
27
28
29
30
31
32
33
34
35
36
37
38
39
40
41
42
43
44
45
46
47
48
49
50
51
52
53
54
55
56
57
58
59
60
61
62
63
64
65
66
67
68
69
70
71
72
73
74
75
76
77
78
79
80
81
82
83
84
85
86
87
88
89
90
91
92
93
94
95
96
97
98
99
100
101
102
103
104
105
106
107
108
109
110
111
112
113
114
115
116
117
118
119
120
121
122
123
124
125
126
127
128
129
130
131
132
133
134
135
136
137
138
139
140
141
142
143
144
145
146
147
148
149
150
151
152
153
154
155
156
157
158
159
160
161
162
163
164
165
166
167
168
169
170
171
172
173
174
175
176
177
178
179
180
181
182
183
184
185
186
187
188
189
190
191
192
193
194
195
196
197
198
199
200
201
202
203
204
205
206
207
208
209
210
211
212
213
214
215
216
217
218
219
220
221
222
223
224
225
226
227
228
229
230
231
232
233
234
235
236
237
238
239
240
241
242
243
244
245
246
247
248
249
250
251
252
253
254
255
256
257
258
259
260
261
262
263
264
265
266
267
268
269
270
271
272
273
274
275
276
277
278
279
280
281
282
283
284
285
286
287
288
289
290
291
292
293
294
295
296
297
298
299
300
301
302
303
304
305
306
307
308
309
310
311
312
313
314
315
316
317
318
319
320
321
322
323
324
325
326
327
328
329
330
331
332
333
334
335
336
337
338
339
340
341
342
343
344
345
346
347
348
349
350
351
352
353
354
355
356
357
358
359
360
361
362
363
364
365
366
367
368
369
370
371
372
373
374
375
376
377
378
379
380
381
382
383
384
385
386
387
388
389
390
391
392
393
394
395
396
397
398
399
400
401
402
403
404
405
406
407
408
409
410
411
412
413
414
415
416
417
418
419
420
421
422
423
424
425
426
427
428
429
430
431
432
433
434
435
436
437
438
439
440
441
442
443
444
445
446
447
448
449
450
451
452
453
454
455
456
457
458
459
460
461
462
463
464
465
466
467
468
469
470
471
472
473
474
475
476
477
478
479
480
481
482
483
484
485
486
487
488
489
490
491
492
493
494
495
496
497
498
499
500
501
502
503
504
505
506
507
508
509
510
511
512
513
514
515
516
517
518
519
520
521
522
523
524
525
526
527
528
529
530
531
532
533
534
535
536
537
538
539
540
541
542
543
544
545
546
547
548
549
550
551
552
553
554
555
556
557
558
559
560
561
562
563
564
565
566
567
568
569
570
571
572
573
574
575
576
577
578
579
580
581
582
583
584
585
586
587
588
589
590
591
592
593
594
595
596
597
598
599
600
601
602
603
604
605
606
607
608
609
610
611
612
613
614
615
616
617
618
619
620
621
622
623
624
625
626
627
628
629
630
631
632
633
634
635
636
637
638
639
640
641
642
643
644
645
646
647
648
649
650
651
652
653
654
655
656
657
658
659
660
661
662
663
664
665
666
667
668
669
670
671
672
673
674
675
676
677
678
679
680
681
682
683
684
685
686
687
688
689
690
691
692
693
694
695
696
697
698
699
700
701
702
703
704
705
706
707
708
709
710
711
712
713
714
715
716
717
718
719
720
721
722
723
724
725
726
727
728
729
730
731
732
733
734
735
736
737
738
739
740
741
742
743
744
745
746
747
748
749
750
751
752
753
754
755
756
757
758
759
760
761
762
763
764
765
766
767
768
769
770
771
772
773
774
775
776
777
778
779
780
781
782
783
784
785
786
787
788
789
790
791
792
793
794
795
796
797
798
799
800
801
802
803
804
805
806
807
808
809
810
811
812
813
814
815
816
817
818
819
820
821
822
823
824
825
826
827
828
829
830
831
832
833
834
835
836
837
838
839
840
841
842
843
844
845
846
847
848
849
850
851
852
853
854
855
856
857
858
859
860
861
862
863
864
865
866
867
868
869
870
871
872
873
874
875
876
877
878
879
880
881
882
883
884
885
886
887
888
889
890
891
892
893
894
895
896
897
898
899
900
901
902
903
904
905
906
907
908
909
910
911
912
913
914
915
916
917
918
919
920
921
922
923
924
925
926
927
928
929
930
931
932
933
934
935
936
937
938
939
940
941
942
943
944
945
946
947
948
949
950
951
952
953
954
955
956
957
958
959
960
961
962
963
964
965
966
967
968
969
970
971
972
973
974
975
976
977
978
979
980
981
982
983
984
985
986
987
988
989
990
991
992
993
994
995
996
997
998
999
1000
/*
* librdkafka - The Apache Kafka C/C++ library
*
* Copyright (c) 2015-2022, Magnus Edenhill,
* 2023, Confluent Inc.
* All rights reserved.
*
* Redistribution and use in source and binary forms, with or without
* modification, are permitted provided that the following conditions are met:
*
* 1. Redistributions of source code must retain the above copyright notice,
* this list of conditions and the following disclaimer.
* 2. Redistributions in binary form must reproduce the above copyright notice,
* this list of conditions and the following disclaimer in the documentation
* and/or other materials provided with the distribution.
*
* THIS SOFTWARE IS PROVIDED BY THE COPYRIGHT HOLDERS AND CONTRIBUTORS "AS IS"
* AND ANY EXPRESS OR IMPLIED WARRANTIES, INCLUDING, BUT NOT LIMITED TO, THE
* IMPLIED WARRANTIES OF MERCHANTABILITY AND FITNESS FOR A PARTICULAR PURPOSE
* ARE DISCLAIMED. IN NO EVENT SHALL THE COPYRIGHT OWNER OR CONTRIBUTORS BE
* LIABLE FOR ANY DIRECT, INDIRECT, INCIDENTAL, SPECIAL, EXEMPLARY, OR
* CONSEQUENTIAL DAMAGES (INCLUDING, BUT NOT LIMITED TO, PROCUREMENT OF
* SUBSTITUTE GOODS OR SERVICES; LOSS OF USE, DATA, OR PROFITS; OR BUSINESS
* INTERRUPTION) HOWEVER CAUSED AND ON ANY THEORY OF LIABILITY, WHETHER IN
* CONTRACT, STRICT LIABILITY, OR TORT (INCLUDING NEGLIGENCE OR OTHERWISE)
* ARISING IN ANY WAY OUT OF THE USE OF THIS SOFTWARE, EVEN IF ADVISED OF THE
* POSSIBILITY OF SUCH DAMAGE.
*/
#include "rdkafka_int.h"
#include "rdkafka_topic.h"
#include "rdkafka_broker.h"
#include "rdkafka_request.h"
#include "rdkafka_offset.h"
#include "rdkafka_partition.h"
#include "rdkafka_fetcher.h"
#include "rdregex.h"
#include "rdports.h" /* rd_qsort_r() */
#include "rdunittest.h"
const char *rd_kafka_fetch_states[] = {"none", "stopping",
"stopped", "offset-query",
"offset-wait", "validate-epoch-wait",
"active"};
static rd_kafka_op_res_t rd_kafka_toppar_op_serve(rd_kafka_t *rk,
rd_kafka_q_t *rkq,
rd_kafka_op_t *rko,
rd_kafka_q_cb_type_t cb_type,
void *opaque);
static void rd_kafka_toppar_offset_retry(rd_kafka_toppar_t *rktp,
int backoff_ms,
const char *reason);
static RD_INLINE int32_t
rd_kafka_toppar_version_new_barrier0(rd_kafka_toppar_t *rktp,
const char *func,
int line) {
int32_t version = rd_atomic32_add(&rktp->rktp_version, 1);
rd_kafka_dbg(rktp->rktp_rkt->rkt_rk, TOPIC, "BARRIER",
"%s [%" PRId32 "]: %s:%d: new version barrier v%" PRId32,
rktp->rktp_rkt->rkt_topic->str, rktp->rktp_partition, func,
line, version);
return version;
}
#define rd_kafka_toppar_version_new_barrier(rktp) \
rd_kafka_toppar_version_new_barrier0(rktp, __FUNCTION__, __LINE__)
/**
* Toppar based OffsetResponse handling.
* This is used for updating the low water mark for consumer lag.
*/
static void rd_kafka_toppar_lag_handle_Offset(rd_kafka_t *rk,
rd_kafka_broker_t *rkb,
rd_kafka_resp_err_t err,
rd_kafka_buf_t *rkbuf,
rd_kafka_buf_t *request,
void *opaque) {
rd_kafka_toppar_t *rktp = opaque;
rd_kafka_topic_partition_list_t *offsets;
rd_kafka_topic_partition_t *rktpar;
offsets = rd_kafka_topic_partition_list_new(1);
/* Parse and return Offset */
err = rd_kafka_handle_ListOffsets(rk, rkb, err, rkbuf, request, offsets,
NULL);
if (err == RD_KAFKA_RESP_ERR__IN_PROGRESS) {
rd_kafka_topic_partition_list_destroy(offsets);
return; /* Retrying */
}
if (!err && !(rktpar = rd_kafka_topic_partition_list_find(
offsets, rktp->rktp_rkt->rkt_topic->str,
rktp->rktp_partition)))
err = RD_KAFKA_RESP_ERR__UNKNOWN_PARTITION;
if (!err && !rktpar->err) {
rd_kafka_toppar_lock(rktp);
rktp->rktp_lo_offset = rktpar->offset;
rd_kafka_toppar_unlock(rktp);
}
rd_kafka_topic_partition_list_destroy(offsets);
rktp->rktp_wait_consumer_lag_resp = 0;
rd_kafka_toppar_destroy(rktp); /* from request.opaque */
}
/**
* Request information from broker to keep track of consumer lag.
*
* @locality toppar handle thread
* @locks none
*/
static void rd_kafka_toppar_consumer_lag_req(rd_kafka_toppar_t *rktp) {
rd_kafka_topic_partition_list_t *partitions;
rd_kafka_topic_partition_t *rktpar;
if (rktp->rktp_wait_consumer_lag_resp)
return; /* Previous request not finished yet */
rd_kafka_toppar_lock(rktp);
/* Offset requests can only be sent to the leader replica.
*
* Note: If rktp is delegated to a preferred replica, it is
* certain that FETCH >= v5 and so rktp_lo_offset will be
* updated via LogStartOffset in the FETCH response.
*/
if (!rktp->rktp_leader || (rktp->rktp_leader != rktp->rktp_broker)) {
rd_kafka_toppar_unlock(rktp);
return;
}
/* Also don't send a timed log start offset request if leader
* broker supports FETCH >= v5, since this will be set when
* doing fetch requests.
*/
if (rd_kafka_broker_ApiVersion_supported(
rktp->rktp_broker, RD_KAFKAP_Fetch, 0, 5, NULL) == 5) {
rd_kafka_toppar_unlock(rktp);
return;
}
rktp->rktp_wait_consumer_lag_resp = 1;
partitions = rd_kafka_topic_partition_list_new(1);
rktpar = rd_kafka_topic_partition_list_add(
partitions, rktp->rktp_rkt->rkt_topic->str, rktp->rktp_partition);
rktpar->offset = RD_KAFKA_OFFSET_BEGINNING;
rd_kafka_topic_partition_set_current_leader_epoch(
rktpar, rktp->rktp_leader_epoch);
/* Ask for oldest offset. The newest offset is automatically
* propagated in FetchResponse.HighwaterMark. */
rd_kafka_ListOffsetsRequest(rktp->rktp_broker, partitions,
RD_KAFKA_REPLYQ(rktp->rktp_ops, 0),
rd_kafka_toppar_lag_handle_Offset,
-1, /* don't set an absolute timeout */
rd_kafka_toppar_keep(rktp));
rd_kafka_toppar_unlock(rktp);
rd_kafka_topic_partition_list_destroy(partitions);
}
/**
* Request earliest offset for a partition
*
* Locality: toppar handler thread
*/
static void rd_kafka_toppar_consumer_lag_tmr_cb(rd_kafka_timers_t *rkts,
void *arg) {
rd_kafka_toppar_t *rktp = arg;
rd_kafka_toppar_consumer_lag_req(rktp);
}
/**
* @brief Update rktp_op_version.
* Enqueue an RD_KAFKA_OP_BARRIER type of operation
* when the op_version is updated.
*
* @locks_required rd_kafka_toppar_lock() must be held.
* @locality Toppar handler thread
*/
void rd_kafka_toppar_op_version_bump(rd_kafka_toppar_t *rktp, int32_t version) {
rd_kafka_op_t *rko;
rktp->rktp_op_version = version;
rko = rd_kafka_op_new(RD_KAFKA_OP_BARRIER);
rko->rko_version = version;
rko->rko_prio = RD_KAFKA_PRIO_FLASH;
rko->rko_rktp = rd_kafka_toppar_keep(rktp);
rd_kafka_q_enq(rktp->rktp_fetchq, rko);
}
/**
* Add new partition to topic.
*
* Locks: rd_kafka_topic_wrlock() must be held.
* Locks: rd_kafka_wrlock() must be held.
*/
rd_kafka_toppar_t *rd_kafka_toppar_new0(rd_kafka_topic_t *rkt,
int32_t partition,
const char *func,
int line) {
rd_kafka_toppar_t *rktp;
rktp = rd_calloc(1, sizeof(*rktp));
rktp->rktp_partition = partition;
rktp->rktp_rkt = rkt;
rktp->rktp_leader_id = -1;
rktp->rktp_broker_id = -1;
rktp->rktp_leader_epoch = -1;
rd_interval_init(&rktp->rktp_lease_intvl);
rd_interval_init(&rktp->rktp_new_lease_intvl);
rd_interval_init(&rktp->rktp_new_lease_log_intvl);
rd_interval_init(&rktp->rktp_metadata_intvl);
/* Mark partition as unknown (does not exist) until we see the
* partition in topic metadata. */
if (partition != RD_KAFKA_PARTITION_UA)
rktp->rktp_flags |= RD_KAFKA_TOPPAR_F_UNKNOWN;
rktp->rktp_fetch_state = RD_KAFKA_TOPPAR_FETCH_NONE;
rktp->rktp_fetch_msg_max_bytes =
rkt->rkt_rk->rk_conf.fetch_msg_max_bytes;
rktp->rktp_offset_fp = NULL;
rd_kafka_offset_stats_reset(&rktp->rktp_offsets);
rd_kafka_offset_stats_reset(&rktp->rktp_offsets_fin);
rktp->rktp_ls_offset = RD_KAFKA_OFFSET_INVALID;
rktp->rktp_hi_offset = RD_KAFKA_OFFSET_INVALID;
rktp->rktp_lo_offset = RD_KAFKA_OFFSET_INVALID;
rd_kafka_fetch_pos_init(&rktp->rktp_query_pos);
rd_kafka_fetch_pos_init(&rktp->rktp_next_fetch_start);
rd_kafka_fetch_pos_init(&rktp->rktp_last_next_fetch_start);
rd_kafka_fetch_pos_init(&rktp->rktp_offset_validation_pos);
rd_kafka_fetch_pos_init(&rktp->rktp_app_pos);
rd_kafka_fetch_pos_init(&rktp->rktp_stored_pos);
rd_kafka_fetch_pos_init(&rktp->rktp_committing_pos);
rd_kafka_fetch_pos_init(&rktp->rktp_committed_pos);
rd_kafka_msgq_init(&rktp->rktp_msgq);
rd_kafka_msgq_init(&rktp->rktp_xmit_msgq);
mtx_init(&rktp->rktp_lock, mtx_plain);
rd_refcnt_init(&rktp->rktp_refcnt, 0);
rktp->rktp_fetchq = rd_kafka_consume_q_new(rkt->rkt_rk);
rktp->rktp_ops = rd_kafka_q_new(rkt->rkt_rk);
rktp->rktp_ops->rkq_serve = rd_kafka_toppar_op_serve;
rktp->rktp_ops->rkq_opaque = rktp;
rd_atomic32_init(&rktp->rktp_version, 1);
rktp->rktp_op_version = rd_atomic32_get(&rktp->rktp_version);
rd_atomic32_init(&rktp->rktp_msgs_inflight, 0);
rd_kafka_pid_reset(&rktp->rktp_eos.pid);
/* Consumer: If statistics is available we query the log start offset
* of each partition.
* Since the oldest offset only moves on log retention, we cap this
* value on the low end to a reasonable value to avoid flooding
* the brokers with OffsetRequests when our statistics interval is low.
* FIXME: Use a global timer to collect offsets for all partitions
* FIXME: This timer is superfulous for FETCH >= v5 because the log
* start offset is included in fetch responses.
* */
if (rktp->rktp_rkt->rkt_rk->rk_conf.stats_interval_ms > 0 &&
rkt->rkt_rk->rk_type == RD_KAFKA_CONSUMER &&
rktp->rktp_partition != RD_KAFKA_PARTITION_UA) {
int intvl = rkt->rkt_rk->rk_conf.stats_interval_ms;
if (intvl < 10 * 1000 /* 10s */)
intvl = 10 * 1000;
rd_kafka_timer_start(
&rkt->rkt_rk->rk_timers, &rktp->rktp_consumer_lag_tmr,
intvl * 1000ll, rd_kafka_toppar_consumer_lag_tmr_cb, rktp);
}
rktp->rktp_rkt = rd_kafka_topic_keep(rkt);
rd_kafka_q_fwd_set(rktp->rktp_ops, rkt->rkt_rk->rk_ops);
rd_kafka_dbg(rkt->rkt_rk, TOPIC, "TOPPARNEW",
"NEW %s [%" PRId32 "] %p refcnt %p (at %s:%d)",
rkt->rkt_topic->str, rktp->rktp_partition, rktp,
&rktp->rktp_refcnt, func, line);
return rd_kafka_toppar_keep(rktp);
}
/**
* Removes a toppar from its duties, global lists, etc.
*
* Locks: rd_kafka_toppar_lock() MUST be held
*/
static void rd_kafka_toppar_remove(rd_kafka_toppar_t *rktp) {
rd_kafka_dbg(rktp->rktp_rkt->rkt_rk, TOPIC, "TOPPARREMOVE",
"Removing toppar %s [%" PRId32 "] %p",
rktp->rktp_rkt->rkt_topic->str, rktp->rktp_partition,
rktp);
rd_kafka_timer_stop(&rktp->rktp_rkt->rkt_rk->rk_timers,
&rktp->rktp_validate_tmr, 1 /*lock*/);
rd_kafka_timer_stop(&rktp->rktp_rkt->rkt_rk->rk_timers,
&rktp->rktp_offset_query_tmr, 1 /*lock*/);
rd_kafka_timer_stop(&rktp->rktp_rkt->rkt_rk->rk_timers,
&rktp->rktp_consumer_lag_tmr, 1 /*lock*/);
rd_kafka_q_fwd_set(rktp->rktp_ops, NULL);
}
/**
* Final destructor for partition.
*/
void rd_kafka_toppar_destroy_final(rd_kafka_toppar_t *rktp) {
rd_kafka_toppar_remove(rktp);
rd_kafka_dbg(rktp->rktp_rkt->rkt_rk, TOPIC, "DESTROY",
"%s [%" PRId32 "]: %p DESTROY_FINAL",
rktp->rktp_rkt->rkt_topic->str, rktp->rktp_partition,
rktp);
/* Clear queues */
rd_kafka_assert(rktp->rktp_rkt->rkt_rk,
rd_kafka_msgq_len(&rktp->rktp_xmit_msgq) == 0);
rd_kafka_dr_msgq(rktp->rktp_rkt, &rktp->rktp_msgq,
RD_KAFKA_RESP_ERR__DESTROY);
rd_kafka_q_destroy_owner(rktp->rktp_fetchq);
rd_kafka_q_destroy_owner(rktp->rktp_ops);
rd_kafka_replyq_destroy(&rktp->rktp_replyq);
rd_kafka_topic_destroy0(rktp->rktp_rkt);
mtx_destroy(&rktp->rktp_lock);
if (rktp->rktp_leader)
rd_kafka_broker_destroy(rktp->rktp_leader);
rd_refcnt_destroy(&rktp->rktp_refcnt);
rd_free(rktp->rktp_stored_metadata);
rd_free(rktp);
}
/**
* Set toppar fetching state.
*
* @locality any
* @locks_required rd_kafka_toppar_lock() MUST be held.
*/
void rd_kafka_toppar_set_fetch_state(rd_kafka_toppar_t *rktp, int fetch_state) {
if ((int)rktp->rktp_fetch_state == fetch_state)
return;
rd_kafka_dbg(
rktp->rktp_rkt->rkt_rk, TOPIC, "PARTSTATE",
"Partition %.*s [%" PRId32 "] changed fetch state %s -> %s",
RD_KAFKAP_STR_PR(rktp->rktp_rkt->rkt_topic), rktp->rktp_partition,
rd_kafka_fetch_states[rktp->rktp_fetch_state],
rd_kafka_fetch_states[fetch_state]);
rktp->rktp_fetch_state = fetch_state;
if (fetch_state == RD_KAFKA_TOPPAR_FETCH_ACTIVE)
rd_kafka_dbg(
rktp->rktp_rkt->rkt_rk, CONSUMER | RD_KAFKA_DBG_TOPIC,
"FETCH",
"Partition %.*s [%" PRId32 "] start fetching at %s",
RD_KAFKAP_STR_PR(rktp->rktp_rkt->rkt_topic),
rktp->rktp_partition,
rd_kafka_fetch_pos2str(rktp->rktp_next_fetch_start));
}
/**
* Returns the appropriate toppar for a given rkt and partition.
* The returned toppar has increased refcnt and must be unreffed by calling
* rd_kafka_toppar_destroy().
* May return NULL.
*
* If 'ua_on_miss' is true the UA (unassigned) toppar is returned if
* 'partition' was not known locally, else NULL is returned.
*
* Locks: Caller must hold rd_kafka_topic_*lock()
*/
rd_kafka_toppar_t *rd_kafka_toppar_get0(const char *func,
int line,
const rd_kafka_topic_t *rkt,
int32_t partition,
int ua_on_miss) {
rd_kafka_toppar_t *rktp;
if (partition >= 0 && partition < rkt->rkt_partition_cnt)
rktp = rkt->rkt_p[partition];
else if (partition == RD_KAFKA_PARTITION_UA || ua_on_miss)
rktp = rkt->rkt_ua;
else
return NULL;
if (rktp)
return rd_kafka_toppar_keep_fl(func, line, rktp);
return NULL;
}
/**
* Same as rd_kafka_toppar_get() but no need for locking and
* looks up the topic first.
*
* Locality: any
* Locks: none
*/
rd_kafka_toppar_t *rd_kafka_toppar_get2(rd_kafka_t *rk,
const char *topic,
int32_t partition,
int ua_on_miss,
int create_on_miss) {
rd_kafka_topic_t *rkt;
rd_kafka_toppar_t *rktp;
rd_kafka_wrlock(rk);
/* Find or create topic */
if (unlikely(!(rkt = rd_kafka_topic_find(rk, topic, 0 /*no-lock*/)))) {
if (!create_on_miss) {
rd_kafka_wrunlock(rk);
return NULL;
}
rkt = rd_kafka_topic_new0(rk, topic, NULL, NULL, 0 /*no-lock*/);
if (!rkt) {
rd_kafka_wrunlock(rk);
rd_kafka_log(rk, LOG_ERR, "TOPIC",
"Failed to create local topic \"%s\": %s",
topic, rd_strerror(errno));
return NULL;
}
}
rd_kafka_wrunlock(rk);
rd_kafka_topic_wrlock(rkt);
rktp = rd_kafka_toppar_desired_add(rkt, partition);
rd_kafka_topic_wrunlock(rkt);
rd_kafka_topic_destroy0(rkt);
return rktp;
}
/**
* Returns a toppar if it is available in the cluster.
* '*errp' is set to the error-code if lookup fails.
*
* Locks: topic_*lock() MUST be held
*/
rd_kafka_toppar_t *rd_kafka_toppar_get_avail(const rd_kafka_topic_t *rkt,
int32_t partition,
int ua_on_miss,
rd_kafka_resp_err_t *errp) {
rd_kafka_toppar_t *rktp;
switch (rkt->rkt_state) {
case RD_KAFKA_TOPIC_S_UNKNOWN:
/* No metadata received from cluster yet.
* Put message in UA partition and re-run partitioner when
* cluster comes up. */
partition = RD_KAFKA_PARTITION_UA;
break;
case RD_KAFKA_TOPIC_S_NOTEXISTS:
/* Topic not found in cluster.
* Fail message immediately. */
*errp = RD_KAFKA_RESP_ERR__UNKNOWN_TOPIC;
return NULL;
case RD_KAFKA_TOPIC_S_ERROR:
/* Permanent topic error. */
*errp = rkt->rkt_err;
return NULL;
case RD_KAFKA_TOPIC_S_EXISTS:
/* Topic exists in cluster. */
/* Topic exists but has no partitions.
* This is usually an transient state following the
* auto-creation of a topic. */
if (unlikely(rkt->rkt_partition_cnt == 0)) {
partition = RD_KAFKA_PARTITION_UA;
break;
}
/* Check that partition exists. */
if (partition >= rkt->rkt_partition_cnt) {
*errp = RD_KAFKA_RESP_ERR__UNKNOWN_PARTITION;
return NULL;
}
break;
default:
rd_kafka_assert(rkt->rkt_rk, !*"NOTREACHED");
break;
}
/* Get new partition */
rktp = rd_kafka_toppar_get(rkt, partition, 0);
if (unlikely(!rktp)) {
/* Unknown topic or partition */
if (rkt->rkt_state == RD_KAFKA_TOPIC_S_NOTEXISTS)
*errp = RD_KAFKA_RESP_ERR__UNKNOWN_TOPIC;
else
*errp = RD_KAFKA_RESP_ERR__UNKNOWN_PARTITION;
return NULL;
}
return rktp;
}
/**
* Looks for partition 'i' in topic 'rkt's desired list.
*
* The desired partition list is the list of partitions that are desired
* (e.g., by the consumer) but not yet seen on a broker.
* As soon as the partition is seen on a broker the toppar is moved from
* the desired list and onto the normal rkt_p array.
* When the partition on the broker goes away a desired partition is put
* back on the desired list.
*
* Locks: rd_kafka_topic_*lock() must be held.
* Note: 'rktp' refcount is increased.
*/
rd_kafka_toppar_t *rd_kafka_toppar_desired_get(rd_kafka_topic_t *rkt,
int32_t partition) {
rd_kafka_toppar_t *rktp;
int i;
RD_LIST_FOREACH(rktp, &rkt->rkt_desp, i) {
if (rktp->rktp_partition == partition)
return rd_kafka_toppar_keep(rktp);
}
return NULL;
}
/**
* Link toppar on desired list.
*
* Locks: rd_kafka_topic_wrlock() and toppar_lock() must be held.
*/
void rd_kafka_toppar_desired_link(rd_kafka_toppar_t *rktp) {
if (rktp->rktp_flags & RD_KAFKA_TOPPAR_F_ON_DESP)
return; /* Already linked */
rd_kafka_toppar_keep(rktp);
rd_list_add(&rktp->rktp_rkt->rkt_desp, rktp);
rd_interval_reset(&rktp->rktp_rkt->rkt_desp_refresh_intvl);
rktp->rktp_flags |= RD_KAFKA_TOPPAR_F_ON_DESP;
}
/**
* Unlink toppar from desired list.
*
* Locks: rd_kafka_topic_wrlock() and toppar_lock() must be held.
*/
void rd_kafka_toppar_desired_unlink(rd_kafka_toppar_t *rktp) {
if (!(rktp->rktp_flags & RD_KAFKA_TOPPAR_F_ON_DESP))
return; /* Not linked */
rktp->rktp_flags &= ~RD_KAFKA_TOPPAR_F_ON_DESP;
rd_list_remove(&rktp->rktp_rkt->rkt_desp, rktp);
rd_interval_reset(&rktp->rktp_rkt->rkt_desp_refresh_intvl);
rd_kafka_toppar_destroy(rktp);
}
/**
* @brief If rktp is not already desired:
* - mark as DESIRED|~REMOVE
* - add to desired list if unknown
*
* @remark toppar_lock() MUST be held
*/
void rd_kafka_toppar_desired_add0(rd_kafka_toppar_t *rktp) {
if ((rktp->rktp_flags & RD_KAFKA_TOPPAR_F_DESIRED))
return;
rd_kafka_dbg(rktp->rktp_rkt->rkt_rk, TOPIC, "DESIRED",
"%s [%" PRId32 "]: marking as DESIRED",
rktp->rktp_rkt->rkt_topic->str, rktp->rktp_partition);
/* If toppar was marked for removal this is no longer
* the case since the partition is now desired. */
rktp->rktp_flags &= ~RD_KAFKA_TOPPAR_F_REMOVE;
rktp->rktp_flags |= RD_KAFKA_TOPPAR_F_DESIRED;
if (rktp->rktp_flags & RD_KAFKA_TOPPAR_F_UNKNOWN) {
rd_kafka_dbg(rktp->rktp_rkt->rkt_rk, TOPIC, "DESIRED",
"%s [%" PRId32 "]: adding to DESIRED list",
rktp->rktp_rkt->rkt_topic->str,
rktp->rktp_partition);
rd_kafka_toppar_desired_link(rktp);
}
}
/**
* Adds 'partition' as a desired partition to topic 'rkt', or updates
* an existing partition to be desired.
*
* Locks: rd_kafka_topic_wrlock() must be held.
*/
rd_kafka_toppar_t *rd_kafka_toppar_desired_add(rd_kafka_topic_t *rkt,
int32_t partition) {
rd_kafka_toppar_t *rktp;
rktp = rd_kafka_toppar_get(rkt, partition, 0 /*no_ua_on_miss*/);
if (!rktp)
rktp = rd_kafka_toppar_desired_get(rkt, partition);
if (!rktp)
rktp = rd_kafka_toppar_new(rkt, partition);
rd_kafka_toppar_lock(rktp);
rd_kafka_toppar_desired_add0(rktp);
rd_kafka_toppar_unlock(rktp);
return rktp; /* Callers refcount */
}
/**
* Unmarks an 'rktp' as desired.
*
* Locks: rd_kafka_topic_wrlock() and rd_kafka_toppar_lock() MUST be held.
*/
void rd_kafka_toppar_desired_del(rd_kafka_toppar_t *rktp) {
if (!(rktp->rktp_flags & RD_KAFKA_TOPPAR_F_DESIRED))
return;
rktp->rktp_flags &= ~RD_KAFKA_TOPPAR_F_DESIRED;
rd_kafka_toppar_desired_unlink(rktp);
rd_kafka_dbg(rktp->rktp_rkt->rkt_rk, TOPIC, "DESP",
"Removing (un)desired topic %s [%" PRId32 "]",
rktp->rktp_rkt->rkt_topic->str, rktp->rktp_partition);
if (rktp->rktp_flags & RD_KAFKA_TOPPAR_F_UNKNOWN) {
/* If this partition does not exist in the cluster
* and is no longer desired, remove it. */
rd_kafka_toppar_broker_leave_for_remove(rktp);
}
}
/**
* Append message at tail of 'rktp' message queue.
*/
void rd_kafka_toppar_enq_msg(rd_kafka_toppar_t *rktp,
rd_kafka_msg_t *rkm,
rd_ts_t now) {
rd_kafka_q_t *wakeup_q = NULL;
rd_kafka_toppar_lock(rktp);
if (!rkm->rkm_u.producer.msgid &&
rktp->rktp_partition != RD_KAFKA_PARTITION_UA)
rkm->rkm_u.producer.msgid = ++rktp->rktp_msgid;
if (rktp->rktp_partition == RD_KAFKA_PARTITION_UA ||
rktp->rktp_rkt->rkt_conf.queuing_strategy == RD_KAFKA_QUEUE_FIFO) {
/* No need for enq_sorted(), this is the oldest message. */
rd_kafka_msgq_enq(&rktp->rktp_msgq, rkm);
} else {
rd_kafka_msgq_enq_sorted(rktp->rktp_rkt, &rktp->rktp_msgq, rkm);
}
if (unlikely(rktp->rktp_partition != RD_KAFKA_PARTITION_UA &&
rd_kafka_msgq_may_wakeup(&rktp->rktp_msgq, now) &&
(wakeup_q = rktp->rktp_msgq_wakeup_q))) {
/* Wake-up broker thread */
rktp->rktp_msgq.rkmq_wakeup.signalled = rd_true;
rd_kafka_q_keep(wakeup_q);
}
rd_kafka_toppar_unlock(rktp);
if (unlikely(wakeup_q != NULL)) {
rd_kafka_q_yield(wakeup_q);
rd_kafka_q_destroy(wakeup_q);
}
}
/**
* @brief Insert \p srcq before \p insert_before in \p destq.
*
* If \p srcq and \p destq overlaps only part of the \p srcq will be inserted.
*
* Upon return \p srcq will contain any remaining messages that require
* another insert position in \p destq.
*/
static void rd_kafka_msgq_insert_msgq_before(rd_kafka_msgq_t *destq,
rd_kafka_msg_t *insert_before,
rd_kafka_msgq_t *srcq,
int (*cmp)(const void *a,
const void *b)) {
rd_kafka_msg_t *slast;
rd_kafka_msgq_t tmpq;
if (!insert_before) {
/* Append all of srcq to destq */
rd_kafka_msgq_concat(destq, srcq);
rd_kafka_msgq_verify_order(NULL, destq, 0, rd_false);
return;
}
slast = rd_kafka_msgq_last(srcq);
rd_dassert(slast);
if (cmp(slast, insert_before) > 0) {
rd_kafka_msg_t *new_sfirst;
int cnt;
int64_t bytes;
/* destq insert_before resides somewhere between
* srcq.first and srcq.last, find the first message in
* srcq that is > insert_before and split srcq into
* a left part that contains the messages to insert before
* insert_before, and a right part that will need another
* insert position. */
new_sfirst = rd_kafka_msgq_find_pos(srcq, NULL, insert_before,
cmp, &cnt, &bytes);
rd_assert(new_sfirst);
/* split srcq into two parts using the divider message */
rd_kafka_msgq_split(srcq, &tmpq, new_sfirst, cnt, bytes);
rd_kafka_msgq_verify_order(NULL, srcq, 0, rd_false);
rd_kafka_msgq_verify_order(NULL, &tmpq, 0, rd_false);
} else {
rd_kafka_msgq_init(&tmpq);
}
/* srcq now contains messages up to the first message in destq,
* insert srcq at insert_before in destq. */
rd_dassert(!TAILQ_EMPTY(&destq->rkmq_msgs));
rd_dassert(!TAILQ_EMPTY(&srcq->rkmq_msgs));
TAILQ_INSERT_LIST_BEFORE(&destq->rkmq_msgs, insert_before,
&srcq->rkmq_msgs, rd_kafka_msgs_head_s,
rd_kafka_msg_t *, rkm_link);
destq->rkmq_msg_cnt += srcq->rkmq_msg_cnt;
destq->rkmq_msg_bytes += srcq->rkmq_msg_bytes;
srcq->rkmq_msg_cnt = 0;
srcq->rkmq_msg_bytes = 0;
rd_kafka_msgq_verify_order(NULL, destq, 0, rd_false);
rd_kafka_msgq_verify_order(NULL, srcq, 0, rd_false);
/* tmpq contains the remaining messages in srcq, move it over. */
rd_kafka_msgq_move(srcq, &tmpq);
rd_kafka_msgq_verify_order(NULL, srcq, 0, rd_false);
}
/**
* @brief Insert all messages from \p srcq into \p destq in their sorted
* position (using \p cmp)
*/
void rd_kafka_msgq_insert_msgq(rd_kafka_msgq_t *destq,
rd_kafka_msgq_t *srcq,
int (*cmp)(const void *a, const void *b)) {
rd_kafka_msg_t *sfirst, *dlast, *start_pos = NULL;
if (unlikely(RD_KAFKA_MSGQ_EMPTY(srcq))) {
/* srcq is empty */
return;
}
if (unlikely(RD_KAFKA_MSGQ_EMPTY(destq))) {
/* destq is empty, simply move the srcq. */
rd_kafka_msgq_move(destq, srcq);
rd_kafka_msgq_verify_order(NULL, destq, 0, rd_false);
return;
}
/* Optimize insertion by bulk-moving messages in place.
* We know that:
* - destq is sorted but might not be continous (1,2,3,7)
* - srcq is sorted but might not be continous (4,5,6,8)
* - there migt be (multiple) overlaps between the two, e.g:
* destq = (1,2,3,7), srcq = (4,5,6,8)
* - there may be millions of messages.
*/
rd_kafka_msgq_verify_order(NULL, destq, 0, rd_false);
rd_kafka_msgq_verify_order(NULL, srcq, 0, rd_false);
dlast = rd_kafka_msgq_last(destq);
sfirst = rd_kafka_msgq_first(srcq);
/* Most common case, all of srcq goes after destq */
if (likely(cmp(dlast, sfirst) < 0)) {
rd_kafka_msgq_concat(destq, srcq);
rd_kafka_msgq_verify_order(NULL, destq, 0, rd_false);
rd_assert(RD_KAFKA_MSGQ_EMPTY(srcq));
return;
}
/* Insert messages from srcq into destq in non-overlapping
* chunks until srcq is exhausted. */
while (likely(sfirst != NULL)) {
rd_kafka_msg_t *insert_before;
/* Get insert position in destq of first element in srcq */
insert_before = rd_kafka_msgq_find_pos(destq, start_pos, sfirst,
cmp, NULL, NULL);
/* Insert as much of srcq as possible at insert_before */
rd_kafka_msgq_insert_msgq_before(destq, insert_before, srcq,
cmp);
/* Remember the current destq position so the next find_pos()
* does not have to re-scan destq and what was
* added from srcq. */
start_pos = insert_before;
/* For next iteration */
sfirst = rd_kafka_msgq_first(srcq);
rd_kafka_msgq_verify_order(NULL, destq, 0, rd_false);
rd_kafka_msgq_verify_order(NULL, srcq, 0, rd_false);
}
rd_kafka_msgq_verify_order(NULL, destq, 0, rd_false);
rd_assert(RD_KAFKA_MSGQ_EMPTY(srcq));
}
/**
* @brief Inserts messages from \p srcq according to their sorted position
* into \p destq, filtering out messages that can not be retried.
*
* @param incr_retry Increment retry count for messages.
* @param max_retries Maximum retries allowed per message.
* @param backoff Absolute retry backoff for retried messages.
* @param exponential_backoff If true the backoff should be exponential with
* 2**(retry_count - 1)*retry_ms with jitter. The
* \p backoff is ignored.
* @param retry_ms The retry ms used for exponential backoff calculation
* @param retry_max_ms The max backoff limit for exponential backoff calculation
*
* @returns 0 if all messages were retried, or 1 if some messages
* could not be retried.
*/
int rd_kafka_retry_msgq(rd_kafka_msgq_t *destq,
rd_kafka_msgq_t *srcq,
int incr_retry,
int max_retries,
rd_ts_t backoff,
rd_kafka_msg_status_t status,
int (*cmp)(const void *a, const void *b),
rd_bool_t exponential_backoff,
int retry_ms,
int retry_max_ms) {
rd_kafka_msgq_t retryable = RD_KAFKA_MSGQ_INITIALIZER(retryable);
rd_kafka_msg_t *rkm, *tmp;
rd_ts_t now;
int64_t jitter = rd_jitter(100 - RD_KAFKA_RETRY_JITTER_PERCENT,
100 + RD_KAFKA_RETRY_JITTER_PERCENT);
/* Scan through messages to see which ones are eligible for retry,
* move the retryable ones to temporary queue and
* set backoff time for first message and optionally
* increase retry count for each message.
* Sorted insert is not necessary since the original order
* srcq order is maintained.
*
* Start timestamp for calculating backoff is common,
* to avoid that messages from the same batch
* have different backoff, as they need to be retried
* by reconstructing the same batch, when idempotency is
* enabled. */
now = rd_clock();
TAILQ_FOREACH_SAFE(rkm, &srcq->rkmq_msgs, rkm_link, tmp) {
if (rkm->rkm_u.producer.retries + incr_retry > max_retries)
continue;
rd_kafka_msgq_deq(srcq, rkm, 1);
rd_kafka_msgq_enq(&retryable, rkm);
rkm->rkm_u.producer.retries += incr_retry;
if (exponential_backoff) {
/* In some cases, like failed Produce requests do not
* increment the retry count, see
* rd_kafka_handle_Produce_error. */
if (rkm->rkm_u.producer.retries > 0)
backoff =
(1 << (rkm->rkm_u.producer.retries - 1)) *
retry_ms;
else
backoff = retry_ms;
/* Multiplied by 10 as backoff should be in nano
* seconds. */
backoff = jitter * backoff * 10;
if (backoff > retry_max_ms * 1000)
backoff = retry_max_ms * 1000;
backoff = now + backoff;
}
rkm->rkm_u.producer.ts_backoff = backoff;
/* Don't downgrade a message from any form of PERSISTED
* to NOT_PERSISTED, since the original cause of indicating
* PERSISTED can't be changed.
* E.g., a previous ack or in-flight timeout. */
if (likely(!(status == RD_KAFKA_MSG_STATUS_NOT_PERSISTED &&
rkm->rkm_status !=
RD_KAFKA_MSG_STATUS_NOT_PERSISTED)))
rkm->rkm_status = status;
}
/* No messages are retryable */
if (RD_KAFKA_MSGQ_EMPTY(&retryable))
return 0;
/* Insert retryable list at sorted position */
rd_kafka_msgq_insert_msgq(destq, &retryable, cmp);
return 1;
}
/**
* @brief Inserts messages from \p rkmq according to their sorted position
* into the partition's message queue.
*
* @param incr_retry Increment retry count for messages.
* @param status Set status on each message.
*
* @returns 0 if all messages were retried, or 1 if some messages
* could not be retried.
*
* @locality Broker thread (but not necessarily the leader broker thread)
*/
int rd_kafka_toppar_retry_msgq(rd_kafka_toppar_t *rktp,
rd_kafka_msgq_t *rkmq,
int incr_retry,
rd_kafka_msg_status_t status) {
rd_kafka_t *rk = rktp->rktp_rkt->rkt_rk;
int retry_ms = rk->rk_conf.retry_backoff_ms;
int retry_max_ms = rk->rk_conf.retry_backoff_max_ms;
int r;
if (rd_kafka_terminating(rk))
return 1;
rd_kafka_toppar_lock(rktp);
/* Exponential backoff applied. */
r = rd_kafka_retry_msgq(&rktp->rktp_msgq, rkmq, incr_retry,
rk->rk_conf.max_retries,
0 /* backoff will be calculated */, status,
rktp->rktp_rkt->rkt_conf.msg_order_cmp, rd_true,
retry_ms, retry_max_ms);
rd_kafka_toppar_unlock(rktp);
return r;
}
/**
* @brief Insert sorted message list \p rkmq at sorted position in \p rktp 's