Versions Compared

Key

  • This line was added.
  • This line was removed.
  • Formatting was changed.

...

There is one new method exposed on the partitioner interface. There will also be one new class. 


Code Block
languagejava
/**
 * Computes a partition andbased otheron relevant data. Knows whether a new batch will be created for the record.
 * 
 * @param topic The topic name
 * @param key The key to partition on (or null if no key)
 * @param keyBytes The serialized key to partition on( or null if no key)
 * @param value The value to partition on or null
 * @param valueBytes The serialized value to partition on or null
 * @param cluster The current cluster metadata
 * @param isNewBatch A boolean to specify whether a new batch will be created
 * @return ComputedPartition object to specify details about the The record's computed partition
 */
default ComputedPartitionint computePartition(String topic, Object key, byte[] keyBytes, Object value, byte[] valueBytes, Cluster cluster, boolean isNewBatch) {
    return new ComputedPartition(partition(topic, key, keyBytes, value, valueBytes, cluster));
}
Code Block
languagejava
    /**
     * Stores both the partition and whether computePartition should be called when a new batch
     * is about to be created.
     */
    class ComputedPartition{
        int part;
        boolean willCallOnNewBatch;
        
        public ComputedPartition(int part) {
             this.part = part;
             this.willCallOnNewBatch = false;
        }
        
        public ComputedPartition(int part, boolean willCallOnNewBatch) {
             this.part = part;
             this.willCallOnNewBatch = willCallOnNewBatch;
        }
        
        public boolean willCallOnNewBatch() { return this.willCallOnNewBatch; }
        public int partition() { return this.part; }
    }
 


The method computePartition will replace partition in the KafkaProducer. It will return a computePartition object that contains both the partition and a boolean that determines whether we call the method again exactly once for a record if that record would have created a new batch.an int to represent the new partition.The sticky partitioner will define this method so that when we have non-keyed, no explicit partition-id records, willCallOnNewBatch will be true and we will update the sticky partition when there is a new batch. All other cases will have the expected behaviorto update the sticky partition when a new batch is about to be created. This includes changing the sticky partition even when there will be a new batch on a keyed value. Test results show that this change will not significantly affect latency in the keyed value case.

The default of this method will result in no change to the current partitioning behavior for other user-defined partitioners. If a user wants to implement a sticky partitioner in their own partitioner class, this method can be overridden.

...

Change the behavior of the default partitioner in the no explicit partition, key = null case. Choose the “sticky” partition for the given topic. The “sticky” partition is changed when the record accumulator is allocating a new batch for a topic on a given partition.

These changes will slightly modify the code path for records that have keys as well, but the changes will not affect latency significantly.

If the DefaultPartitioner's partition method is called, it will no longer return the partition and instead throw an error.

...