# Why your Kafka Streams suppress() emits nothing

*Debug a suppress() that never fires.*

You wired up a windowed aggregation, added `suppress(Suppressed.untilWindowCloses(...))` to get one clean result per window instead of the noisy intermediate stream, deployed it, and the output topic is empty. No errors. No warnings. The aggregation is clearly running; the downstream topic just never receives a thing. In a test it might work; in production, or against a low-traffic partition, it goes silent.

This is the single most-reported `suppress` problem, and it is almost never a bug in Kafka Streams. It is a direct consequence of how `untilWindowCloses` decides a window is finished. Once you internalize the one rule below, every variant of "suppress isn't emitting" becomes a checklist you can walk in a few minutes.

**What you'll learn:**
- The one rule that governs when `untilWindowCloses` fires, and why an idle partition breaks it
- A root-cause checklist: idle streams, one lagging partition, test harnesses, buffer config
- The dummy-future-event workaround, why it works, and why it's a smell
- When to abandon `suppress` for a wall-clock punctuator instead

## The one rule: suppress fires on stream-time, not the clock

`suppress(Suppressed.untilWindowCloses(...))` buffers every windowed result and releases it at exactly one moment: when the window *closes*. A window closes when **stream-time** passes the window's end plus its grace period. That is the whole mechanism, and every failure below is a way of stream-time not getting there.

Stream-time is not your wall clock. It is Kafka Streams' internal sense of how far through the data a task has progressed, defined as the **maximum record timestamp the task has seen so far**. It is monotonic, it only moves forward, and, the part that bites, **it only advances when a new record arrives** carrying a timestamp later than the current maximum. No new record, no movement. The concept is covered in full in [windowing](https://www.conduktor.io/kafka-streams/windowing); here we use it as the lens for debugging.

So for a window covering 10:00–11:00 with 5 minutes of grace, `suppress` releases that window's result the instant a record flows through the task with a timestamp ≥ 11:05. Not when your server clock reaches 11:05, when a *record stamped* 11:05-or-later is actually processed by that task.

```java
events
    .groupByKey()
    .windowedBy(TimeWindows.ofSizeAndGrace(Duration.ofHours(1), Duration.ofMinutes(5)))
    .count()
    .suppress(Suppressed.untilWindowCloses(Suppressed.BufferConfig.unbounded()))
    .toStream()
    .to("hourly-final");   // emits ONLY when stream-time crosses window-end + grace
```

Read that last comment as a warning, not a guarantee. If stream-time never crosses 11:05, `hourly-final` stays empty, forever, no matter how much real time passes.

## The root-cause checklist

### 1. Stream-time isn't advancing (the idle / low-traffic / sparse-key partition)

This is the cause in the large majority of cases. The window for 10:00–11:00 can only close when a record with timestamp ≥ 11:05 arrives **on the same partition**. If the last event of the hour landed at 10:58 and the stream then went quiet, stream-time is frozen at 10:58. The window stays open. `suppress` keeps buffering and emits nothing.

This bites hardest on:

- **Low-traffic topics**, a few records an hour, with long gaps. The window that should have closed at 11:05 waits for the next record, which might be at 11:40, or tomorrow morning, or never.
- **Sparse keys on quiet partitions**, the windowed *result* is per key, but the close decision is not: any record on the partition that pushes stream-time past window-end + grace closes *every* key's buffered window there. So a rarely-active key (a test tenant, a dormant account) is fine on a busy partition; other keys' traffic closes its windows too. It only goes silent when it sits on a partition where *all* keys go quiet, or alone on a low-traffic partition.
- **End-of-day / weekend tails**, the last window before traffic dies is the one that never fires. You see results all day, then the final one is missing.

The fix is conceptual first: there must be a *newer record, with a later timestamp, on the same partition* to push stream-time past `window-end + grace`. If your traffic naturally provides that, the result fires on its own, just later than wall-clock intuition suggests. If it doesn't, you've discovered that `suppress(untilWindowCloses)` is the wrong tool for a sparse stream, and you want the punctuator approach in the last section.

> 🚫 *"suppress() is broken, it buffers forever and never emits."*
> It is doing exactly what `untilWindowCloses` specifies: hold the result until stream-time crosses window-end + grace. On an idle or sparse partition, stream-time never gets there, so the window never closes. The behavior is correct; the stream is quiet.

### 2. One lagging partition holds back the whole task

Stream-time is tracked **per task**. With a single input topic, each task owns exactly one partition of it, so a lagging partition stalls only the keys living on that partition: still "some keys emit, some don't", just clustered by partition. Tasks that own *multiple* partitions arise when a subtopology reads several source topics (joins, `merge()`). There, Kafka Streams chooses the *lowest-timestamp* buffered record to process next, to stay time-ordered, so a partition that is genuinely behind (heavy lag, a slow upstream producer, an offset reset on one partition only) anchors the task's notion of time and holds back windows fed by the faster partitions. Two limits on that holdback: it only applies when the lagging partition has data sitting on the broker. An idle, *empty* partition does not block under the default `max.task.idle.ms=0`; the task processes what it has. And the anchor is transient: the moment one fresh record from the straggler is processed, stream-time jumps to the maximum seen and every closeable window flushes at once.

How to spot it: the symptom is partial, not total. Some keys emit, some don't, and the ones that don't tend to cluster. Check per-partition consumer lag on the input topic, an uneven lag profile, where one or two partitions trail the rest, is the tell. Even out the lag (fix the slow producer, let the lagging partition catch up, or address the offset reset) and the held-back windows close.

### 3. Your test passes but production doesn't (TopologyTestDriver)

A `suppress` test that "works" while production stays silent usually means the test accidentally advances stream-time and production doesn't. With `TopologyTestDriver`, *you* control timestamps explicitly. If your test pipes a record with a later timestamp after the window's data, even incidentally, it pushes stream-time past the window end and the suppressed result pops out, leaving you convinced the topology is correct.

To test `suppress` honestly, pipe one final record with a timestamp past `window-end + grace` to close the window, then assert on the output:

```java
TestInputTopic<String, String> in =
    driver.createInputTopic("events", new StringSerializer(), new StringSerializer());

// Records inside the 10:00–11:00 window
in.pipeInput("k", "a", Instant.parse("2026-06-08T10:30:00Z"));
in.pipeInput("k", "b", Instant.parse("2026-06-08T10:58:00Z"));

// Nothing emitted yet, the window is still open.
assertTrue(out.isEmpty());

// One record past window-end (11:00) + grace (5m) → stream-time crosses 11:05 → window closes
in.pipeInput("k", "c", Instant.parse("2026-06-08T11:06:00Z"));
assertFalse(out.isEmpty());   // NOW the suppressed result for 10:00–11:00 fires
```

One detail that surprises when correlating output: the emitted result carries the timestamp of the *last record that entered the window* (10:58 here), not the window end and not the 11:06 record that closed it.

The lesson transfers straight to production: the thing that closes a window in a test is the same thing that closes it live, a later-timestamped record on that partition. `TopologyTestDriver` is also synchronous and applies no record cache, so it differs from production in other ways, a test that emits on every record won't reproduce production's cached, less-frequent emissions.

### 4. BufferConfig: unbounded vs bounded, and what happens when it fills

`suppress` holds un-emitted results in an in-memory buffer until the window closes. That buffer is in heap, but it's backed by its own changelog topic (enabled by default, named `<application.id>-KTABLE-SUPPRESS-STATE-STORE-<n>-changelog`, right next to the aggregation's `KSTREAM-AGGREGATE-STATE-STORE` changelog in a topic list), so a restart or rebalance rebuilds it from the changelog rather than dropping un-emitted results, at the cost of making the buffer part of [state restore](https://www.conduktor.io/kafka-streams/state-restore). A disk-spilling buffer has been proposed but never implemented, so a very large suppression buffer lives in memory. How you size that buffer changes the failure mode, but note up front: **a too-small buffer is a different problem from stream-time not advancing.** If your output is empty because the stream is idle (cases 1–3), no `BufferConfig` change will help; the window genuinely hasn't closed.

`BufferConfig` matters when the *volume of open windows* is large, many keys, wide windows, long grace, so the buffer itself becomes the constraint:

| BufferConfig | Behavior when full | Risk |
|---|---|---|
| `unbounded()` | Never "full", grows with open windows | Heap pressure / OOM if open-window count explodes |
| `maxBytes(...)` / `maxRecords(...)` + `shutDownWhenFull()` | App **shuts down** when the cap is hit | Crash you'll see immediately, but it's a hard stop |
| `maxBytes(...)` / `maxRecords(...)` + `emitEarlyWhenFull()` | Emits the *oldest* buffered records early to stay under the cap | **Breaks the "one final result per window" guarantee**, and only compiles with `untilTimeLimit`, not `untilWindowCloses` |

```java
// "Strict" final-result semantics, bounded memory, fail loudly if it can't keep up:
.suppress(Suppressed.untilWindowCloses(
    Suppressed.BufferConfig.maxBytes(64 * 1024 * 1024L).shutDownWhenFull()))
```

The trap here is reaching for `emitEarlyWhenFull()` to "fix" missing output. The type system blocks the worst version of this: `untilWindowCloses` accepts only a `StrictBufferConfig`, and `emitEarlyWhenFull()` returns an `EagerBufferConfig`, so the combination is a compile error. Using it means switching to `untilTimeLimit`, abandoning final-result semantics at the API level. It doesn't fix the idle-stream case at all, and where it *does* emit, it gives you partial windows, which is precisely what `untilWindowCloses` was supposed to prevent. Use `emitEarlyWhenFull()` only when you have explicitly accepted early emissions as a memory-safety valve, not as a remedy for silence.

## The dummy future-event workaround, and why it's a smell

The folklore fix for the idle-partition case is to periodically inject a synthetic record with a *future* timestamp onto the input, on every partition, purely to drag stream-time forward and force open windows to close.

It works, and the reason it works is now obvious: a record with a timestamp past `window-end + grace` is *exactly* what `suppress` is waiting for. Push stream-time forward artificially and the buffered results flush.

But it's a smell, and worth naming as one:

- The synthetic records are fake data flowing through your real topology. Your aggregation logic must recognize and skip them, or they pollute results.
- You have to emit them on **every partition** (stream-time is per task), which means knowing the partition layout and producing accordingly.
- The future timestamp can prematurely *expire* genuinely late records that would otherwise have landed within grace, you've moved the late-data cutoff forward by hand.
- It's an out-of-band heartbeat bolted onto a data path that was never meant to carry one.

If you find yourself building a heartbeat-injector, that's a signal that `suppress(untilWindowCloses)` doesn't fit a stream this sparse. The cleaner answer is below.

## When to use a wall-clock punctuator instead

`suppress(untilWindowCloses)` ties emission to *data-driven* stream-time. When your stream is too sparse for that to be reliable, stop fighting it and drive emission from the *wall clock* instead, using the Processor API.

A `WALL_CLOCK_TIME` punctuator fires on a fixed real-time schedule regardless of whether records are arriving, so it does not depend on stream-time advancing. You keep your windowed (or plain) aggregate in a state store, and on each punctuator tick you scan the store and emit any windows whose end-plus-grace is now in the past according to the wall clock. This trades a guarantee for liveness: you accept that "closed" is now decided by real time (and thus is approximate with respect to event-time) in exchange for results that actually come out on a quiet stream.

```java
// In a Processor: emit finished windows on a real-time tick, independent of stream-time
context.schedule(Duration.ofMinutes(1), PunctuationType.WALL_CLOCK_TIME, timestamp -> {
    // iterate the windowed store, forward + delete windows whose end+grace < now
});
```

This is a deliberate design choice, not a hack: it's the supported escape hatch for low-traffic windows. Note the contrast with `STREAM_TIME` punctuators, which fire on stream-time advancing and therefore suffer the *same* idle-partition silence as `suppress`. The full mechanics, `schedule`, `Punctuator`, store iteration, cancelling in `close()`, are in [the Processor API](https://www.conduktor.io/kafka-streams/processor-api).

**Why is my Kafka Streams suppress() not emitting any records?**

`suppress(untilWindowCloses)` releases a window only when stream-time crosses window-end plus grace, and stream-time advances only when a new record arrives with a later timestamp on the same partition. On an idle or low-traffic partition stream-time freezes, so the window never closes and the buffer never flushes, the behavior is correct, the stream is just quiet.

**Why does suppress only emit when new records arrive?**

Because it fires on stream-time, not your wall clock. Stream-time is the maximum record timestamp the task has seen, so a window covering 10:00–11:00 with 5m grace releases only when a record stamped ≥ 11:05 is actually processed, not when your server clock reaches 11:05.

**Why does my suppress test pass but production emits nothing?**

Your test almost certainly advances stream-time and production doesn't. With `TopologyTestDriver` you set timestamps explicitly, so piping a later-timestamped record after the window's data closes it. To test honestly, pipe one final record past `window-end + grace` and assert the output then appears.

**Will a bigger BufferConfig fix my missing suppress output?**

No, if the cause is an idle stream. `BufferConfig` only matters when the volume of open windows is large; it has no effect when the window simply hasn't closed because stream-time isn't advancing. And `emitEarlyWhenFull()` won't even compile with `untilWindowCloses`, it forces you onto `untilTimeLimit`, which gives partial windows, the exact thing final-result semantics were meant to prevent.

**How do I emit window results on a low-traffic stream?**

Stop using `suppress(untilWindowCloses)` and drive emission from the wall clock with a `WALL_CLOCK_TIME` punctuator in the Processor API. It fires on a fixed real-time schedule regardless of whether records arrive: keep the aggregate in a state store and on each tick scan it and emit windows whose end-plus-grace is now past. (A `STREAM_TIME` punctuator won't help, it suffers the same idle silence.)

> **See it in practice with Conduktor**
> When a `suppress` emits nothing, the first question is whether the window simply hasn't closed yet versus a real backlog. [Conduktor Console](https://docs.conduktor.io/guide/manage-kafka/kafka-resources/topics) lets you watch per-partition consumer group lag on the input topic, to spot the one lagging partition holding a task back, and inspect the windowed aggregation's changelog topic to confirm state is being written even while the output stays quiet.

## Next steps

- [Windowing](https://www.conduktor.io/kafka-streams/windowing), stream-time, grace, and why windows close when they do
- [Aggregations](https://www.conduktor.io/kafka-streams/aggregations), the continuous-update model that `suppress` collapses into one result
- [The Processor API](https://www.conduktor.io/kafka-streams/processor-api), wall-clock punctuators for emitting on sparse streams
