sink_node.go 12 KB

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