config.go 5.0 KB

123456789101112131415161718192021222324252627282930313233343536373839404142434445464748495051525354555657585960616263646566676869707172737475767778798081828384858687888990919293949596979899100101102103104105106107108109110111112113114115116117118119120121122123124125126127128129130131132133134135136137138139140141142143144145146
  1. package cluster
  2. import (
  3. "regexp"
  4. "time"
  5. "github.com/Shopify/sarama"
  6. )
  7. var minVersion = sarama.V0_9_0_0
  8. type ConsumerMode uint8
  9. const (
  10. ConsumerModeMultiplex ConsumerMode = iota
  11. ConsumerModePartitions
  12. )
  13. // Config extends sarama.Config with Group specific namespace
  14. type Config struct {
  15. sarama.Config
  16. // Group is the namespace for group management properties
  17. Group struct {
  18. // The strategy to use for the allocation of partitions to consumers (defaults to StrategyRange)
  19. PartitionStrategy Strategy
  20. // By default, messages and errors from the subscribed topics and partitions are all multiplexed and
  21. // made available through the consumer's Messages() and Errors() channels.
  22. //
  23. // Users who require low-level access can enable ConsumerModePartitions where individual partitions
  24. // are exposed on the Partitions() channel. Messages and errors must then be consumed on the partitions
  25. // themselves.
  26. Mode ConsumerMode
  27. Offsets struct {
  28. Retry struct {
  29. // The numer retries when committing offsets (defaults to 3).
  30. Max int
  31. }
  32. Synchronization struct {
  33. // The duration allowed for other clients to commit their offsets before resumption in this client, e.g. during a rebalance
  34. // NewConfig sets this to the Consumer.MaxProcessingTime duration of the Sarama configuration
  35. DwellTime time.Duration
  36. }
  37. }
  38. Session struct {
  39. // The allowed session timeout for registered consumers (defaults to 30s).
  40. // Must be within the allowed server range.
  41. Timeout time.Duration
  42. }
  43. Heartbeat struct {
  44. // Interval between each heartbeat (defaults to 3s). It should be no more
  45. // than 1/3rd of the Group.Session.Timout setting
  46. Interval time.Duration
  47. }
  48. // Return specifies which group channels will be populated. If they are set to true,
  49. // you must read from the respective channels to prevent deadlock.
  50. Return struct {
  51. // If enabled, rebalance notification will be returned on the
  52. // Notifications channel (default disabled).
  53. Notifications bool
  54. }
  55. Topics struct {
  56. // An additional whitelist of topics to subscribe to.
  57. Whitelist *regexp.Regexp
  58. // An additional blacklist of topics to avoid. If set, this will precede over
  59. // the Whitelist setting.
  60. Blacklist *regexp.Regexp
  61. }
  62. Member struct {
  63. // Custom metadata to include when joining the group. The user data for all joined members
  64. // can be retrieved by sending a DescribeGroupRequest to the broker that is the
  65. // coordinator for the group.
  66. UserData []byte
  67. }
  68. }
  69. }
  70. // NewConfig returns a new configuration instance with sane defaults.
  71. func NewConfig() *Config {
  72. c := &Config{
  73. Config: *sarama.NewConfig(),
  74. }
  75. c.Group.PartitionStrategy = StrategyRange
  76. c.Group.Offsets.Retry.Max = 3
  77. c.Group.Offsets.Synchronization.DwellTime = c.Consumer.MaxProcessingTime
  78. c.Group.Session.Timeout = 30 * time.Second
  79. c.Group.Heartbeat.Interval = 3 * time.Second
  80. c.Config.Version = minVersion
  81. return c
  82. }
  83. // Validate checks a Config instance. It will return a
  84. // sarama.ConfigurationError if the specified values don't make sense.
  85. func (c *Config) Validate() error {
  86. if c.Group.Heartbeat.Interval%time.Millisecond != 0 {
  87. sarama.Logger.Println("Group.Heartbeat.Interval only supports millisecond precision; nanoseconds will be truncated.")
  88. }
  89. if c.Group.Session.Timeout%time.Millisecond != 0 {
  90. sarama.Logger.Println("Group.Session.Timeout only supports millisecond precision; nanoseconds will be truncated.")
  91. }
  92. if c.Group.PartitionStrategy != StrategyRange && c.Group.PartitionStrategy != StrategyRoundRobin {
  93. sarama.Logger.Println("Group.PartitionStrategy is not supported; range will be assumed.")
  94. }
  95. if !c.Version.IsAtLeast(minVersion) {
  96. sarama.Logger.Println("Version is not supported; 0.9. will be assumed.")
  97. c.Version = minVersion
  98. }
  99. if err := c.Config.Validate(); err != nil {
  100. return err
  101. }
  102. // validate the Group values
  103. switch {
  104. case c.Group.Offsets.Retry.Max < 0:
  105. return sarama.ConfigurationError("Group.Offsets.Retry.Max must be >= 0")
  106. case c.Group.Offsets.Synchronization.DwellTime <= 0:
  107. return sarama.ConfigurationError("Group.Offsets.Synchronization.DwellTime must be > 0")
  108. case c.Group.Offsets.Synchronization.DwellTime > 10*time.Minute:
  109. return sarama.ConfigurationError("Group.Offsets.Synchronization.DwellTime must be <= 10m")
  110. case c.Group.Heartbeat.Interval <= 0:
  111. return sarama.ConfigurationError("Group.Heartbeat.Interval must be > 0")
  112. case c.Group.Session.Timeout <= 0:
  113. return sarama.ConfigurationError("Group.Session.Timeout must be > 0")
  114. case !c.Metadata.Full && c.Group.Topics.Whitelist != nil:
  115. return sarama.ConfigurationError("Metadata.Full must be enabled when Group.Topics.Whitelist is used")
  116. case !c.Metadata.Full && c.Group.Topics.Blacklist != nil:
  117. return sarama.ConfigurationError("Metadata.Full must be enabled when Group.Topics.Blacklist is used")
  118. }
  119. // ensure offset is correct
  120. switch c.Consumer.Offsets.Initial {
  121. case sarama.OffsetOldest, sarama.OffsetNewest:
  122. default:
  123. return sarama.ConfigurationError("Consumer.Offsets.Initial must be either OffsetOldest or OffsetNewest")
  124. }
  125. return nil
  126. }