소스 검색

优化设备消息topic,增加设备预警接口

zhou-hao 5 년 전
부모
커밋
c3c2230ba9
27개의 변경된 파일875개의 추가작업 그리고 37개의 파일을 삭제
  1. 25 0
      jetlinks-components/gateway-component/src/main/java/org/jetlinks/community/gateway/DeviceMessageUtils.java
  2. 2 1
      jetlinks-manager/device-manager/src/main/java/org/jetlinks/community/device/entity/DeviceInstanceEntity.java
  3. 10 8
      jetlinks-manager/device-manager/src/main/java/org/jetlinks/community/device/measurements/DeviceDashboardObject.java
  4. 10 2
      jetlinks-manager/device-manager/src/main/java/org/jetlinks/community/device/measurements/DeviceEventMeasurement.java
  5. 2 1
      jetlinks-manager/device-manager/src/main/java/org/jetlinks/community/device/measurements/DeviceEventsMeasurement.java
  6. 14 6
      jetlinks-manager/device-manager/src/main/java/org/jetlinks/community/device/measurements/DevicePropertiesMeasurement.java
  7. 9 3
      jetlinks-manager/device-manager/src/main/java/org/jetlinks/community/device/measurements/DevicePropertyMeasurement.java
  8. 2 1
      jetlinks-manager/device-manager/src/main/java/org/jetlinks/community/device/measurements/message/DeviceMessageMeasurementProvider.java
  9. 2 2
      jetlinks-manager/device-manager/src/main/java/org/jetlinks/community/device/measurements/status/DeviceStatusChangeMeasurement.java
  10. 2 2
      jetlinks-manager/device-manager/src/main/java/org/jetlinks/community/device/measurements/status/DeviceStatusMeasurementProvider.java
  11. 5 3
      jetlinks-manager/device-manager/src/main/java/org/jetlinks/community/device/message/DeviceMessageConnector.java
  12. 1 1
      jetlinks-manager/device-manager/src/main/java/org/jetlinks/community/device/message/writer/TimeSeriesMessageWriterConnector.java
  13. 3 3
      jetlinks-manager/device-manager/src/main/java/org/jetlinks/community/device/service/LocalDeviceInstanceService.java
  14. 1 1
      jetlinks-manager/device-manager/src/main/java/org/jetlinks/community/device/web/DeviceMessageController.java
  15. 1 2
      jetlinks-manager/device-manager/src/main/java/org/jetlinks/community/device/web/GatewayDeviceController.java
  16. 4 1
      jetlinks-manager/rule-engine-manager/pom.xml
  17. 205 0
      jetlinks-manager/rule-engine-manager/src/main/java/org/jetlinks/community/rule/engine/device/DeviceAlarmRule.java
  18. 173 0
      jetlinks-manager/rule-engine-manager/src/main/java/org/jetlinks/community/rule/engine/device/DeviceAlarmRuleNode.java
  19. 76 0
      jetlinks-manager/rule-engine-manager/src/main/java/org/jetlinks/community/rule/engine/entity/DeviceAlarmEntity.java
  20. 60 0
      jetlinks-manager/rule-engine-manager/src/main/java/org/jetlinks/community/rule/engine/entity/DeviceAlarmHistoryEntity.java
  21. 20 0
      jetlinks-manager/rule-engine-manager/src/main/java/org/jetlinks/community/rule/engine/enums/AlarmState.java
  22. 62 0
      jetlinks-manager/rule-engine-manager/src/main/java/org/jetlinks/community/rule/engine/model/DeviceAlarmModelParser.java
  23. 43 0
      jetlinks-manager/rule-engine-manager/src/main/java/org/jetlinks/community/rule/engine/service/DeviceAlarmHistoryService.java
  24. 39 0
      jetlinks-manager/rule-engine-manager/src/main/java/org/jetlinks/community/rule/engine/service/DeviceAlarmService.java
  25. 67 0
      jetlinks-manager/rule-engine-manager/src/main/java/org/jetlinks/community/rule/engine/web/DeviceAlarmController.java
  26. 30 0
      jetlinks-manager/rule-engine-manager/src/main/java/org/jetlinks/community/rule/engine/web/DeviceAlarmHistoryController.java
  27. 7 0
      pom.xml

+ 25 - 0
jetlinks-components/gateway-component/src/main/java/org/jetlinks/community/gateway/DeviceMessageUtils.java

@@ -0,0 +1,25 @@
+package org.jetlinks.community.gateway;
+
+import com.alibaba.fastjson.JSON;
+import org.jetlinks.core.message.DeviceMessage;
+import org.jetlinks.core.message.MessageType;
+
+import java.nio.charset.StandardCharsets;
+import java.util.Map;
+import java.util.Optional;
+
+public class DeviceMessageUtils {
+
+    public static Optional<DeviceMessage> convert(TopicMessage message){
+        if (message.getMessage() instanceof EncodableMessage) {
+            Object nativeMessage = ((EncodableMessage) message.getMessage()).getNativePayload();
+            if (nativeMessage instanceof DeviceMessage) {
+                return Optional.of((DeviceMessage)nativeMessage);
+            } else if (nativeMessage instanceof Map) {
+                return MessageType.convertMessage(((Map<String, Object>) nativeMessage));
+            }
+        }
+        return MessageType.convertMessage(JSON.parseObject(message.getMessage().getPayload().toString(StandardCharsets.UTF_8)));
+    }
+
+}

+ 2 - 1
jetlinks-manager/device-manager/src/main/java/org/jetlinks/community/device/entity/DeviceInstanceEntity.java

@@ -101,7 +101,8 @@ public class DeviceInstanceEntity extends GenericEntity<String> implements Recor
             .productId(this.getProductId())
             .build()
             .addConfig(DeviceConfigKey.parentGatewayId, this.getParentId());
-
+        info.addConfig("deviceName", name);
+        info.addConfig("productName", productName);
         if (!CollectionUtils.isEmpty(configuration)) {
             configuration.forEach(info::addConfig);
         }

+ 10 - 8
jetlinks-manager/device-manager/src/main/java/org/jetlinks/community/device/measurements/DeviceDashboardObject.java

@@ -24,7 +24,8 @@ public class DeviceDashboardObject implements DashboardObject {
 
     private DeviceDashboardObject(String id, String name,
                                   DeviceProductOperator productOperator,
-                                  MessageGateway messageGateway, TimeSeriesManager timeSeriesManager) {
+                                  MessageGateway messageGateway,
+                                  TimeSeriesManager timeSeriesManager) {
         this.id = id;
         this.name = name;
         this.productOperator = productOperator;
@@ -34,7 +35,8 @@ public class DeviceDashboardObject implements DashboardObject {
 
     public static DeviceDashboardObject of(String id, String name,
                                            DeviceProductOperator productOperator,
-                                           MessageGateway messageGateway, TimeSeriesManager timeSeriesManager) {
+                                           MessageGateway messageGateway,
+                                           TimeSeriesManager timeSeriesManager) {
         return new DeviceDashboardObject(id, name, productOperator, messageGateway, timeSeriesManager);
     }
 
@@ -59,17 +61,17 @@ public class DeviceDashboardObject implements DashboardObject {
 
             productOperator.getMetadata()
                 .flatMapIterable(DeviceMetadata::getEvents)
-                .map(event -> new DeviceEventMeasurement(messageGateway, event, timeSeriesManager.getService(DeviceTimeSeriesMetric.deviceEventMetric(id, event.getId())))),
+                .map(event -> new DeviceEventMeasurement(productOperator.getId(), messageGateway, event, timeSeriesManager.getService(DeviceTimeSeriesMetric.deviceEventMetric(id, event.getId())))),
 
             productOperator.getMetadata()
-                .map(metadata -> new DevicePropertiesMeasurement(messageGateway, metadata, timeSeriesManager.getService(DeviceTimeSeriesMetric.devicePropertyMetric(id)))),
+                .map(metadata -> new DevicePropertiesMeasurement(productOperator.getId(),messageGateway, metadata, timeSeriesManager.getService(DeviceTimeSeriesMetric.devicePropertyMetric(id)))),
 
             productOperator.getMetadata()
                 .map(metadata -> new DeviceEventsMeasurement(productOperator.getId(), messageGateway, metadata, timeSeriesManager)),
 
             productOperator.getMetadata()
                 .flatMapIterable(DeviceMetadata::getProperties)
-                .map(event -> new DevicePropertyMeasurement(messageGateway, event, timeSeriesManager.getService(DeviceTimeSeriesMetric.devicePropertyMetric(id))))
+                .map(event -> new DevicePropertyMeasurement(productOperator.getId(),messageGateway, event, timeSeriesManager.getService(DeviceTimeSeriesMetric.devicePropertyMetric(id))))
         );
     }
 
@@ -77,7 +79,7 @@ public class DeviceDashboardObject implements DashboardObject {
     public Mono<Measurement> getMeasurement(String id) {
         if ("properties".equals(id)) {
             return productOperator.getMetadata()
-                .map(metadata -> new DevicePropertiesMeasurement(messageGateway, metadata, timeSeriesManager.getService(DeviceTimeSeriesMetric.devicePropertyMetric(this.id))));
+                .map(metadata -> new DevicePropertiesMeasurement(productOperator.getId(),messageGateway, metadata, timeSeriesManager.getService(DeviceTimeSeriesMetric.devicePropertyMetric(this.id))));
         }
         if ("events".equals(id)) {
             return productOperator.getMetadata()
@@ -85,10 +87,10 @@ public class DeviceDashboardObject implements DashboardObject {
         }
         return productOperator.getMetadata()
             .flatMap(metadata -> Mono.justOrEmpty(metadata.getEvent(id)))
-            .<Measurement>map(event -> new DeviceEventMeasurement(messageGateway, event, timeSeriesManager.getService(DeviceTimeSeriesMetric.deviceEventMetric(this.id, event.getId()))))
+            .<Measurement>map(event -> new DeviceEventMeasurement(productOperator.getId(),messageGateway, event, timeSeriesManager.getService(DeviceTimeSeriesMetric.deviceEventMetric(this.id, event.getId()))))
             //事件没获取到则尝试获取属性
             .switchIfEmpty(productOperator.getMetadata()
                 .flatMap(metadata -> Mono.justOrEmpty(metadata.getProperty(id)))
-                .map(event -> new DevicePropertyMeasurement(messageGateway, event, timeSeriesManager.getService(DeviceTimeSeriesMetric.devicePropertyMetric(this.id)))));
+                .map(event -> new DevicePropertyMeasurement(productOperator.getId(),messageGateway, event, timeSeriesManager.getService(DeviceTimeSeriesMetric.devicePropertyMetric(this.id)))));
     }
 }

+ 10 - 2
jetlinks-manager/device-manager/src/main/java/org/jetlinks/community/device/measurements/DeviceEventMeasurement.java

@@ -27,8 +27,14 @@ class DeviceEventMeasurement extends StaticMeasurement {
 
     private TimeSeriesService eventTsService;
 
-    public DeviceEventMeasurement(MessageGateway messageGateway, EventMetadata eventMetadata, TimeSeriesService eventTsService) {
+    private String productId;
+
+    public DeviceEventMeasurement(String productId,
+                                  MessageGateway messageGateway,
+                                  EventMetadata eventMetadata,
+                                  TimeSeriesService eventTsService) {
         super(MetadataMeasurementDefinition.of(eventMetadata));
+        this.productId=productId;
         this.messageGateway = messageGateway;
         this.eventMetadata = eventMetadata;
         this.eventTsService = eventTsService;
@@ -39,6 +45,7 @@ class DeviceEventMeasurement extends StaticMeasurement {
         .add("deviceId", "设备", "指定设备", new StringType().expand("selector", "device-selector"))
         .add("history", "历史数据量", "查询出历史数据后开始推送实时数据", new IntType().min(0).expand("defaultValue", 10));
 
+
     Flux<SimpleMeasurementValue> fromHistory(String deviceId, int history) {
         return history <= 0 ? Flux.empty() : QueryParamEntity.newQuery()
             .doPaging(0, history)
@@ -48,9 +55,10 @@ class DeviceEventMeasurement extends StaticMeasurement {
             .sort(MeasurementValue.sort());
     }
 
+
     Flux<MeasurementValue> fromRealTime(String deviceId) {
         return messageGateway
-            .subscribe(Collections.singletonList(new Subscription("/device/" + deviceId + "/message/event/" + eventMetadata.getId())), true)
+            .subscribe(Collections.singletonList(new Subscription("/device/"+productId+"/" + deviceId + "/message/event/" + eventMetadata.getId())), true)
             .flatMap(val -> Mono.justOrEmpty(DeviceMessageUtils.convert(val)))
             .cast(EventMessage.class)
             .map(msg -> SimpleMeasurementValue.of(msg.getData(), msg.getTimestamp()));

+ 2 - 1
jetlinks-manager/device-manager/src/main/java/org/jetlinks/community/device/measurements/DeviceEventsMeasurement.java

@@ -21,6 +21,7 @@ import java.util.concurrent.atomic.AtomicLong;
 
 class DeviceEventsMeasurement extends StaticMeasurement {
 
+
     private MessageGateway messageGateway;
 
     private TimeSeriesManager timeSeriesManager;
@@ -62,7 +63,7 @@ class DeviceEventsMeasurement extends StaticMeasurement {
 
     Flux<MeasurementValue> fromRealTime(String deviceId) {
         return messageGateway
-            .subscribe(Subscription.asList("/device/" + deviceId + "/message/event/*")
+            .subscribe(Subscription.asList("/device/"+productId+"/" + deviceId + "/message/event/*")
                 , "realtime-device-events-measurement:" + Math.abs(num.incrementAndGet())
                 , true)
             .flatMap(val -> Mono.justOrEmpty(DeviceMessageUtils.convert(val)))

+ 14 - 6
jetlinks-manager/device-manager/src/main/java/org/jetlinks/community/device/measurements/DevicePropertiesMeasurement.java

@@ -18,6 +18,7 @@ import reactor.core.publisher.Mono;
 
 import java.util.HashMap;
 import java.util.Map;
+import java.util.concurrent.atomic.AtomicLong;
 import java.util.stream.Collectors;
 import java.util.stream.Stream;
 
@@ -28,15 +29,22 @@ class DevicePropertiesMeasurement extends StaticMeasurement {
     private TimeSeriesService timeSeriesService;
 
     private DeviceMetadata metadata;
+    private String productId;
 
-    public DevicePropertiesMeasurement(MessageGateway messageGateway, DeviceMetadata deviceMetadata, TimeSeriesService timeSeriesService) {
+    public DevicePropertiesMeasurement(String productId,
+                                       MessageGateway messageGateway,
+                                       DeviceMetadata deviceMetadata,
+                                       TimeSeriesService timeSeriesService) {
         super(MeasurementDefinition.of("properties", "属性记录"));
+        this.productId = productId;
         this.messageGateway = messageGateway;
         this.timeSeriesService = timeSeriesService;
         this.metadata = deviceMetadata;
         addDimension(new RealTimeDevicePropertyDimension());
     }
 
+    static AtomicLong num = new AtomicLong();
+
     Flux<SimpleMeasurementValue> fromHistory(String deviceId, int history) {
         return history <= 0 ? Flux.empty() : Flux.fromIterable(metadata.getProperties())
             .flatMap(propertyMetadata -> QueryParamEntity.newQuery()
@@ -71,11 +79,11 @@ class DevicePropertiesMeasurement extends StaticMeasurement {
 
     Flux<MeasurementValue> fromRealTime(String deviceId) {
         return messageGateway
-            .subscribe(Stream.of(
-                "/device/" + deviceId + "/message/property/report"
-                , "/device/" + deviceId + "/message/property/*/reply")
-                .map(Subscription::new)
-                .collect(Collectors.toList()), true)
+            .subscribe(Subscription.asList(
+                "/device/" + productId + "/" + deviceId + "/message/property/report",
+                "/device/" + productId + "/" + deviceId + "/message/property/*/reply")
+                , "realtime-device-properties-measurement:" + Math.abs(num.incrementAndGet())
+                , true)
             .flatMap(val -> Mono.justOrEmpty(DeviceMessageUtils.convert(val)))
             .flatMap(msg -> {
                 if (msg instanceof ReportPropertyMessage) {

+ 9 - 3
jetlinks-manager/device-manager/src/main/java/org/jetlinks/community/device/measurements/DevicePropertyMeasurement.java

@@ -30,8 +30,14 @@ class DevicePropertyMeasurement extends StaticMeasurement {
 
     private TimeSeriesService timeSeriesService;
 
-    public DevicePropertyMeasurement(MessageGateway messageGateway, PropertyMetadata metadata, TimeSeriesService timeSeriesService) {
+    private String productId;
+
+    public DevicePropertyMeasurement(String productId,
+                                     MessageGateway messageGateway,
+                                     PropertyMetadata metadata,
+                                     TimeSeriesService timeSeriesService) {
         super(MetadataMeasurementDefinition.of(metadata));
+        this.productId=productId;
         this.messageGateway = messageGateway;
         this.metadata = metadata;
         this.timeSeriesService = timeSeriesService;
@@ -61,8 +67,8 @@ class DevicePropertyMeasurement extends StaticMeasurement {
     Flux<MeasurementValue> fromRealTime(String deviceId) {
         return messageGateway
             .subscribe(Stream.of(
-                "/device/" + deviceId + "/message/property/report"
-                , "/device/" + deviceId + "/message/property/*/reply")
+                "/device/"+productId+"/" + deviceId + "/message/property/report"
+                , "/device/"+productId+"/" + deviceId + "/message/property/*/reply")
                 .map(Subscription::new)
                 .collect(Collectors.toList()), true)
             .flatMap(val -> Mono.justOrEmpty(DeviceMessageUtils.convert(val)))

+ 2 - 1
jetlinks-manager/device-manager/src/main/java/org/jetlinks/community/device/measurements/message/DeviceMessageMeasurementProvider.java

@@ -21,6 +21,7 @@ import java.time.Duration;
 @Component
 public class DeviceMessageMeasurementProvider extends StaticMeasurementProvider {
 
+
     MeterRegistry registry;
 
     public DeviceMessageMeasurementProvider(MessageGateway messageGateway,
@@ -34,7 +35,7 @@ public class DeviceMessageMeasurementProvider extends StaticMeasurementProvider
 
     }
 
-    @Subscribe("/device/*/message/**")
+    @Subscribe("/device/*/*/message/**")
     public Mono<Void> incrementMessage(TopicMessage message) {
         return Mono.fromRunnable(() -> {
             registry

+ 2 - 2
jetlinks-manager/device-manager/src/main/java/org/jetlinks/community/device/measurements/status/DeviceStatusChangeMeasurement.java

@@ -142,8 +142,8 @@ class DeviceStatusChangeMeasurement extends StaticMeasurement {
                     //从消息网关订阅消息
                     return messageGateway
                         .subscribe(Arrays.asList(
-                            new Subscription("/device/" + deviceId + "/online"),
-                            new Subscription("/device/" + deviceId + "/offline")), true)
+                            new Subscription("/device/*/" + deviceId + "/online"),
+                            new Subscription("/device/*/" + deviceId + "/offline")), true)
                         .flatMap(val -> Mono.justOrEmpty(DeviceMessageUtils.convert(val)))
                         .map(msg -> SimpleMeasurementValue.of(msg.getMessageType().name().toLowerCase(), msg.getTimestamp()))
                         ;

+ 2 - 2
jetlinks-manager/device-manager/src/main/java/org/jetlinks/community/device/measurements/status/DeviceStatusMeasurementProvider.java

@@ -51,7 +51,7 @@ public class DeviceStatusMeasurementProvider extends StaticMeasurementProvider {
             "target", "msgType", "productId");
     }
 
-    @Subscribe("/device/*/online")
+    @Subscribe("/device/*/*/online")
     public Mono<Void> incrementOnline(TopicMessage msg){
         return Mono.fromRunnable(()->{
             String productId = parseProductId(msg);
@@ -62,7 +62,7 @@ public class DeviceStatusMeasurementProvider extends StaticMeasurementProvider {
         });
     }
 
-    @Subscribe("/device/*/offline")
+    @Subscribe("/device/*/*/offline")
     public Mono<Void> incrementOffline(TopicMessage msg){
         return Mono.fromRunnable(()->{
             String productId = parseProductId(msg);

+ 5 - 3
jetlinks-manager/device-manager/src/main/java/org/jetlinks/community/device/message/DeviceMessageConnector.java

@@ -35,8 +35,7 @@ public class DeviceMessageConnector
     private FluxSink<TopicMessage> sink = messageProcessor.sink();
 
     //将设备注册中心到配置追加到消息header中,下游订阅者可直接使用.
-    private String[] appendConfigHeader = {"orgId", "productId"};
-
+    private String[] appendConfigHeader = { "orgId", "productId","deviceName"};
     //设备注册中心
     private final DeviceRegistry registry;
 
@@ -105,7 +104,10 @@ public class DeviceMessageConnector
                 .switchIfEmpty(Mono.fromSupplier(() -> Values.of(new HashMap<>())))
                 .flatMap(configs -> {
                     configs.getAllValues().forEach(deviceMessage::addHeader);
-                    String topic = "/device/".concat(deviceId).concat(createDeviceMessageTopic(message));
+                    String productId = deviceMessage.getHeader("productId").map(String::valueOf).orElse("null");
+                    String topic = String.join("",
+                        "/device", "/", productId, "/", deviceId, createDeviceMessageTopic(message)
+                    );
                     if (message instanceof ChildDeviceMessage) { //子设备消息
                         return onMessage(((ChildDeviceMessage) message).getChildDeviceMessage())
                             .thenReturn(topic);

+ 1 - 1
jetlinks-manager/device-manager/src/main/java/org/jetlinks/community/device/message/writer/TimeSeriesMessageWriterConnector.java

@@ -61,7 +61,7 @@ public class TimeSeriesMessageWriterConnector{
     private Mono<Void> doIndex(DeviceMessage message) {
         Map<String, Object> headers = Optional.ofNullable(message.getHeaders()).orElse(Collections.emptyMap());
 
-        String productId = (String) headers.get("productId");
+        String productId = (String) headers.getOrDefault("productId","null");
 
         DeviceOperationLogEntity operationLog = new DeviceOperationLogEntity();
         operationLog.setId(IDGenerator.MD5.generate());

+ 3 - 3
jetlinks-manager/device-manager/src/main/java/org/jetlinks/community/device/service/LocalDeviceInstanceService.java

@@ -368,7 +368,7 @@ public class LocalDeviceInstanceService extends GenericReactiveCrudService<Devic
 
         //订阅设备上下线
         FluxUtils.bufferRate(messageGateway
-            .subscribe(Subscription.asList("/device/*/online", "/device/*/offline"), "device-state-synchronizer", false)
+            .subscribe(Subscription.asList("/device/*/*/online", "/device/*/*/offline"), "device-state-synchronizer", false)
             .flatMap(message -> Mono.justOrEmpty(DeviceMessageUtils.convert(message))
                 .map(DeviceMessage::getDeviceId)), 800, 200, Duration.ofSeconds(2))
             .publishOn(Schedulers.parallel())
@@ -431,7 +431,7 @@ public class LocalDeviceInstanceService extends GenericReactiveCrudService<Devic
     }
 
 
-    @Subscribe("/device/*/message/children/*/register")
+    @Subscribe("/device/*/*/message/children/*/register")
     public Mono<Void> autoBindChildrenDevice(ChildDeviceMessage message) {
         String childId = message.getChildDeviceId();
         Message childMessage = message.getChildDeviceMessage();
@@ -451,7 +451,7 @@ public class LocalDeviceInstanceService extends GenericReactiveCrudService<Devic
         return Mono.empty();
     }
 
-    @Subscribe("/device/*/message/children/*/unregister")
+    @Subscribe("/device/*/*/message/children/*/unregister")
     public Mono<Void> autoUnbindChildrenDevice(ChildDeviceMessage message) {
         String childId = message.getChildDeviceId();
         Message childMessage = message.getChildDeviceMessage();

+ 1 - 1
jetlinks-manager/device-manager/src/main/java/org/jetlinks/community/device/web/DeviceMessageController.java

@@ -52,7 +52,7 @@ public class DeviceMessageController {
     @GetMapping(value = "/{deviceId}/event", produces = MediaType.TEXT_EVENT_STREAM_VALUE)
     public Flux<Object> getEvent(@PathVariable String deviceId) {
         return messageGateway
-            .subscribe("/device/".concat(deviceId).concat("/message/event/**"))
+            .subscribe("/device/*/".concat(deviceId).concat("/message/event/**"))
             .map(TopicMessage::getMessage)
             .map(msg -> msg.getPayload().toString(StandardCharsets.UTF_8))
             ;

+ 1 - 2
jetlinks-manager/device-manager/src/main/java/org/jetlinks/community/device/web/GatewayDeviceController.java

@@ -64,8 +64,7 @@ public class GatewayDeviceController {
         return getGatewayProductList()
             .flatMap(productIdList ->
                 param.toNestQuery(q -> q.in(DeviceInstanceEntity::getProductId, productIdList))
-                    .execute(Mono::just)
-                    .as(instanceService::queryPager)
+                    .execute(instanceService::queryPager)
                     .filter(r -> r.getTotal() > 0)
                     .flatMap(result -> {
                         Map<String, DeviceInstanceEntity> mapping =

+ 4 - 1
jetlinks-manager/rule-engine-manager/pom.xml

@@ -12,7 +12,10 @@
     </parent>
     <artifactId>rule-engine-manager</artifactId>
     <dependencies>
-
+        <dependency>
+            <groupId>org.jetlinks</groupId>
+            <artifactId>reactor-ql</artifactId>
+        </dependency>
         <dependency>
             <groupId>org.hswebframework.web</groupId>
             <artifactId>hsweb-authorization-api</artifactId>

+ 205 - 0
jetlinks-manager/rule-engine-manager/src/main/java/org/jetlinks/community/rule/engine/device/DeviceAlarmRule.java

@@ -0,0 +1,205 @@
+package org.jetlinks.community.rule.engine.device;
+
+import lombok.AllArgsConstructor;
+import lombok.Getter;
+import lombok.Setter;
+import org.jetlinks.rule.engine.api.executor.RuleNodeConfiguration;
+import org.jetlinks.rule.engine.api.model.RuleNodeModel;
+import org.jetlinks.rule.engine.executor.ExecutableRuleNodeFactoryStrategy;
+import org.springframework.util.CollectionUtils;
+import org.springframework.util.StringUtils;
+
+import java.io.Serializable;
+import java.util.List;
+import java.util.Map;
+import java.util.stream.Collectors;
+import java.util.stream.Stream;
+
+/**
+ * 设备预警规则
+ *
+ * @author zhouhao
+ * @since 1.1
+ */
+@Getter
+@Setter
+public class DeviceAlarmRule implements Serializable {
+
+    /**
+     * 规则ID
+     */
+    private String id;
+
+    /**
+     * 规则名称
+     */
+    private String name;
+
+    /**
+     * 产品ID,不能为空
+     */
+    private String productId;
+
+    /**
+     * 产品名称,不能为空
+     */
+    private String productName;
+
+    /**
+     * 设备ID,当对特定对设备设置规则时,不能为空
+     */
+    private String deviceId;
+
+    /**
+     * 设备名称
+     */
+    private String deviceName;
+
+    /**
+     * 类型类型,属性或者事件.
+     */
+    private MessageType type;
+
+    /**
+     * 要单独获取哪些字段信息
+     */
+    private List<Property> properties;
+
+    /**
+     * 执行条件
+     */
+    private List<Condition> conditions;
+
+    /**
+     * 警告发生后的操作,指向其他规则节点,如发送消息通知.
+     */
+    private List<Operation> operations;
+
+
+    public List<String> getPlainColumns() {
+        Stream<String> conditionColumns = conditions
+            .stream()
+            .map(condition -> condition.getColumn(type));
+
+        if (CollectionUtils.isEmpty(properties)) {
+            return conditionColumns.collect(Collectors.toList());
+        }
+        return Stream.concat(conditionColumns, properties
+            .stream()
+            .map(property -> type.getPropertyPrefix() + property.toString()))
+            .collect(Collectors.toList());
+    }
+
+    @Getter
+    @Setter
+    public static class Operation implements Serializable {
+
+        /**
+         * 执行器
+         *
+         * @see RuleNodeModel#getExecutor()
+         * @see ExecutableRuleNodeFactoryStrategy#getSupportType()
+         */
+        private String executor;
+
+        /**
+         * 执行器配置
+         *
+         * @see RuleNodeModel#getConfiguration()
+         * @see RuleNodeConfiguration
+         */
+        private Map<String, Object> configuration;
+    }
+
+    @AllArgsConstructor
+    @Getter
+    public enum MessageType {
+        //上线
+        online("/device/%s/%s/online", "this.") {
+            @Override
+            public String getTopic(String productId, String deviceId, String key) {
+                return String.format(getTopicTemplate(), productId, StringUtils.isEmpty(deviceId) ? "*" : deviceId);
+            }
+        },
+        //离线
+        offline("/device/%s/%s/offline", "this.") {
+            @Override
+            public String getTopic(String productId, String deviceId, String key) {
+                return String.format(getTopicTemplate(), productId, StringUtils.isEmpty(deviceId) ? "*" : deviceId);
+            }
+        },
+        //属性
+        properties("/device/%s/%s/message/property/**", "this.properties.") {
+            @Override
+            public String getTopic(String productId, String deviceId, String key) {
+                return String.format(getTopicTemplate(), productId, StringUtils.isEmpty(deviceId) ? "*" : deviceId);
+            }
+        },
+        //事件
+        event("/device/%s/%s/message/event/%s", "this.data.") {
+            @Override
+            public String getTopic(String productId, String deviceId, String property) {
+                return String.format(getTopicTemplate(), productId, StringUtils.isEmpty(deviceId) ? "*" : deviceId, property);
+            }
+        };
+
+        private String topicTemplate;
+
+        private String propertyPrefix;
+
+        public abstract String getTopic(String productId, String deviceId, String key);
+    }
+
+    @Getter
+    @Setter
+    public static class Condition implements Serializable {
+
+        //物模型属性或者事件的标识 如: fire_alarm
+        private String modelId;
+
+        //过滤条件key 如: temperature.value = ?
+        private String key;
+
+        //过滤条件值
+        private String value;
+
+        //操作符, 等于,大于,小于....
+        private Operator operator = Operator.eq;
+
+        public String getColumn(MessageType type) {
+            return type.getPropertyPrefix() + (key.trim()) + " " + (key.trim());
+        }
+
+        public String createExpression(MessageType type) {
+            return type.getPropertyPrefix() + (key.trim()) + " " + operator.symbol + " ? ";
+        }
+    }
+
+
+    @AllArgsConstructor
+    @Getter
+    public enum Operator {
+        eq("="),
+        not("!="),
+        gt(">"),
+        lt("<"),
+        gte(">="),
+        lte("<="),
+        like("like");
+        private String symbol;
+
+    }
+
+    @Getter
+    @Setter
+    public static class Property implements Serializable {
+        private String property;
+
+        private String alias;
+
+        @Override
+        public String toString() {
+            return property.concat(" ").concat(StringUtils.hasText(alias) ? alias : property);
+        }
+    }
+}

+ 173 - 0
jetlinks-manager/rule-engine-manager/src/main/java/org/jetlinks/community/rule/engine/device/DeviceAlarmRuleNode.java

@@ -0,0 +1,173 @@
+package org.jetlinks.community.rule.engine.device;
+
+import lombok.AllArgsConstructor;
+import lombok.Getter;
+import lombok.Setter;
+import lombok.extern.slf4j.Slf4j;
+import org.apache.commons.collections.CollectionUtils;
+import org.jetlinks.core.message.DeviceMessage;
+import org.jetlinks.community.gateway.DeviceMessageUtils;
+import org.jetlinks.community.gateway.MessageGateway;
+import org.jetlinks.community.gateway.Subscription;
+import org.jetlinks.reactor.ql.ReactorQL;
+import org.jetlinks.reactor.ql.ReactorQLContext;
+import org.jetlinks.reactor.ql.ReactorQLRecord;
+import org.jetlinks.rule.engine.api.RuleData;
+import org.jetlinks.rule.engine.api.executor.ExecutionContext;
+import org.jetlinks.rule.engine.api.model.NodeType;
+import org.jetlinks.rule.engine.executor.CommonExecutableRuleNodeFactoryStrategy;
+import org.jetlinks.rule.engine.executor.node.RuleNodeConfig;
+import org.reactivestreams.Publisher;
+import org.springframework.stereotype.Component;
+import org.springframework.util.StringUtils;
+import reactor.core.publisher.Flux;
+import reactor.core.publisher.Mono;
+
+import java.util.*;
+import java.util.function.Function;
+import java.util.stream.Collectors;
+
+@Slf4j(topic = "system.rule.engine.device.alarm")
+@Component
+@AllArgsConstructor
+public class DeviceAlarmRuleNode extends CommonExecutableRuleNodeFactoryStrategy<DeviceAlarmRuleNode.Config> {
+
+    private MessageGateway messageGateway;
+
+    @Override
+    public Function<RuleData, ? extends Publisher<?>> createExecutor(ExecutionContext context, DeviceAlarmRuleNode.Config config) {
+
+        return Mono::just;
+    }
+
+    @Override
+    protected void onStarted(ExecutionContext context, DeviceAlarmRuleNode.Config config) {
+        context.onStop(
+            config.doSubscribe(messageGateway)
+                .flatMap(result -> {
+                    //输出到下一节点
+                    return context.getOutput()
+                        .write(Mono.just(RuleData.create(result)))
+                        .then();
+                })
+                .onErrorResume(err -> context.onError(RuleData.create(err.getMessage()), err))
+                .subscribe()::dispose
+        );
+    }
+
+    @Override
+    public String getSupportType() {
+        return "device_alarm";
+    }
+
+
+    @Getter
+    @Setter
+    public static class Config implements RuleNodeConfig {
+        static List<String> default_columns = Arrays.asList(
+            "timestamp", "deviceId"
+        );
+
+        private DeviceAlarmRule rule;
+
+        @Override
+        public void validate() {
+            if (CollectionUtils.isEmpty(rule.getConditions())) {
+                throw new IllegalArgumentException("预警条件不能为空");
+            }
+        }
+
+        private ReactorQL createQL() {
+            List<String> columns = new ArrayList<>(default_columns);
+            List<String> wheres = new ArrayList<>();
+            columns.addAll(rule.getPlainColumns());
+
+            for (DeviceAlarmRule.Condition condition : rule.getConditions()) {
+                wheres.add(condition.createExpression(rule.getType()));
+            }
+
+            String sql = "select " + String.join(",", columns) +
+                " from msg where " + String.join(" or ", wheres);
+
+            log.debug("create device alarm sql : {}", sql);
+            return ReactorQL.builder().sql(sql).build();
+        }
+
+        public Flux<Map<String, Object>> doSubscribe(MessageGateway gateway) {
+            Set<String> topics = new HashSet<>();
+
+            List<Object> binds = new ArrayList<>();
+
+            for (DeviceAlarmRule.Condition condition : rule.getConditions()) {
+                String topic = rule.getType().getTopic(rule.getProductId(), rule.getDeviceId(), condition.getModelId());
+                topics.add(topic);
+                binds.add(condition.getValue());
+            }
+            List<Subscription> subscriptions = topics.stream().map(Subscription::new).collect(Collectors.toList());
+
+            ReactorQLContext context = ReactorQLContext
+                .ofDatasource(ignore ->
+                    gateway
+                        .subscribe(subscriptions, "device_alarm:" + rule.getId(), false)
+                        .flatMap(msg -> Mono.justOrEmpty(DeviceMessageUtils.convert(msg).map(DeviceMessage::toJson)))
+                        .doOnNext(json -> {
+                            if (StringUtils.hasText(rule.getDeviceName())) {
+                                json.putIfAbsent("deviceName", rule.getDeviceName());
+                            }
+                            if (StringUtils.hasText(rule.getProductName())) {
+                                json.putIfAbsent("productName", rule.getProductName());
+                            }
+                            json.put("productId", rule.getProductId());
+                            json.put("alarmId", rule.getId());
+                            json.put("alarmName", rule.getName());
+                        })
+                );
+
+            binds.forEach(context::bind);
+            return createQL()
+                .start(context)
+                .map(ReactorQLRecord::asMap)
+                .flatMap(map -> {
+                    map.put("productId", rule.getProductId());
+                    map.put("alarmId", rule.getId());
+                    map.put("alarmName", rule.getName());
+                    if (StringUtils.hasText(rule.getDeviceName())) {
+                        map.putIfAbsent("deviceName", rule.getDeviceName());
+                    }
+                    if (StringUtils.hasText(rule.getProductName())) {
+                        map.putIfAbsent("productName", rule.getProductName());
+                    }
+                    if (StringUtils.hasText(rule.getDeviceId())) {
+                        map.putIfAbsent("deviceId", rule.getDeviceId());
+                    }
+                    if (!map.containsKey("deviceName")) {
+                        map.putIfAbsent("deviceName", map.get("deviceId"));
+                    }
+                    if (!map.containsKey("productName")) {
+                        map.putIfAbsent("productName", map.get("productId"));
+                    }
+                    if (log.isDebugEnabled()) {
+                        log.debug("发生设备预警:{}", map);
+                    }
+                    // 推送警告到消息网关中
+                    // /rule-engine/device/alarm/{productId}/{deviceId}/{ruleId}
+                    return gateway
+                        .publish(String.format(
+                            "/rule-engine/device/alarm/%s/%s/%s",
+                            rule.getProductId(), map.get("deviceId"), rule.getId()
+                        ), map)
+                        .then(Mono.just(map));
+                });
+        }
+
+        @Override
+        public NodeType getNodeType() {
+            return NodeType.MAP;
+        }
+
+        @Override
+        public void setNodeType(NodeType nodeType) {
+
+        }
+    }
+}

+ 76 - 0
jetlinks-manager/rule-engine-manager/src/main/java/org/jetlinks/community/rule/engine/entity/DeviceAlarmEntity.java

@@ -0,0 +1,76 @@
+package org.jetlinks.community.rule.engine.entity;
+
+import com.alibaba.fastjson.JSON;
+import lombok.Getter;
+import lombok.Setter;
+import org.hswebframework.ezorm.rdb.mapping.annotation.ColumnType;
+import org.hswebframework.ezorm.rdb.mapping.annotation.DefaultValue;
+import org.hswebframework.ezorm.rdb.mapping.annotation.EnumCodec;
+import org.hswebframework.ezorm.rdb.mapping.annotation.JsonCodec;
+import org.hswebframework.web.api.crud.entity.GenericEntity;
+import org.jetlinks.community.rule.engine.device.DeviceAlarmRule;
+import org.jetlinks.community.rule.engine.enums.AlarmState;
+import org.jetlinks.community.rule.engine.model.DeviceAlarmModelParser;
+
+import javax.persistence.Column;
+import javax.persistence.Index;
+import javax.persistence.Table;
+import javax.validation.constraints.NotBlank;
+import java.sql.JDBCType;
+
+@Getter
+@Setter
+@Table(name = "rule_dev_alarm", indexes = {
+    @Index(name = "idx_rda_tid", columnList = "target,target_id")
+})
+public class DeviceAlarmEntity extends GenericEntity<String> {
+
+    //device or product
+    @Column(length = 32, nullable = false, updatable = false)
+    @NotBlank(message = "[target]不能为空")
+    private String target;
+
+    //deviceId or productId
+    @Column(length = 32, nullable = false, updatable = false)
+    @NotBlank(message = "[targetId]不能为空")
+    private String targetId;
+
+    //名称
+    @Column
+    private String name;
+
+    //说明
+    @Column
+    private String description;
+
+    //规则
+    @Column(nullable = false)
+    @JsonCodec
+    @ColumnType(jdbcType = JDBCType.CLOB, javaType = String.class)
+    private DeviceAlarmRule alarmRule;
+
+    @Column
+    @EnumCodec
+    @ColumnType(javaType = String.class)
+    @DefaultValue("stopped")
+    private AlarmState state;
+
+    public RuleInstanceEntity toRuleInstance() {
+        RuleInstanceEntity instanceEntity = new RuleInstanceEntity();
+        if (alarmRule == null) {
+            throw new IllegalArgumentException("未设置告警规则");
+        }
+        alarmRule.setId(getId());
+        alarmRule.setName(name);
+
+        instanceEntity.setModelVersion(1);
+        instanceEntity.setId(getId());
+        instanceEntity.setModelId(getId());
+        instanceEntity.setDescription(description);
+        instanceEntity.setModelType(DeviceAlarmModelParser.format);
+        instanceEntity.setName("设备告警:" + name);
+        instanceEntity.setModelMeta(JSON.toJSONString(this));
+        instanceEntity.setCreateTimeNow();
+        return instanceEntity;
+    }
+}

+ 60 - 0
jetlinks-manager/rule-engine-manager/src/main/java/org/jetlinks/community/rule/engine/entity/DeviceAlarmHistoryEntity.java

@@ -0,0 +1,60 @@
+package org.jetlinks.community.rule.engine.entity;
+
+import lombok.Getter;
+import lombok.Setter;
+import org.hswebframework.ezorm.rdb.mapping.annotation.ColumnType;
+import org.hswebframework.ezorm.rdb.mapping.annotation.DefaultValue;
+import org.hswebframework.ezorm.rdb.mapping.annotation.JsonCodec;
+import org.hswebframework.web.api.crud.entity.GenericEntity;
+import org.hswebframework.web.crud.generator.Generators;
+
+import javax.persistence.Column;
+import javax.persistence.Index;
+import javax.persistence.Table;
+import java.sql.JDBCType;
+import java.util.Date;
+import java.util.Map;
+
+/**
+ * 设备告警记录
+ *
+ * @author zhouhao
+ * @since 1.1
+ */
+@Table(name = "rule_dev_alarm_history", indexes = {
+    @Index(name = "idx_rahi_product_id", columnList = "product_id"),
+    @Index(name = "idx_rahi_device_id", columnList = "device_id"),
+    @Index(name = "idx_rahi_alarm_id", columnList = "alarm_id")
+})
+@Getter
+@Setter
+public class DeviceAlarmHistoryEntity extends GenericEntity<String> {
+
+    @Column(length = 32, nullable = false, updatable = false)
+    private String productId;
+
+    @Column(updatable = false)
+    private String productName;
+
+    @Column(length = 32, nullable = false, updatable = false)
+    private String deviceId;
+
+    @Column(nullable = false, updatable = false)
+    private String deviceName;
+
+    @Column(length = 32, nullable = false, updatable = false)
+    private String alarmId;
+
+    @Column(nullable = false)
+    private String alarmName;
+
+    @Column
+    @DefaultValue(generator = Generators.CURRENT_TIME)
+    private Date alarmTime;
+
+    @Column
+    @ColumnType(javaType = String.class,jdbcType = JDBCType.CLOB)
+    @JsonCodec
+    private Map<String,Object> alarmData;
+
+}

+ 20 - 0
jetlinks-manager/rule-engine-manager/src/main/java/org/jetlinks/community/rule/engine/enums/AlarmState.java

@@ -0,0 +1,20 @@
+package org.jetlinks.community.rule.engine.enums;
+
+import lombok.AllArgsConstructor;
+import lombok.Getter;
+import org.hswebframework.web.dict.EnumDict;
+
+@AllArgsConstructor
+@Getter
+public enum AlarmState implements EnumDict<String> {
+
+    running("运行中"),
+    stopped("已停止");
+
+    private String text;
+
+    @Override
+    public String getValue() {
+        return name();
+    }
+}

+ 62 - 0
jetlinks-manager/rule-engine-manager/src/main/java/org/jetlinks/community/rule/engine/model/DeviceAlarmModelParser.java

@@ -0,0 +1,62 @@
+package org.jetlinks.community.rule.engine.model;
+
+import com.alibaba.fastjson.JSON;
+import org.apache.commons.collections.CollectionUtils;
+import org.hswebframework.web.bean.FastBeanCopier;
+import org.jetlinks.community.rule.engine.device.DeviceAlarmRule;
+import org.jetlinks.community.rule.engine.entity.DeviceAlarmEntity;
+import org.jetlinks.rule.engine.api.cluster.RunMode;
+import org.jetlinks.rule.engine.api.model.RuleLink;
+import org.jetlinks.rule.engine.api.model.RuleModel;
+import org.jetlinks.rule.engine.api.model.RuleNodeModel;
+import org.jetlinks.rule.engine.model.RuleModelParserStrategy;
+import org.springframework.stereotype.Component;
+
+import java.util.Collections;
+
+@Component
+public class DeviceAlarmModelParser implements RuleModelParserStrategy {
+
+    public static String format = "device_alarm";
+    @Override
+    public String getFormat() {
+        return format;
+    }
+
+    @Override
+    public RuleModel parse(String modelDefineString) {
+        DeviceAlarmEntity rule = FastBeanCopier.copy(JSON.parseObject(modelDefineString),DeviceAlarmEntity::new);
+
+        RuleModel model = new RuleModel();
+        model.setId("device_alarm:".concat(rule.getId()));
+        model.setName(rule.getName());
+        model.setRunMode(RunMode.CLUSTER);
+
+        RuleNodeModel conditionNode = new RuleNodeModel();
+        conditionNode.setId("conditions");
+        conditionNode.setName("预警条件");
+        conditionNode.setExecutor("device_alarm");
+        conditionNode.setConfiguration(Collections.singletonMap("rule",rule.getAlarmRule()));
+        model.getNodes().add(conditionNode);
+        if (CollectionUtils.isNotEmpty(rule.getAlarmRule().getOperations())) {
+            int index = 0;
+            for (DeviceAlarmRule.Operation operation : rule.getAlarmRule().getOperations()) {
+                RuleNodeModel action = new RuleNodeModel();
+                action.setId("device_alarm_action:" + index);
+                action.setName("执行动作:" + index);
+                action.setExecutor(operation.getExecutor());
+                action.setConfiguration(operation.getConfiguration());
+
+                RuleLink link=new RuleLink();
+                link.setId(action.getId().concat(":").concat(conditionNode.getId()));
+                link.setName("执行动作:"+index);
+                link.setSource(conditionNode);
+                link.setTarget(action);
+                model.getNodes().add(action);
+                action.getInputs().add(link);
+                conditionNode.getOutputs().add(link);
+            }
+        }
+        return model;
+    }
+}

+ 43 - 0
jetlinks-manager/rule-engine-manager/src/main/java/org/jetlinks/community/rule/engine/service/DeviceAlarmHistoryService.java

@@ -0,0 +1,43 @@
+package org.jetlinks.community.rule.engine.service;
+
+import org.hswebframework.web.bean.FastBeanCopier;
+import org.hswebframework.web.crud.service.GenericReactiveCrudService;
+import org.jetlinks.core.utils.FluxUtils;
+import org.jetlinks.community.gateway.annotation.Subscribe;
+import org.jetlinks.community.rule.engine.entity.DeviceAlarmHistoryEntity;
+import org.springframework.stereotype.Service;
+import reactor.core.publisher.EmitterProcessor;
+import reactor.core.publisher.FluxSink;
+import reactor.core.publisher.Mono;
+
+import javax.annotation.PostConstruct;
+import java.time.Duration;
+import java.util.Date;
+import java.util.Map;
+
+@Service
+public class DeviceAlarmHistoryService extends GenericReactiveCrudService<DeviceAlarmHistoryEntity, String> {
+
+
+    EmitterProcessor<DeviceAlarmHistoryEntity> processor = EmitterProcessor.create(false);
+    FluxSink<DeviceAlarmHistoryEntity> sink = processor.sink();
+
+    @PostConstruct
+    public void init() {
+        FluxUtils.bufferRate(processor, 500, 200, Duration.ofSeconds(2))
+            .flatMap(list -> this.insertBatch(Mono.just(list)))
+            .subscribe();
+    }
+
+    @Subscribe("/rule-engine/device/alarm/**")
+    public Mono<Void> saveAlarm(Map<String, Object> message) {
+        DeviceAlarmHistoryEntity entity = FastBeanCopier.copy(message, DeviceAlarmHistoryEntity::new);
+        if (message.containsKey("timestamp")) {
+            entity.setAlarmTime(new Date((Long) message.get("timestamp")));
+        }
+        entity.setAlarmData(message);
+        return Mono
+            .fromRunnable(() -> sink.next(entity));
+    }
+
+}

+ 39 - 0
jetlinks-manager/rule-engine-manager/src/main/java/org/jetlinks/community/rule/engine/service/DeviceAlarmService.java

@@ -0,0 +1,39 @@
+package org.jetlinks.community.rule.engine.service;
+
+import org.hswebframework.web.crud.service.GenericReactiveCrudService;
+import org.jetlinks.community.rule.engine.entity.DeviceAlarmEntity;
+import org.jetlinks.community.rule.engine.enums.AlarmState;
+import org.springframework.stereotype.Service;
+import reactor.core.publisher.Mono;
+
+@Service
+public class DeviceAlarmService extends GenericReactiveCrudService<DeviceAlarmEntity, String> {
+
+    private final RuleInstanceService instanceService;
+
+    @SuppressWarnings("all")
+    public DeviceAlarmService(RuleInstanceService instanceService) {
+        this.instanceService = instanceService;
+    }
+
+    public Mono<Void> start(String id) {
+        return findById(id)
+            .flatMap(this::doStart);
+    }
+
+    public Mono<Void> stop(String id) {
+        return instanceService.stop(id);
+    }
+
+    private Mono<Void> doStart(DeviceAlarmEntity entity) {
+        return instanceService
+            .save(Mono.just(entity.toRuleInstance()))
+            .then(instanceService.start(entity.getId()))
+            .then(createUpdate()
+                .set(DeviceAlarmEntity::getState, AlarmState.running)
+                .where(entity::getId).execute())
+            .then();
+    }
+
+
+}

+ 67 - 0
jetlinks-manager/rule-engine-manager/src/main/java/org/jetlinks/community/rule/engine/web/DeviceAlarmController.java

@@ -0,0 +1,67 @@
+package org.jetlinks.community.rule.engine.web;
+
+import org.hswebframework.web.authorization.annotation.Authorize;
+import org.hswebframework.web.authorization.annotation.QueryAction;
+import org.hswebframework.web.authorization.annotation.Resource;
+import org.hswebframework.web.authorization.annotation.SaveAction;
+import org.hswebframework.web.crud.web.reactive.ReactiveServiceCrudController;
+import org.jetlinks.community.rule.engine.entity.DeviceAlarmEntity;
+import org.jetlinks.community.rule.engine.service.DeviceAlarmService;
+import org.springframework.web.bind.annotation.*;
+import reactor.core.publisher.Flux;
+import reactor.core.publisher.Mono;
+
+@RestController
+@RequestMapping(value = "/device/alarm")
+@Resource(id = "device-alarm", name = "设备告警")
+@Authorize
+public class DeviceAlarmController implements ReactiveServiceCrudController<DeviceAlarmEntity, String> {
+
+    private final DeviceAlarmService alarmService;
+
+    public DeviceAlarmController(DeviceAlarmService alarmService) {
+        this.alarmService = alarmService;
+    }
+
+    @Override
+    public DeviceAlarmService getService() {
+        return alarmService;
+    }
+
+    @GetMapping("/{target}/{targetId}")
+    @QueryAction
+    public Flux<DeviceAlarmEntity> getProductAlarms(@PathVariable String target,
+                                                    @PathVariable String targetId) {
+        return alarmService.createQuery()
+            .where(DeviceAlarmEntity::getTarget, target)
+            .and(DeviceAlarmEntity::getTargetId, targetId)
+            .fetch();
+    }
+
+    @PatchMapping("/{target}/{targetId}")
+    @QueryAction
+    public Mono<Void> saveAlarm(@PathVariable String target,
+                                @PathVariable String targetId,
+                                @RequestBody Mono<DeviceAlarmEntity> payload) {
+        return payload
+            .doOnNext(dev -> {
+                dev.setTarget(target);
+                dev.setTargetId(targetId);
+            })
+            .as(alarmService::save)
+            .then();
+    }
+
+    @PostMapping("/{id}/_start")
+    @SaveAction
+    public Mono<Void> startAlarm(@PathVariable String id) {
+        return alarmService.start(id);
+    }
+
+    @PostMapping("/{id}/_stop")
+    @SaveAction
+    public Mono<Void> stopAlarm(@PathVariable String id) {
+        return alarmService.stop(id);
+    }
+
+}

+ 30 - 0
jetlinks-manager/rule-engine-manager/src/main/java/org/jetlinks/community/rule/engine/web/DeviceAlarmHistoryController.java

@@ -0,0 +1,30 @@
+package org.jetlinks.community.rule.engine.web;
+
+import org.hswebframework.web.authorization.annotation.Authorize;
+import org.hswebframework.web.authorization.annotation.Resource;
+import org.hswebframework.web.crud.web.reactive.ReactiveServiceQueryController;
+import org.jetlinks.community.rule.engine.entity.DeviceAlarmHistoryEntity;
+import org.jetlinks.community.rule.engine.service.DeviceAlarmHistoryService;
+import org.springframework.web.bind.annotation.RequestMapping;
+import org.springframework.web.bind.annotation.RestController;
+
+@RestController
+@RequestMapping("/device/alarm/history")
+@Resource(id = "device-alarm", name = "设备告警")
+@Authorize
+public class DeviceAlarmHistoryController implements ReactiveServiceQueryController<DeviceAlarmHistoryEntity, String> {
+
+    private final DeviceAlarmHistoryService historyService;
+
+
+    public DeviceAlarmHistoryController(DeviceAlarmHistoryService historyService) {
+        this.historyService = historyService;
+    }
+
+    @Override
+    public DeviceAlarmHistoryService getService() {
+        return historyService;
+    }
+
+
+}

+ 7 - 0
pom.xml

@@ -27,6 +27,7 @@
         <netty.version>4.1.46.Final</netty.version>
         <elasticsearch.version>6.8.6</elasticsearch.version>
         <reactor.excel.version>1.0-BUILD-SNAPSHOT</reactor.excel.version>
+        <reactor.ql.version>1.0-SNAPSHOT</reactor.ql.version>
     </properties>
 
     <build>
@@ -153,6 +154,12 @@
     <dependencyManagement>
 
         <dependencies>
+            <dependency>
+                <groupId>org.jetlinks</groupId>
+                <artifactId>reactor-ql</artifactId>
+                <version>${reactor.ql.version}</version>
+            </dependency>
+
             <dependency>
                 <groupId>org.hswebframework</groupId>
                 <artifactId>reactor-excel</artifactId>