Skip to content

Commit

Permalink
Merge pull request #678 from Shopify/produce-timestamp
Browse files Browse the repository at this point in the history
Support and provide producer message timestamps
  • Loading branch information
eapache authored Jun 10, 2016
2 parents 1ab4225 + 90b61cb commit e4bd495
Show file tree
Hide file tree
Showing 5 changed files with 54 additions and 7 deletions.
11 changes: 11 additions & 0 deletions async_producer.go
Original file line number Diff line number Diff line change
Expand Up @@ -135,6 +135,11 @@ type ProducerMessage struct {
// Partition is the partition that the message was sent to. This is only
// guaranteed to be defined if the message was successfully delivered.
Partition int32
// Timestamp is the timestamp assigned to the message by the broker. This
// is only guaranteed to be defined if the message was successfully
// delivered, RequiredAcks is not NoResponse, and the Kafka broker is at
// least version 0.10.0.
Timestamp time.Time

retries int
flags flagSet
Expand Down Expand Up @@ -722,6 +727,12 @@ func (bp *brokerProducer) handleSuccess(sent *produceSet, response *ProduceRespo
switch block.Err {
// Success
case ErrNoError:
if bp.parent.conf.Version.IsAtLeast(V0_10_0_0) {
timestamp := time.Unix(block.Timestamp, 0)
for _, msg := range msgs {
msg.Timestamp = timestamp
}
}
for i, msg := range msgs {
msg.Offset = block.Offset + int64(i)
}
Expand Down
15 changes: 15 additions & 0 deletions broker.go
Original file line number Diff line number Diff line change
Expand Up @@ -198,6 +198,21 @@ func (b *Broker) GetAvailableOffsets(request *OffsetRequest) (*OffsetResponse, e
}

func (b *Broker) Produce(request *ProduceRequest) (*ProduceResponse, error) {
switch request.version() {
case 0:
break
case 1:
if !b.conf.Version.IsAtLeast(V0_9_0_0) {
return nil, ErrUnsupportedVersion
}
case 2:
if !b.conf.Version.IsAtLeast(V0_10_0_0) {
return nil, ErrUnsupportedVersion
}
default:
return nil, ErrUnsupportedVersion
}

var response *ProduceResponse
var err error

Expand Down
3 changes: 2 additions & 1 deletion produce_request.go
Original file line number Diff line number Diff line change
Expand Up @@ -19,6 +19,7 @@ const (
type ProduceRequest struct {
RequiredAcks RequiredAcks
Timeout int32
Version int16 // v1 requires Kafka 0.9, v2 requires Kafka 0.10
msgSets map[string]map[int32]*MessageSet
}

Expand Down Expand Up @@ -110,7 +111,7 @@ func (p *ProduceRequest) key() int16 {
}

func (p *ProduceRequest) version() int16 {
return 0
return p.Version
}

func (p *ProduceRequest) AddMessage(topic string, partition int32, msg *Message) {
Expand Down
29 changes: 23 additions & 6 deletions produce_response.go
Original file line number Diff line number Diff line change
@@ -1,11 +1,12 @@
package sarama

type ProduceResponseBlock struct {
Err KError
Offset int64
Err KError
Offset int64
Timestamp int64 // only provided if Version >= 2
}

func (pr *ProduceResponseBlock) decode(pd packetDecoder) (err error) {
func (pr *ProduceResponseBlock) decode(pd packetDecoder, version int16) (err error) {
tmp, err := pd.getInt16()
if err != nil {
return err
Expand All @@ -17,14 +18,24 @@ func (pr *ProduceResponseBlock) decode(pd packetDecoder) (err error) {
return err
}

if version >= 2 {
if pr.Timestamp, err = pd.getInt64(); err != nil {
return err
}
}

return nil
}

type ProduceResponse struct {
Blocks map[string]map[int32]*ProduceResponseBlock
Blocks map[string]map[int32]*ProduceResponseBlock
Version int16
ThrottleTime int32 // only provided if Version >= 1
}

func (pr *ProduceResponse) decode(pd packetDecoder, version int16) (err error) {
pr.Version = version

numTopics, err := pd.getArrayLength()
if err != nil {
return err
Expand All @@ -51,14 +62,20 @@ func (pr *ProduceResponse) decode(pd packetDecoder, version int16) (err error) {
}

block := new(ProduceResponseBlock)
err = block.decode(pd)
err = block.decode(pd, version)
if err != nil {
return err
}
pr.Blocks[name][id] = block
}
}

if pr.Version >= 1 {
if pr.ThrottleTime, err = pd.getInt32(); err != nil {
return err
}
}

return nil
}

Expand Down Expand Up @@ -90,7 +107,7 @@ func (r *ProduceResponse) key() int16 {
}

func (r *ProduceResponse) version() int16 {
return 0
return r.Version
}

func (pr *ProduceResponse) GetBlock(topic string, partition int32) *ProduceResponseBlock {
Expand Down
3 changes: 3 additions & 0 deletions produce_set.go
Original file line number Diff line number Diff line change
Expand Up @@ -67,6 +67,9 @@ func (ps *produceSet) buildRequest() *ProduceRequest {
RequiredAcks: ps.parent.conf.Producer.RequiredAcks,
Timeout: int32(ps.parent.conf.Producer.Timeout / time.Millisecond),
}
if ps.parent.conf.Version.IsAtLeast(V0_10_0_0) {
req.Version = 2
}

for topic, partitionSet := range ps.msgs {
for partition, set := range partitionSet {
Expand Down

0 comments on commit e4bd495

Please sign in to comment.