VOL-3049 initial support changes for alarm notifications

Change-Id: Ic6e17196e6955a669e2bf8fc248bdbdbca1654c5
diff --git a/internal/pkg/onuadaptercore/alarm_manager.go b/internal/pkg/onuadaptercore/alarm_manager.go
new file mode 100644
index 0000000..dcf3a18
--- /dev/null
+++ b/internal/pkg/onuadaptercore/alarm_manager.go
@@ -0,0 +1,302 @@
+/*
+ * Copyright 2021-present Open Networking Foundation
+ *
+ * Licensed under the Apache License, Version 2.0 (the "License");
+ * you may not use this file except in compliance with the License.
+ * You may obtain a copy of the License at
+ *
+ * http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+
+//Package adaptercoreonu provides the utility for onu devices, flows and statistics
+package adaptercoreonu
+
+import (
+	"context"
+	"errors"
+	"fmt"
+	"sync"
+	"time"
+
+	"github.com/opencord/omci-lib-go"
+	me "github.com/opencord/omci-lib-go/generated"
+	"github.com/opencord/voltha-lib-go/v4/pkg/events/eventif"
+	"github.com/opencord/voltha-lib-go/v4/pkg/log"
+	"github.com/opencord/voltha-protos/v4/go/voltha"
+)
+
+const (
+	circuitPackClassID                             = me.CircuitPackClassID
+	physicalPathTerminationPointEthernetUniClassID = me.PhysicalPathTerminationPointEthernetUniClassID
+	onuGClassID                                    = me.OnuGClassID
+	aniGClassID                                    = me.AniGClassID
+)
+
+type alarmInfo struct {
+	classID    me.ClassID
+	instanceID uint16
+	alarmNo    uint8
+}
+
+type alarms map[alarmInfo]struct{}
+
+type onuAlarmManager struct {
+	pDeviceHandler             *deviceHandler
+	eventProxy                 eventif.EventProxy
+	stopProcessingOmciMessages chan bool
+	eventChannel               chan Message
+	onuAlarmManagerLock        sync.RWMutex
+	processMessage             bool
+	activeAlarms               alarms
+}
+
+type onuDevice struct {
+	classID me.ClassID
+	alarmno uint8
+}
+type onuDeviceEvent struct {
+	EventName        string
+	EventCategory    eventif.EventCategory
+	EventSubCategory eventif.EventSubCategory
+	EventDescription string
+}
+
+func newAlarmManager(ctx context.Context, dh *deviceHandler) *onuAlarmManager {
+	var alarmManager onuAlarmManager
+	logger.Debugw(ctx, "init-alarmManager", log.Fields{"device-id": dh.deviceID})
+	alarmManager.pDeviceHandler = dh
+	alarmManager.eventProxy = dh.EventProxy // Or event proxy should be on cluster address ??
+	alarmManager.eventChannel = make(chan Message)
+	alarmManager.stopProcessingOmciMessages = make(chan bool)
+	alarmManager.processMessage = false
+	alarmManager.activeAlarms = make(map[alarmInfo]struct{})
+	return &alarmManager
+}
+
+// getDeviceEventData returns the event data for a device
+func (am *onuAlarmManager) getDeviceEventData(ctx context.Context, classID me.ClassID, alarmNo uint8) (onuDeviceEvent, error) {
+	onuEventsList := map[onuDevice]onuDeviceEvent{
+		{classID: circuitPackClassID, alarmno: 0}: {EventName: "ONU_EQUIPMENT",
+			EventCategory: voltha.EventCategory_EQUIPMENT, EventSubCategory: voltha.EventSubCategory_ONU, EventDescription: "onu equipment"},
+		{classID: circuitPackClassID, alarmno: 2}: {EventName: "ONU_SELF_TEST_FAIL",
+			EventCategory: voltha.EventCategory_EQUIPMENT, EventSubCategory: voltha.EventSubCategory_ONU, EventDescription: "onu self-test failure"},
+		{classID: circuitPackClassID, alarmno: 3}: {EventName: "ONU_LASER_EOL",
+			EventCategory: voltha.EventCategory_EQUIPMENT, EventSubCategory: voltha.EventSubCategory_ONU, EventDescription: "onu laser EOL"},
+		{classID: circuitPackClassID, alarmno: 4}: {EventName: "ONU_TEMP_YELLOW",
+			EventCategory: voltha.EventCategory_ENVIRONMENT, EventSubCategory: voltha.EventSubCategory_ONU, EventDescription: "onu temperature yellow"},
+		{classID: circuitPackClassID, alarmno: 5}: {EventName: "ONU_TEMP_RED",
+			EventCategory: voltha.EventCategory_ENVIRONMENT, EventSubCategory: voltha.EventSubCategory_ONU, EventDescription: "onu temperature red"},
+		{classID: physicalPathTerminationPointEthernetUniClassID, alarmno: 0}: {EventName: "ONU_Ethernet_UNI", EventCategory: voltha.EventCategory_EQUIPMENT,
+			EventSubCategory: voltha.EventSubCategory_ONU, EventDescription: "LAN Loss Of Signal"},
+		{classID: onuGClassID, alarmno: 0}: {EventName: "ONU_EQUIPMENT",
+			EventCategory: voltha.EventCategory_EQUIPMENT, EventSubCategory: voltha.EventSubCategory_ONU, EventDescription: "onu equipment"},
+		{classID: onuGClassID, alarmno: 6}: {EventName: "ONU_SELF_TEST_FAIL",
+			EventCategory: voltha.EventCategory_EQUIPMENT, EventSubCategory: voltha.EventSubCategory_ONU, EventDescription: "onu self-test failure"},
+		{classID: onuGClassID, alarmno: 7}: {EventName: "ONU_DYING_GASP",
+			EventCategory: voltha.EventCategory_EQUIPMENT, EventSubCategory: voltha.EventSubCategory_ONU, EventDescription: "onu DYING_GASP"},
+		{classID: onuGClassID, alarmno: 8}: {EventName: "ONU_TEMP_YELLOW",
+			EventCategory: voltha.EventCategory_ENVIRONMENT, EventSubCategory: voltha.EventSubCategory_ONU, EventDescription: "onu temperature yellow"},
+		{classID: onuGClassID, alarmno: 9}: {EventName: "ONU_TEMP_RED",
+			EventCategory: voltha.EventCategory_ENVIRONMENT, EventSubCategory: voltha.EventSubCategory_ONU, EventDescription: "onu temperature red"},
+		{classID: onuGClassID, alarmno: 10}: {EventName: "ONU_VOLTAGE_YELLOW",
+			EventCategory: voltha.EventCategory_ENVIRONMENT, EventSubCategory: voltha.EventSubCategory_ONU, EventDescription: "onu voltage yellow"},
+		{classID: onuGClassID, alarmno: 11}: {EventName: "ONU_VOLTAGE_RED",
+			EventCategory: voltha.EventCategory_ENVIRONMENT, EventSubCategory: voltha.EventSubCategory_ONU, EventDescription: "onu voltage red"},
+		{classID: aniGClassID, alarmno: 0}: {EventName: "ONU_LOW_RX_OPTICAL",
+			EventCategory: voltha.EventCategory_COMMUNICATION, EventSubCategory: voltha.EventSubCategory_ONU, EventDescription: "onu low rx optical power"},
+		{classID: aniGClassID, alarmno: 1}: {EventName: "ONU_HIGH_RX_OPTICAL",
+			EventCategory: voltha.EventCategory_COMMUNICATION, EventSubCategory: voltha.EventSubCategory_ONU, EventDescription: "onu high rx optical power"},
+		{classID: aniGClassID, alarmno: 4}: {EventName: "ONU_LOW_TX_OPTICAL",
+			EventCategory: voltha.EventCategory_COMMUNICATION, EventSubCategory: voltha.EventSubCategory_ONU, EventDescription: "onu low tx optical power"},
+		{classID: aniGClassID, alarmno: 5}: {EventName: "ONU_HIGH_TX_OPTICAL",
+			EventCategory: voltha.EventCategory_COMMUNICATION, EventSubCategory: voltha.EventSubCategory_ONU, EventDescription: "onu high tx optical power"},
+		{classID: aniGClassID, alarmno: 6}: {EventName: "ONU_LASER_BIAS_CURRENT",
+			EventCategory: voltha.EventCategory_EQUIPMENT, EventSubCategory: voltha.EventSubCategory_ONU, EventDescription: "onu laser bias current"},
+	}
+	if onuEventDetails, ok := onuEventsList[onuDevice{classID: classID, alarmno: alarmNo}]; ok {
+		return onuEventDetails, nil
+	}
+	return onuDeviceEvent{}, errors.New("onu Event Detail not found")
+
+}
+
+func (am *onuAlarmManager) startOMCIAlarmMessageProcessing(ctx context.Context) {
+	am.onuAlarmManagerLock.Lock()
+	am.processMessage = true
+	am.onuAlarmManagerLock.Unlock()
+	if stop := <-am.stopProcessingOmciMessages; stop {
+		am.onuAlarmManagerLock.Lock()
+		am.processMessage = false
+		am.onuAlarmManagerLock.Unlock()
+	}
+}
+
+func (am *onuAlarmManager) handleOmciAlarmNotificationMessage(ctx context.Context, msg OmciMessage) {
+	logger.Debugw(ctx, "OMCI Alarm Notification Msg", log.Fields{"device-id": am.pDeviceHandler.deviceID,
+		"msgType": msg.OmciMsg.MessageType})
+	am.onuAlarmManagerLock.Lock()
+	defer am.onuAlarmManagerLock.Unlock()
+
+	if am.processMessage {
+		msgLayer := (*msg.OmciPacket).Layer(omci.LayerTypeAlarmNotification)
+		if msgLayer == nil {
+			logger.Errorw(ctx, "Omci Msg layer could not be detected for Alarm Notification",
+				log.Fields{"device-id": am.pDeviceHandler.deviceID})
+			return
+		}
+		msgObj, msgOk := msgLayer.(*omci.AlarmNotificationMsg)
+		if !msgOk {
+			logger.Errorw(ctx, "Omci Msg layer could not be assigned for Alarm Notification",
+				log.Fields{"device-id": am.pDeviceHandler.deviceID})
+			return
+		}
+		//Alarm Notification decoding at omci lib validates that the me class ID supports the
+		// alarm notifications.
+		logger.Debugw(ctx, "Alarm Notification Data", log.Fields{"device-id": am.pDeviceHandler.deviceID, "data-fields": msgObj})
+		if err := am.processAlarmData(ctx, msgObj); err != nil {
+			logger.Errorw(ctx, "unable-to-process-alarm-notification", log.Fields{"device-id": am.pDeviceHandler.deviceID})
+		}
+
+	} else {
+		logger.Warnw(ctx, "ignoring-alarm-notification-received-for-me-as-channel-for-processing-is-closed",
+			log.Fields{"device-id": am.pDeviceHandler.deviceID})
+	}
+}
+
+func (am *onuAlarmManager) processAlarmData(ctx context.Context, msg *omci.AlarmNotificationMsg) error {
+	classID := msg.EntityClass
+	sequenceNo := msg.AlarmSequenceNumber
+	meInstance := msg.EntityInstance
+	alarmBitmap := msg.AlarmBitmap
+	logger.Debugw(ctx, "processing-alarm-data", log.Fields{"class-id": classID, "instance-id": meInstance,
+		"alarmBitMap": alarmBitmap, "sequence-no": sequenceNo})
+	/*
+		if sequenceNo > 0 {
+			// TODO Need Auditing if sequence no does not matches, after incrementing the last sequence no by 1
+		}
+	*/
+
+	entity, omciErr := me.LoadManagedEntityDefinition(classID,
+		me.ParamData{EntityID: meInstance})
+	if omciErr.StatusCode() != me.Success {
+		//log error and return
+		logger.Error(ctx, "unable-to-get-managed-entity", log.Fields{"class-id": classID, "instance-id": meInstance})
+		return fmt.Errorf("unable-to-get-managed-entity-class-%d-instance-%d", classID, meInstance)
+	}
+	meAlarmMap := entity.GetAlarmMap()
+	if meAlarmMap == nil {
+		logger.Error(ctx, "unable-to-get-managed-entity-alarm-map", log.Fields{"class-id": classID, "instance-id": meInstance})
+		return fmt.Errorf("unable-to-get-managed-entity-alarm-map-%d-instance-%d", classID, meInstance)
+	}
+	// Loop over the supported alarm list for this me
+	for alarmNo := range meAlarmMap {
+		// Check if alarmNo was previously active in the alarms, if yes clear it and remove it from active alarms
+		_, exists := am.activeAlarms[alarmInfo{
+			classID:    classID,
+			instanceID: meInstance,
+			alarmNo:    alarmNo,
+		}]
+		if exists {
+			// Clear this alarm if It is cleared now, in that case IsAlarmClear would return true
+			cleared, err := msg.IsAlarmClear(alarmNo)
+			if err != nil {
+				logger.Warnw(ctx, "unable-to-find-out-alarm-is-cleared", log.Fields{"class-id": classID,
+					"instance-id": meInstance, "alarm-no": alarmNo})
+				return err
+			}
+			if cleared {
+				// Clear this alarm.
+				am.clearAlarm(ctx, classID, meInstance, alarmNo)
+			}
+		} else {
+			// If alarm entry was not present in the list of active alarms, we need to see if this alarm is now active
+			// or not, if yes then raise it.
+			raised, err := msg.IsAlarmActive(alarmNo)
+			if err != nil {
+				logger.Warnw(ctx, "unable-to-find-out-alarm-is-raised", log.Fields{"class-id": classID,
+					"instance-id": meInstance, "alarm-no": alarmNo})
+				return err
+			}
+			if raised {
+				am.raiseAlarm(ctx, classID, meInstance, alarmNo)
+			}
+		}
+	}
+	return nil
+}
+
+func (am *onuAlarmManager) raiseAlarm(ctx context.Context, classID me.ClassID, instanceID uint16, alarm uint8) {
+	am.activeAlarms[alarmInfo{
+		classID:    classID,
+		instanceID: instanceID,
+		alarmNo:    alarm,
+	}] = struct{}{}
+	go am.sendAlarm(ctx, classID, instanceID, alarm, true)
+}
+
+func (am *onuAlarmManager) clearAlarm(ctx context.Context, classID me.ClassID, instanceID uint16, alarm uint8) {
+	go am.sendAlarm(ctx, classID, instanceID, alarm, false)
+	delete(am.activeAlarms, alarmInfo{
+		classID:    classID,
+		instanceID: instanceID,
+		alarmNo:    alarm,
+	})
+}
+
+func (am *onuAlarmManager) getIntfIDAlarm(ctx context.Context, classID me.ClassID, instanceID uint16) *uint32 {
+	var intfID *uint32
+	if classID == circuitPackClassID || classID == physicalPathTerminationPointEthernetUniClassID {
+		for _, uniPort := range am.pDeviceHandler.uniEntityMap {
+			if uniPort.entityID == instanceID {
+				intfID = &uniPort.portNo
+				return intfID
+			}
+		}
+	} else if classID == aniGClassID || classID == onuGClassID {
+		intfID = &am.pDeviceHandler.ponPortNumber
+		return intfID
+	} else {
+		logger.Warnw(ctx, "me-not-supported", log.Fields{"class-id": classID, "instance-id": instanceID})
+	}
+	return nil
+}
+
+func (am *onuAlarmManager) sendAlarm(ctx context.Context, classID me.ClassID, instanceID uint16, alarm uint8, raised bool) {
+	context := make(map[string]string)
+	intfID := am.getIntfIDAlarm(ctx, classID, instanceID)
+	onuID := am.pDeviceHandler.deviceID
+	serialNo := am.pDeviceHandler.pOnuOmciDevice.serialNumber
+	if intfID == nil {
+		logger.Warn(ctx, "intf-id-for-alarm-not-found", log.Fields{"alarm-no": alarm, "class-id": classID})
+		return
+	}
+	context["onu-intf-id"] = fmt.Sprintf("%d", *intfID)
+	context["onu-id"] = onuID
+	context["onu-serial-number"] = serialNo
+
+	raisedTimestamp := time.Now().UnixNano()
+	eventDetails, err := am.getDeviceEventData(ctx, classID, alarm)
+	if err != nil {
+		logger.Warn(ctx, "event-details-for-alarm-not-found", log.Fields{"alarm-no": alarm, "class-id": classID})
+		return
+	}
+	suffixDesc := "Raised"
+	if !raised {
+		suffixDesc = "Cleared"
+	}
+	deviceEvent := &voltha.DeviceEvent{
+		ResourceId:      onuID,
+		DeviceEventName: fmt.Sprintf("%s_RAISE_EVENT", eventDetails.EventName),
+		Description: fmt.Sprintf("%s Event - %s - %s", eventDetails.EventDescription, eventDetails.EventName,
+			suffixDesc),
+		Context: context,
+	}
+	_ = am.eventProxy.SendDeviceEvent(ctx, deviceEvent, eventDetails.EventCategory, eventDetails.EventSubCategory,
+		raisedTimestamp)
+}
diff --git a/internal/pkg/onuadaptercore/device_handler.go b/internal/pkg/onuadaptercore/device_handler.go
index 379afa8..5db246b 100644
--- a/internal/pkg/onuadaptercore/device_handler.go
+++ b/internal/pkg/onuadaptercore/device_handler.go
@@ -177,6 +177,7 @@
 	pOnuOmciDevice  *OnuDeviceEntry
 	pOnuTP          *onuUniTechProf
 	pOnuMetricsMgr  *onuMetricsManager
+	pAlarmMgr       *onuAlarmManager
 	exitChannel     chan int
 	lockDevice      sync.RWMutex
 	pOnuIndication  *oop.OnuIndication
@@ -194,6 +195,7 @@
 	collectorIsRunning         bool
 	mutexCollectorFlag         sync.RWMutex
 	stopCollector              chan bool
+	stopAlarmManager           chan bool
 	stopHeartbeatCheck         chan bool
 	uniEntityMap               map[uint32]*onuUniPort
 	lockVlanConfig             sync.Mutex
@@ -219,6 +221,7 @@
 	dh.deviceEntrySet = make(chan bool, 1)
 	dh.collectorIsRunning = false
 	dh.stopCollector = make(chan bool, 2)
+	dh.stopAlarmManager = make(chan bool, 2)
 	dh.stopHeartbeatCheck = make(chan bool, 2)
 	//dh.metrics = pmmetrics.NewPmMetrics(cloned.Id, pmmetrics.Frequency(150), pmmetrics.FrequencyOverride(false), pmmetrics.Grouped(false), pmmetrics.Metrics(pmNames))
 	//TODO initialize the support classes.
@@ -1236,12 +1239,13 @@
 
 //setOnuDeviceEntry sets the ONU device entry within the handler
 func (dh *deviceHandler) setOnuDeviceEntry(
-	apDeviceEntry *OnuDeviceEntry, apOnuTp *onuUniTechProf, apOnuMetricsMgr *onuMetricsManager) {
+	apDeviceEntry *OnuDeviceEntry, apOnuTp *onuUniTechProf, apOnuMetricsMgr *onuMetricsManager, apOnuAlarmMgr *onuAlarmManager) {
 	dh.lockDevice.Lock()
 	defer dh.lockDevice.Unlock()
 	dh.pOnuOmciDevice = apDeviceEntry
 	dh.pOnuTP = apOnuTp
 	dh.pOnuMetricsMgr = apOnuMetricsMgr
+	dh.pAlarmMgr = apOnuAlarmMgr
 }
 
 //addOnuDeviceEntry creates a new ONU device or returns the existing
@@ -1257,8 +1261,9 @@
 		deviceEntry = newOnuDeviceEntry(ctx, dh)
 		onuTechProfProc := newOnuUniTechProf(ctx, dh)
 		onuMetricsMgr := newonuMetricsManager(ctx, dh)
+		onuAlarmManager := newAlarmManager(ctx, dh)
 		//error treatment possible //TODO!!!
-		dh.setOnuDeviceEntry(deviceEntry, onuTechProfProc, onuMetricsMgr)
+		dh.setOnuDeviceEntry(deviceEntry, onuTechProfProc, onuMetricsMgr, onuAlarmManager)
 		// fire deviceEntry ready event to spread to possibly waiting processing
 		dh.deviceEntrySet <- true
 		logger.Debugw(ctx, "onuDeviceEntry-added", log.Fields{"device-id": dh.deviceID})
@@ -1454,6 +1459,8 @@
 		// Start PM collector routine
 		go dh.startCollector(ctx)
 	}
+	go dh.startAlarmManager(ctx)
+
 	return nil
 }
 
@@ -1578,6 +1585,7 @@
 		// Stop collector routine
 		dh.stopCollector <- true
 	}
+	dh.stopAlarmManager <- true
 	return nil
 }
 
@@ -2859,3 +2867,14 @@
 	dh.mutexCollectorFlag.RUnlock()
 	return flagValue
 }
+
+func (dh *deviceHandler) startAlarmManager(ctx context.Context) {
+	logger.Debugf(ctx, "startingAlarmManager")
+
+	// Start routine to process OMCI GET Responses
+	go dh.pAlarmMgr.startOMCIAlarmMessageProcessing(ctx)
+	if stop := <-dh.stopAlarmManager; stop {
+		logger.Debugw(ctx, "stopping-collector-for-onu", log.Fields{"device-id": dh.device.Id})
+		dh.pAlarmMgr.stopProcessingOmciMessages <- true // Stop the OMCI routines if any
+	}
+}
diff --git a/internal/pkg/onuadaptercore/omci_cc.go b/internal/pkg/onuadaptercore/omci_cc.go
index 73b4901..dbdaa93 100644
--- a/internal/pkg/onuadaptercore/omci_cc.go
+++ b/internal/pkg/onuadaptercore/omci_cc.go
@@ -22,7 +22,6 @@
 	"context"
 	"encoding/binary"
 	"encoding/hex"
-	"errors"
 	"fmt"
 	"strconv"
 	"sync"
@@ -200,9 +199,20 @@
 }
 
 // Rx handler for omci messages
-func (oo *omciCC) receiveOnuMessage(ctx context.Context, omciMsg *omci.OMCI) error {
+func (oo *omciCC) receiveOnuMessage(ctx context.Context, omciMsg *omci.OMCI, packet *gp.Packet) error {
 	logger.Debugw(ctx, "rx-onu-autonomous-message", log.Fields{"omciMsgType": omciMsg.MessageType,
 		"payload": hex.EncodeToString(omciMsg.Payload)})
+	switch omciMsg.MessageType {
+	case omci.AlarmNotificationType:
+		data := OmciMessage{
+			OmciMsg:    omciMsg,
+			OmciPacket: packet,
+		}
+		go oo.pBaseDeviceHandler.pAlarmMgr.handleOmciAlarmNotificationMessage(ctx, data)
+		return nil
+	default:
+		return fmt.Errorf("receiveOnuMessageType %s unimplemented", omciMsg.MessageType.String())
+	}
 	/*
 			msgType = rxFrame.fields["message_type"] //assumed OmciOperationsValue
 			rxOnuFrames++
@@ -246,7 +256,6 @@
 				}
 		    }
 	*/
-	return errors.New("receiveOnuMessage unimplemented")
 }
 
 // Rx handler for onu messages
@@ -308,7 +317,7 @@
 		// Not a response
 		logger.Debug(ctx, "RxMsg is no Omci Response Message")
 		if omciMsg.TransactionID == 0 {
-			return oo.receiveOnuMessage(ctx, omciMsg)
+			return oo.receiveOnuMessage(ctx, omciMsg, &packet)
 		}
 		logger.Errorw(ctx, "Unexpected TransCorrId != 0  not accepted for autonomous messages",
 			log.Fields{"msgType": omciMsg.MessageType, "payload": hex.EncodeToString(omciMsg.Payload),
diff --git a/internal/pkg/onuadaptercore/onu_device_entry.go b/internal/pkg/onuadaptercore/onu_device_entry.go
index a9b34cd..81899a1 100644
--- a/internal/pkg/onuadaptercore/onu_device_entry.go
+++ b/internal/pkg/onuadaptercore/onu_device_entry.go
@@ -460,6 +460,7 @@
 	}
 
 	// Alarm Synchronization Database
+
 	//self._alarm_db = None
 	//self._alarm_database_cls = support_classes['alarm-synchronizer']['database']
 	return &onuDeviceEntry