< Blog

Designing Clear Street's First Transactional Outbox Pattern

October 1, 2020
Sachin Kumar
Co-Founder & Chief Technology Officer
Engineering

In this post, we discuss how Clear Street designed its first version of a transactional outbox pattern on top of change data capture with Debezium. In subsequent posts, we’ll discuss problems we encountered with our initial design and how we are thinking about changing it over time. If you’re interested in distributed system patterns, such as transactional outboxes, you might find the journey we had interesting.

Also, it’s our first engineering blog post! 🎉

Back in 2018 when we started Clear Street, we knew that we wanted our microservices to expose two types of APIs, one RESTful gRPC API for CRUD-related operations, and the other, a streaming API with Kafka that captures those CRUD operations. The streaming API is important because it allows downstream services to avoid polling for updates, and instead be simply notified via Kafka with the details of the updates.

Side Effects

Implementing a streaming API has a vexing challenge, specifically when your CRUD operations are done within a PostgreSQL transaction. The question becomes: How do you guarantee that what gets published to Kafka won’t be inconsistent with the database in the event of failures?

This is a classic case of dealing with a side effect as part of a database transaction. In this case, the side effect is publishing to Kafka. Using a two-phase commit (2PC), in principle, this would solve this problem, but our view is that 2PC is known to be slow and, frankly, complicated to implement.

Listen for Changes at the DB

We opted to use Debezium to solve the problem of publishing events onto Kafka as a side effect of a database transaction. Debezium is a Kafka Connector that listens for changes in the database via PostgreSQL’s write-ahead-log, and then publishes those changes onto Kafka, without having to worry about failure scenarios. In other words, if your database transaction successfully commits to the database, you are guaranteed that a corresponding event with the details of what you changed are published to Kafka at-least once.

Using Debezium v0.9.5, here’s an example of what gets published automatically onto Kafka after inserting a row into a table called events that has the columns id, created_at, column_a, column_b in schema bar , in database foo, in PostgresSQL:

Debezium v0.9.5 Kafka output (truncated for brevity)

This message, assuming default Debezium configuration, will get published on Kafka topic foo.bar.events on partition 0. Note that the payload object has before and after which reflect the state of the row before the change and after, respectively. In this case, since we inserted a new row, there is no prior state.

Debezium is smart enough to understand your PostgresSQL schema so that changes that occur on a particular row can be converted into a JSON object. Therefore, your database models are automatically converted into JSON and published directly onto Kafka without any additional work on your part.

Avoiding Leaky Abstractions

While we liked the ability for Debezium to solve our side effect problem, we didn’t like how a service’s internal data models were exposed directly onto Kafka. This presented a glaring hole in our design philosophy that a service’s persistent storage is private to that service and should not be leaked. Leaking this information breaks encapsulation, and tightly couples our database models to our streaming API.

We needed a layer of abstraction on the raw message Debezium actually publishes to Kafka. We decided that a service author should be more explicit in controlling when a message is published and what specific fields are in that message.

To do this, we implemented a transactional outbox pattern as follows:

  • We configured Debezium to only listen for changes on PostgresSQL tables that have the suffix events . We did this via Debezium’s table.whitelist configuration.
  • We decided that any event published from Debezium to Kafka must be encoded as an Avro message. Avro is a first-class citizen in the Kafka ecosystem, so all our Kafka messages are encoded in Avro.
  • Since we decided all our events should be Avro messages, the service author needs to construct an Avro message with the appropriate information that captures the change being made. The service author then commits the resulting message into the events table, along with the schema-id that is part of the Confluent’s Schema-Registry. This means any table suffixed with events , by convention, follows this schema:


PostgreSQL schema definition

The important fields in the schema above are schema_id , partition , and data . The data is the binary Avro encoding of the message, while schema_id is Confluent’s schema-registry’s unique ID that gives you the schema of how data is encoded. Finally, partition is used to control which Kafka partition to publish data on.

With the above conventions, our event tables function as a transactional outbox where inserted rows can be picked up by Debezium, unwrapped and routed to Kafka.

Unwrap & Route

Since Debezium is a Kafka Connector, we can configure it to use custom transformers. We created such transformers that we call EventConverter, and EventRouter. The goal of these transformers is to unpack insertions made into the events table, pull out the Avro encoded message, and publish to Kafka on the right partition.

In a Dockerfile, we extended confluentinc/cp-kafka-connect:5.1.0 , added Debezium v0.9.5, and also added our own JAR files that contain both the EventConverter and EventRouter.

We then stitched it all together by adding the following to our Debezium configuration, where <registry-url> is the URL for our Confluent Schema Registry:

Limitations and Retrospective

What we described above is the transactional outbox pattern we implemented back in 2018. We’ve been using this pattern in production for about two years now. Much of the machinery we had to customize dealt with our desire to ensure our streaming API was loosely coupled with the database, and therefore required Avro encoding/decoding within the Debezium path.

All that said, it unfortunately hasn’t been all rainbows and sunshine. In the next part of this post, coming in the next few weeks, we’ll talk about some of the problems we ran into and how we’re handling them.

Contact Us

Clear Street does not provide investment, legal, regulatory, tax, or compliance advice. Consult professionals in these fields to address your specific circumstances. These materials are: (i) solely an overview of Clear Street’s products and services; (ii) provided for informational purposes only; and (iii) subject to change without notice or obligation to replace any information contained therein.

Products and services are offered by Clear Street LLC as a Broker Dealer member FINRA and SIPC and a Futures Commission Merchant registered with the CFTC and member of NFA. Additional information about Clear Street is available on FINRA BrokerCheck, including its Customer Relationship Summary and NFA BASIC | NFA (futures.org).

Copyright © 2024 Clear Street LLC. All rights reserved. Clear Street and the Shield Logo are Registered Trademarks of Clear Street LLC