sink_node.go 11 KB

123456789101112131415161718192021222324252627282930313233343536373839404142434445464748495051525354555657585960616263646566676869707172737475767778798081828384858687888990919293949596979899100101102103104105106107108109110111112113114115116117118119120121122123124125126127128129130131132133134135136137138139140141142143144145146147148149150151152153154155156157158159160161162163164165166167168169170171172173174175176177178179180181182183184185186187188189190191192193194195196197198199200201202203204205206207208209210211212213214215216217218219220221222223224225226227228229230231232233234235236237238239240241242243244245246247248249250251252253254255256257258259260261262263264265266267268269270271272273274275276277278279280281282283284285286287288289290291292293294295296297298299300301302303304305306307308309310311312313314315316317318319320321322323324325326327328329330331332333334335336337338339340341342343344345346347348349350351352353354355356357358359360361362363364365366367368369370371372373374
  1. // Copyright 2022 EMQ Technologies Co., Ltd.
  2. //
  3. // Licensed under the Apache License, Version 2.0 (the "License");
  4. // you may not use this file except in compliance with the License.
  5. // You may obtain a copy of the License at
  6. //
  7. // http://www.apache.org/licenses/LICENSE-2.0
  8. //
  9. // Unless required by applicable law or agreed to in writing, software
  10. // distributed under the License is distributed on an "AS IS" BASIS,
  11. // WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
  12. // See the License for the specific language governing permissions and
  13. // limitations under the License.
  14. package node
  15. import (
  16. "fmt"
  17. "github.com/lf-edge/ekuiper/internal/binder/io"
  18. "github.com/lf-edge/ekuiper/internal/conf"
  19. "github.com/lf-edge/ekuiper/internal/topo/context"
  20. "github.com/lf-edge/ekuiper/internal/topo/node/cache"
  21. "github.com/lf-edge/ekuiper/internal/topo/node/metric"
  22. "github.com/lf-edge/ekuiper/internal/topo/transform"
  23. "github.com/lf-edge/ekuiper/pkg/api"
  24. "github.com/lf-edge/ekuiper/pkg/cast"
  25. "github.com/lf-edge/ekuiper/pkg/errorx"
  26. "github.com/lf-edge/ekuiper/pkg/infra"
  27. "github.com/lf-edge/ekuiper/pkg/message"
  28. "strings"
  29. "sync"
  30. )
  31. type SinkConf struct {
  32. Concurrency int `json:"concurrency"`
  33. RunAsync bool `json:"runAsync"`
  34. Omitempty bool `json:"omitIfEmpty"`
  35. SendSingle bool `json:"sendSingle"`
  36. DataTemplate string `json:"dataTemplate"`
  37. Format string `json:"format"`
  38. SchemaId string `json:"schemaId"`
  39. BufferLength int `json:"bufferLength"`
  40. conf.SinkConf
  41. }
  42. type SinkNode struct {
  43. *defaultSinkNode
  44. //static
  45. sinkType string
  46. mutex sync.RWMutex
  47. //configs (also static for sinks)
  48. options map[string]interface{}
  49. isMock bool
  50. //states varies after restart
  51. sinks []api.Sink
  52. }
  53. func NewSinkNode(name string, sinkType string, props map[string]interface{}) *SinkNode {
  54. bufferLength := 1024
  55. if c, ok := props["bufferLength"]; ok {
  56. if t, err := cast.ToInt(c, cast.STRICT); err != nil || t <= 0 {
  57. //invalid property bufferLength
  58. } else {
  59. bufferLength = t
  60. }
  61. }
  62. return &SinkNode{
  63. defaultSinkNode: &defaultSinkNode{
  64. input: make(chan interface{}, bufferLength),
  65. defaultNode: &defaultNode{
  66. name: name,
  67. concurrency: 1,
  68. ctx: nil,
  69. },
  70. },
  71. sinkType: sinkType,
  72. options: props,
  73. }
  74. }
  75. // NewSinkNodeWithSink Only for mock source, do not use it in production
  76. func NewSinkNodeWithSink(name string, sink api.Sink, props map[string]interface{}) *SinkNode {
  77. return &SinkNode{
  78. defaultSinkNode: &defaultSinkNode{
  79. input: make(chan interface{}, 1024),
  80. defaultNode: &defaultNode{
  81. name: name,
  82. concurrency: 1,
  83. ctx: nil,
  84. },
  85. },
  86. sinks: []api.Sink{sink},
  87. options: props,
  88. isMock: true,
  89. }
  90. }
  91. func (m *SinkNode) Open(ctx api.StreamContext, result chan<- error) {
  92. m.ctx = ctx
  93. logger := ctx.GetLogger()
  94. logger.Debugf("open sink node %s", m.name)
  95. go func() {
  96. err := infra.SafeRun(func() error {
  97. sconf, err := m.parseConf(logger)
  98. if err != nil {
  99. return err
  100. }
  101. tf, err := transform.GenTransform(sconf.DataTemplate, sconf.Format, sconf.SchemaId)
  102. if err != nil {
  103. msg := fmt.Sprintf("property dataTemplate %v is invalid: %v", sconf.DataTemplate, err)
  104. logger.Warnf(msg)
  105. return fmt.Errorf(msg)
  106. }
  107. ctx = context.WithValue(ctx.(*context.DefaultContext), context.TransKey, tf)
  108. m.reset()
  109. logger.Infof("open sink node %d instances", m.concurrency)
  110. for i := 0; i < m.concurrency; i++ { // workers
  111. go func(instance int) {
  112. panicOrError := infra.SafeRun(func() error {
  113. var (
  114. sink api.Sink
  115. err error
  116. )
  117. if !m.isMock {
  118. logger.Debugf("Trying to get sink for rule %s with options %v\n", ctx.GetRuleId(), m.options)
  119. sink, err = getSink(m.sinkType, m.options)
  120. if err != nil {
  121. return err
  122. }
  123. logger.Debugf("Successfully get the sink %s", m.sinkType)
  124. m.mutex.Lock()
  125. m.sinks = append(m.sinks, sink)
  126. m.mutex.Unlock()
  127. logger.Debugf("Now is to open sink for rule %s.\n", ctx.GetRuleId())
  128. if err := sink.Open(ctx); err != nil {
  129. return err
  130. }
  131. logger.Debugf("Successfully open sink for rule %s.\n", ctx.GetRuleId())
  132. } else {
  133. sink = m.sinks[instance]
  134. }
  135. stats, err := metric.NewStatManager(ctx, "sink")
  136. if err != nil {
  137. return err
  138. }
  139. m.mutex.Lock()
  140. m.statManagers = append(m.statManagers, stats)
  141. m.mutex.Unlock()
  142. if !sconf.EnableCache {
  143. for {
  144. select {
  145. case data := <-m.input:
  146. if temp, processed := m.preprocess(data); processed {
  147. break
  148. } else {
  149. data = temp
  150. }
  151. stats.SetBufferLength(int64(len(m.input)))
  152. stats.IncTotalRecordsIn()
  153. if sconf.RunAsync {
  154. go func() {
  155. p := infra.SafeRun(func() error {
  156. _ = doCollect(ctx, sink, data, stats, sconf)
  157. return nil
  158. })
  159. if p != nil {
  160. infra.DrainError(ctx, p, result)
  161. }
  162. }()
  163. } else {
  164. _ = doCollect(ctx, sink, data, stats, sconf)
  165. }
  166. case <-ctx.Done():
  167. logger.Infof("sink node %s instance %d done", m.name, instance)
  168. if err := sink.Close(ctx); err != nil {
  169. logger.Warnf("close sink node %s instance %d fails: %v", m.name, instance, err)
  170. }
  171. return nil
  172. }
  173. }
  174. } else {
  175. logger.Infof("Creating sink cache")
  176. if sconf.RunAsync { // async mode, the ack must have an id
  177. // is not supported and validated in the configure, should not go here
  178. return fmt.Errorf("async mode is not supported for cache sink")
  179. } else { // sync mode, the ack is already in order
  180. c := cache.NewSyncCache(ctx, m.input, result, stats, &sconf.SinkConf, sconf.BufferLength)
  181. for {
  182. select {
  183. case data := <-c.Out:
  184. if temp, processed := m.preprocess(data); processed {
  185. break
  186. } else {
  187. data = temp
  188. }
  189. ack := true
  190. err := doCollect(ctx, sink, data, stats, sconf)
  191. // Only recoverable error should be cached
  192. if err != nil {
  193. if strings.HasPrefix(err.Error(), errorx.IOErr) { // do not log to prevent a lot of logs!
  194. ack = false
  195. } else {
  196. ctx.GetLogger().Warnf("sink node %s instance %d publish %s error: %v", ctx.GetOpId(), ctx.GetInstanceId(), data, err)
  197. }
  198. } else {
  199. ctx.GetLogger().Debugf("sent data to MQTT: %v", data)
  200. }
  201. select {
  202. case c.Ack <- ack:
  203. case <-ctx.Done():
  204. }
  205. case <-ctx.Done():
  206. logger.Infof("sink node %s instance %d done", m.name, instance)
  207. if err := sink.Close(ctx); err != nil {
  208. logger.Warnf("close sink node %s instance %d fails: %v", m.name, instance, err)
  209. }
  210. return nil
  211. }
  212. }
  213. }
  214. }
  215. return nil
  216. })
  217. if panicOrError != nil {
  218. infra.DrainError(ctx, panicOrError, result)
  219. }
  220. }(i)
  221. }
  222. return nil
  223. })
  224. if err != nil {
  225. infra.DrainError(ctx, err, result)
  226. }
  227. }()
  228. }
  229. func (m *SinkNode) parseConf(logger api.Logger) (*SinkConf, error) {
  230. sconf := &SinkConf{
  231. Concurrency: 1,
  232. RunAsync: false,
  233. Omitempty: false,
  234. SendSingle: false,
  235. DataTemplate: "",
  236. SinkConf: *conf.Config.Sink,
  237. BufferLength: 1024,
  238. }
  239. err := cast.MapToStruct(m.options, sconf)
  240. if err != nil {
  241. return nil, fmt.Errorf("read properties %v fail with error: %v", m.options, err)
  242. }
  243. if sconf.Concurrency <= 0 {
  244. logger.Warnf("invalid type for concurrency property, should be positive integer but found %t", sconf.Concurrency)
  245. sconf.Concurrency = 1
  246. }
  247. m.concurrency = sconf.Concurrency
  248. if sconf.Format == "" {
  249. sconf.Format = "json"
  250. } else if sconf.Format != message.FormatJson && sconf.Format != message.FormatProtobuf {
  251. logger.Warnf("invalid type for format property, should be json or protobuf but found %s", sconf.Format)
  252. sconf.Format = "json"
  253. }
  254. err = cast.MapToStruct(m.options, &sconf.SinkConf)
  255. if err != nil {
  256. return nil, fmt.Errorf("read properties %v to cache conf fail with error: %v", m.options, err)
  257. }
  258. if sconf.SinkConf.EnableCache && sconf.RunAsync {
  259. return nil, fmt.Errorf("cache is not supported for async sink, do not use enableCache and runAsync properties together")
  260. }
  261. err = sconf.SinkConf.Validate()
  262. if err != nil {
  263. return nil, fmt.Errorf("invalid cache properties: %v", err)
  264. }
  265. return sconf, err
  266. }
  267. func (m *SinkNode) reset() {
  268. if !m.isMock {
  269. m.sinks = nil
  270. }
  271. m.statManagers = nil
  272. }
  273. func doCollect(ctx api.StreamContext, sink api.Sink, item interface{}, stats metric.StatManager, sconf *SinkConf) error {
  274. stats.ProcessTimeStart()
  275. defer stats.ProcessTimeEnd()
  276. var outs []map[string]interface{}
  277. switch val := item.(type) {
  278. case error:
  279. outs = []map[string]interface{}{
  280. {"error": val.Error()},
  281. }
  282. case []map[string]interface{}:
  283. outs = val
  284. default:
  285. outs = []map[string]interface{}{
  286. {"error": fmt.Sprintf("result is not a map slice but found %#v", val)},
  287. }
  288. }
  289. if sconf.Omitempty && (item == nil || len(outs) == 0) {
  290. ctx.GetLogger().Debugf("receive empty in sink")
  291. return nil
  292. }
  293. if !sconf.SendSingle {
  294. return doCollectData(ctx, sink, outs, stats)
  295. } else {
  296. var err error
  297. for _, d := range outs {
  298. if sconf.Omitempty && (d == nil || len(d) == 0) {
  299. ctx.GetLogger().Debugf("receive empty in sink")
  300. continue
  301. }
  302. newErr := doCollectData(ctx, sink, d, stats)
  303. if newErr != nil {
  304. err = newErr
  305. }
  306. }
  307. return err
  308. }
  309. }
  310. // doCollectData outData must be map or []map
  311. func doCollectData(ctx api.StreamContext, sink api.Sink, outData interface{}, stats metric.StatManager) error {
  312. select {
  313. case <-ctx.Done():
  314. ctx.GetLogger().Infof("sink node %s instance %d stops data resending", ctx.GetOpId(), ctx.GetInstanceId())
  315. return nil
  316. default:
  317. if err := sink.Collect(ctx, outData); err != nil {
  318. stats.IncTotalExceptions()
  319. return err
  320. } else {
  321. ctx.GetLogger().Debugf("success")
  322. stats.IncTotalRecordsOut()
  323. return nil
  324. }
  325. }
  326. }
  327. func getSink(name string, action map[string]interface{}) (api.Sink, error) {
  328. var (
  329. s api.Sink
  330. err error
  331. )
  332. s, err = io.Sink(name)
  333. if s != nil {
  334. err = s.Configure(action)
  335. if err != nil {
  336. return nil, err
  337. }
  338. return s, nil
  339. } else {
  340. if err != nil {
  341. return nil, err
  342. } else {
  343. return nil, fmt.Errorf("sink %s not found", name)
  344. }
  345. }
  346. }
  347. // AddOutput Override defaultNode
  348. func (m *SinkNode) AddOutput(_ chan<- interface{}, name string) error {
  349. return fmt.Errorf("fail to add output %s, sink %s cannot add output", name, m.name)
  350. }
  351. // Broadcast Override defaultNode
  352. func (m *SinkNode) Broadcast(_ interface{}) error {
  353. return fmt.Errorf("sink %s cannot add broadcast", m.name)
  354. }