@@ -90,6 +90,8 @@ struct rd_kafka_q_io {
90
90
int fd ;
91
91
void * payload ;
92
92
size_t size ;
93
+ rd_ts_t ts_rate ; /**< How often the IO wakeup may be performed (us) */
94
+ rd_ts_t ts_last ; /**< Last IO wakeup */
93
95
/* For callback-based signalling */
94
96
void (* event_cb ) (rd_kafka_t * rk , void * opaque );
95
97
void * event_cb_opaque ;
@@ -284,10 +286,12 @@ static RD_INLINE RD_UNUSED int rd_kafka_q_is_fwded (rd_kafka_q_t *rkq) {
284
286
/**
285
287
* @brief Trigger an IO event for this queue.
286
288
*
289
+ * @param rate_limit if true, rate limit IO-based wakeups.
290
+ *
287
291
* @remark Queue MUST be locked
288
292
*/
289
293
static RD_INLINE RD_UNUSED
290
- void rd_kafka_q_io_event (rd_kafka_q_t * rkq ) {
294
+ void rd_kafka_q_io_event (rd_kafka_q_t * rkq , rd_bool_t rate_limit ) {
291
295
292
296
if (likely (!rkq -> rkq_qio ))
293
297
return ;
@@ -297,6 +301,15 @@ void rd_kafka_q_io_event (rd_kafka_q_t *rkq) {
297
301
return ;
298
302
}
299
303
304
+
305
+ if (rate_limit ) {
306
+ rd_ts_t now = rd_clock ();
307
+ if (likely (rkq -> rkq_qio -> ts_last + rkq -> rkq_qio -> ts_rate > now ))
308
+ return ;
309
+
310
+ rkq -> rkq_qio -> ts_last = now ;
311
+ }
312
+
300
313
/* Ignore errors, not much to do anyway. */
301
314
if (rd_write (rkq -> rkq_qio -> fd , rkq -> rkq_qio -> payload ,
302
315
(int )rkq -> rkq_qio -> size ) == -1 )
@@ -320,7 +333,7 @@ int rd_kafka_op_cmp_prio (const void *_a, const void *_b) {
320
333
* @brief Wake up waiters without enqueuing an op.
321
334
*/
322
335
static RD_INLINE RD_UNUSED void
323
- rd_kafka_q_yield (rd_kafka_q_t * rkq ) {
336
+ rd_kafka_q_yield (rd_kafka_q_t * rkq , rd_bool_t rate_limit ) {
324
337
rd_kafka_q_t * fwdq ;
325
338
326
339
mtx_lock (& rkq -> rkq_lock );
@@ -337,12 +350,12 @@ rd_kafka_q_yield (rd_kafka_q_t *rkq) {
337
350
rkq -> rkq_flags |= RD_KAFKA_Q_F_YIELD ;
338
351
cnd_signal (& rkq -> rkq_cond );
339
352
if (rkq -> rkq_qlen == 0 )
340
- rd_kafka_q_io_event (rkq );
353
+ rd_kafka_q_io_event (rkq , rate_limit );
341
354
342
355
mtx_unlock (& rkq -> rkq_lock );
343
356
} else {
344
357
mtx_unlock (& rkq -> rkq_lock );
345
- rd_kafka_q_yield (fwdq );
358
+ rd_kafka_q_yield (fwdq , rate_limit );
346
359
rd_kafka_q_destroy (fwdq );
347
360
}
348
361
@@ -413,7 +426,7 @@ int rd_kafka_q_enq1 (rd_kafka_q_t *rkq, rd_kafka_op_t *rko,
413
426
rd_kafka_q_enq0 (rkq , rko , at_head );
414
427
cnd_signal (& rkq -> rkq_cond );
415
428
if (rkq -> rkq_qlen == 1 )
416
- rd_kafka_q_io_event (rkq );
429
+ rd_kafka_q_io_event (rkq , rd_false /*no rate-limiting*/ );
417
430
418
431
if (do_lock )
419
432
mtx_unlock (& rkq -> rkq_lock );
@@ -518,7 +531,7 @@ int rd_kafka_q_concat0 (rd_kafka_q_t *rkq, rd_kafka_q_t *srcq, int do_lock) {
518
531
519
532
TAILQ_CONCAT (& rkq -> rkq_q , & srcq -> rkq_q , rko_link );
520
533
if (rkq -> rkq_qlen == 0 )
521
- rd_kafka_q_io_event (rkq );
534
+ rd_kafka_q_io_event (rkq , rd_false /*no rate-limiting*/ );
522
535
rkq -> rkq_qlen += srcq -> rkq_qlen ;
523
536
rkq -> rkq_qsize += srcq -> rkq_qsize ;
524
537
cnd_signal (& rkq -> rkq_cond );
@@ -559,7 +572,7 @@ void rd_kafka_q_prepend0 (rd_kafka_q_t *rkq, rd_kafka_q_t *srcq,
559
572
/* Move srcq to rkq */
560
573
TAILQ_MOVE (& rkq -> rkq_q , & srcq -> rkq_q , rko_link );
561
574
if (rkq -> rkq_qlen == 0 && srcq -> rkq_qlen > 0 )
562
- rd_kafka_q_io_event (rkq );
575
+ rd_kafka_q_io_event (rkq , rd_false /*no rate-limiting*/ );
563
576
rkq -> rkq_qlen += srcq -> rkq_qlen ;
564
577
rkq -> rkq_qsize += srcq -> rkq_qsize ;
565
578
0 commit comments