sink_node.go 12 KB

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