소스 검색

Waap config (#2460)

* revamp wip
Thibault "bui" Koechlin 1 년 전
부모
커밋
4e26e23725

+ 2 - 3
pkg/acquisition/acquisition.go

@@ -37,7 +37,7 @@ import (
 
 type DataSourceUnavailableError struct {
 	Name string
-	Err error
+	Err  error
 }
 
 func (e *DataSourceUnavailableError) Error() string {
@@ -48,7 +48,6 @@ func (e *DataSourceUnavailableError) Unwrap() error {
 	return e.Err
 }
 
-
 // The interface each datasource must implement
 type DataSource interface {
 	GetMetrics() []prometheus.Collector                                 // Returns pointers to metrics that are managed by the module
@@ -76,7 +75,7 @@ var AcquisitionSources = map[string]func() DataSource{
 	"kafka":       func() DataSource { return &kafkaacquisition.KafkaSource{} },
 	"k8s-audit":   func() DataSource { return &k8sauditacquisition.KubernetesAuditSource{} },
 	"s3":          func() DataSource { return &s3acquisition.S3Source{} },
-	"waf":         func() DataSource { return &wafacquisition.WafSource{} },
+	"waf":         func() DataSource { return &wafacquisition.WaapSource{} },
 }
 
 var transformRuntimes = map[string]*vm.Program{}

+ 46 - 0
pkg/acquisition/modules/waf/metrics.go

@@ -0,0 +1,46 @@
+package wafacquisition
+
+import "github.com/prometheus/client_golang/prometheus"
+
+var WafGlobalParsingHistogram = prometheus.NewHistogramVec(
+	prometheus.HistogramOpts{
+		Help:    "Time spent processing a request by the WAF.",
+		Name:    "cs_waf_parsing_time_seconds",
+		Buckets: []float64{0.0005, 0.001, 0.0015, 0.002, 0.0025, 0.003, 0.004, 0.005, 0.0075, 0.01},
+	},
+	[]string{"source"},
+)
+
+var WafInbandParsingHistogram = prometheus.NewHistogramVec(
+	prometheus.HistogramOpts{
+		Help:    "Time spent processing a request by the inband WAF.",
+		Name:    "cs_waf_inband_parsing_time_seconds",
+		Buckets: []float64{0.0005, 0.001, 0.0015, 0.002, 0.0025, 0.003, 0.004, 0.005, 0.0075, 0.01},
+	},
+	[]string{"source"},
+)
+
+var WafOutbandParsingHistogram = prometheus.NewHistogramVec(
+	prometheus.HistogramOpts{
+		Help:    "Time spent processing a request by the WAF.",
+		Name:    "cs_waf_outband_parsing_time_seconds",
+		Buckets: []float64{0.0005, 0.001, 0.0015, 0.002, 0.0025, 0.003, 0.004, 0.005, 0.0075, 0.01},
+	},
+	[]string{"source"},
+)
+
+var WafReqCounter = prometheus.NewCounterVec(
+	prometheus.CounterOpts{
+		Name: "cs_waf_reqs_total",
+		Help: "Total events processed by the WAF.",
+	},
+	[]string{"source"},
+)
+
+var WafRuleHits = prometheus.NewCounterVec(
+	prometheus.CounterOpts{
+		Name: "cs_waf_rule_hits",
+		Help: "Count of triggered rule, by rule_id and type (inband/outofband).",
+	},
+	[]string{"rule_id", "type"},
+)

+ 4 - 17
pkg/acquisition/modules/waf/utils.go

@@ -88,12 +88,9 @@ func LogWaapEvent(evt *types.Event, logger *log.Entry) {
 // 	return nil
 // }
 
-func (r *WafRunner) AccumulateTxToEvent(tx experimental.FullTransaction, kind string, evt *types.Event) error {
+func AccumulateTxToEvent(logger log.Entry, tx experimental.FullTransaction, kind string, evt *types.Event, wr *waf.WaapRuntimeConfig) error {
 
-	//log.Infof("tx addr: %p", tx)
 	if tx.IsInterrupted() {
-		//r.logger.Infof("interrupted() = %t", tx.IsInterrupted())
-		//r.logger.Infof("interrupted.action = %s", tx.Interruption().Action)
 		if evt.Meta == nil {
 			evt.Meta = map[string]string{}
 		}
@@ -112,12 +109,6 @@ func (r *WafRunner) AccumulateTxToEvent(tx experimental.FullTransaction, kind st
 		evt.Waap.Vars = map[string]string{}
 	}
 
-	// collectionsToKeep := []string{
-	// 	"toto",
-	// 	"TX.allowed_methods",
-	// 	"TX.*_score",
-	// }
-
 	tx.Variables().All(func(v variables.RuleVariable, col collection.Collection) bool {
 		for _, variable := range col.FindAll() {
 			key := ""
@@ -129,23 +120,19 @@ func (r *WafRunner) AccumulateTxToEvent(tx experimental.FullTransaction, kind st
 			if variable.Value() == "" {
 				continue
 			}
-			for _, collectionToKeep := range r.VariablesTracking {
+			for _, collectionToKeep := range wr.CompiledVariablesTracking {
 				match := collectionToKeep.MatchString(key)
 				if match {
 					evt.Waap.Vars[key] = variable.Value()
-					r.logger.Debugf("%s.%s = %s", variable.Variable().Name(), variable.Key(), variable.Value())
+					logger.Debugf("%s.%s = %s", variable.Variable().Name(), variable.Key(), variable.Value())
 				} else {
-					r.logger.Debugf("%s.%s != %s (%s) (not kept)", variable.Variable().Name(), variable.Key(), collectionToKeep, variable.Value())
+					logger.Debugf("%s.%s != %s (%s) (not kept)", variable.Variable().Name(), variable.Key(), collectionToKeep, variable.Value())
 				}
 			}
 		}
 		return true
 	})
 
-	//log.Infof("variables: %s", spew.Sdump(tx.Variables()))
-	//log.Infof("tx variables: %+v", tx.Collection(variables.TX))
-	//log.Infof("TX %s", spew.Sdump(tx.MatchedRules()))
-
 	for _, rule := range tx.MatchedRules() {
 		if rule.Message() == "" {
 			continue

+ 274 - 0
pkg/acquisition/modules/waf/waap.go

@@ -0,0 +1,274 @@
+package wafacquisition
+
+import (
+	"context"
+	"encoding/json"
+	"fmt"
+	"net/http"
+
+	corazatypes "github.com/crowdsecurity/coraza/v3/types"
+	"github.com/crowdsecurity/crowdsec/pkg/acquisition/configuration"
+	"github.com/crowdsecurity/crowdsec/pkg/types"
+	"github.com/crowdsecurity/crowdsec/pkg/waf"
+	"github.com/crowdsecurity/go-cs-lib/trace"
+	"github.com/google/uuid"
+	"github.com/pkg/errors"
+	"github.com/prometheus/client_golang/prometheus"
+	log "github.com/sirupsen/logrus"
+	"gopkg.in/tomb.v2"
+	"gopkg.in/yaml.v2"
+)
+
+const (
+	InBand    = "inband"
+	OutOfBand = "outofband"
+)
+
+// configuration structure of the acquis for the Waap
+type WaapSourceConfig struct {
+	ListenAddr                        string `yaml:"listen_addr"`
+	ListenPort                        int    `yaml:"listen_port"`
+	CertFilePath                      string `yaml:"cert_file"`
+	KeyFilePath                       string `yaml:"key_file"`
+	Path                              string `yaml:"path"`
+	Routines                          int    `yaml:"routines"`
+	Debug                             bool   `yaml:"debug"`
+	WaapConfig                        string `yaml:"waap_config"`
+	configuration.DataSourceCommonCfg `yaml:",inline"`
+}
+
+// runtime structure of WaapSourceConfig
+type WaapSource struct {
+	config      WaapSourceConfig
+	logger      *log.Entry
+	mux         *http.ServeMux
+	server      *http.Server
+	addr        string
+	outChan     chan types.Event
+	InChan      chan waf.ParsedRequest
+	WaapRuntime *waf.WaapRuntimeConfig
+
+	WaapRunners []WaapRunner //one for each go-routine
+}
+
+func (wc *WaapSource) UnmarshalConfig(yamlConfig []byte) error {
+	err := yaml.UnmarshalStrict(yamlConfig, wc.config)
+	if err != nil {
+		return errors.Wrap(err, "Cannot parse waf configuration")
+	}
+
+	if wc.config.ListenAddr == "" {
+		return fmt.Errorf("listen_addr cannot be empty")
+	}
+
+	if wc.config.ListenPort == 0 {
+		return fmt.Errorf("listen_port cannot be empty")
+	}
+
+	if wc.config.Path == "" {
+		return fmt.Errorf("path cannot be empty")
+	}
+
+	if wc.config.Path[0] != '/' {
+		wc.config.Path = "/" + wc.config.Path
+	}
+
+	if wc.config.Mode == "" {
+		wc.config.Mode = configuration.TAIL_MODE
+	}
+
+	// always have at least one waf routine
+	if wc.config.Routines == 0 {
+		wc.config.Routines = 1
+	}
+	return nil
+}
+
+func (w *WaapSource) GetMetrics() []prometheus.Collector {
+	return nil
+}
+
+func (w *WaapSource) GetAggregMetrics() []prometheus.Collector {
+	return nil
+}
+
+func logError(error corazatypes.MatchedRule) {
+	msg := error.ErrorLog()
+	log.Infof("[logError][%s]  %s", error.Rule().Severity(), msg)
+}
+
+func (w *WaapSource) Configure(yamlConfig []byte, logger *log.Entry) error {
+	wc := WaapSourceConfig{}
+	err := w.UnmarshalConfig(yamlConfig)
+	if err != nil {
+		return errors.Wrap(err, "unable to parse waf configuration")
+	}
+	w.logger = logger
+	w.config = wc
+
+	w.logger.Tracef("WAF configuration: %+v", w.config)
+
+	w.addr = fmt.Sprintf("%s:%d", wc.ListenAddr, wc.ListenPort)
+
+	w.mux = http.NewServeMux()
+
+	w.server = &http.Server{
+		Addr:    w.addr,
+		Handler: w.mux,
+	}
+
+	w.InChan = make(chan waf.ParsedRequest)
+	w.WaapRunners = make([]WaapRunner, wc.Routines)
+
+	for nbRoutine := 0; nbRoutine < wc.Routines; nbRoutine++ {
+
+		wafUUID := uuid.New().String()
+		wafLogger := &log.Entry{}
+
+		//configure logger
+		if wc.Debug {
+			var clog = log.New()
+			if err := types.ConfigureLogger(clog); err != nil {
+				log.Fatalf("While creating bucket-specific logger : %s", err)
+			}
+			clog.SetLevel(log.DebugLevel)
+			wafLogger = clog.WithFields(log.Fields{
+				"uuid": wafUUID,
+			})
+		} else {
+			wafLogger = log.WithFields(log.Fields{
+				"uuid": wafUUID,
+			})
+		}
+
+		runner := WaapRunner{
+			inChan: w.InChan,
+			UUID:   wafUUID,
+			logger: wafLogger,
+		}
+		w.WaapRunners[nbRoutine] = runner
+		//most likely missign somethign here to actually start the runner :)
+	}
+
+	w.logger.Infof("Created %d waf runners", len(w.WaapRunners))
+
+	//We don´t use the wrapper provided by coraza because we want to fully control what happens when a rule match to send the information in crowdsec
+	w.mux.HandleFunc(w.config.Path, w.waapHandler)
+
+	return nil
+}
+
+func (w *WaapSource) ConfigureByDSN(dsn string, labels map[string]string, logger *log.Entry, uuid string) error {
+	return fmt.Errorf("WAF datasource does not support command line acquisition")
+}
+
+func (w *WaapSource) GetMode() string {
+	return w.config.Mode
+}
+
+func (w *WaapSource) GetName() string {
+	return "waf"
+}
+
+func (w *WaapSource) OneShotAcquisition(out chan types.Event, t *tomb.Tomb) error {
+	return fmt.Errorf("WAF datasource does not support command line acquisition")
+}
+
+func (w *WaapSource) StreamingAcquisition(out chan types.Event, t *tomb.Tomb) error {
+	w.outChan = out
+	t.Go(func() error {
+		defer trace.CatchPanic("crowdsec/acquis/waf/live")
+
+		w.logger.Infof("%d waf runner to start", len(w.WaapRunners))
+		for _, runner := range w.WaapRunners {
+			runner := runner
+			runner.outChan = out
+			t.Go(func() error {
+				defer trace.CatchPanic("crowdsec/acquis/waf/live/runner")
+				return runner.Run(t)
+			})
+		}
+
+		w.logger.Infof("Starting WAF server on %s:%d%s", w.config.ListenAddr, w.config.ListenPort, w.config.Path)
+		t.Go(func() error {
+			var err error
+			if w.config.CertFilePath != "" && w.config.KeyFilePath != "" {
+				err = w.server.ListenAndServeTLS(w.config.CertFilePath, w.config.KeyFilePath)
+			} else {
+				err = w.server.ListenAndServe()
+			}
+
+			if err != nil && err != http.ErrServerClosed {
+				return errors.Wrap(err, "WAF server failed")
+			}
+			return nil
+		})
+		<-t.Dying()
+		w.logger.Infof("Stopping WAF server on %s:%d%s", w.config.ListenAddr, w.config.ListenPort, w.config.Path)
+		w.server.Shutdown(context.TODO())
+		return nil
+	})
+	return nil
+}
+
+func (w *WaapSource) CanRun() error {
+	return nil
+}
+
+func (w *WaapSource) GetUuid() string {
+	return w.config.UniqueId
+}
+
+func (w *WaapSource) Dump() interface{} {
+	return w
+}
+
+type BodyResponse struct {
+	Action string `json:"action"`
+}
+
+// should this be in the runner ?
+func (w *WaapSource) waapHandler(rw http.ResponseWriter, r *http.Request) {
+	// parse the request only once
+	parsedRequest, err := waf.NewParsedRequestFromRequest(r)
+	if err != nil {
+		log.Errorf("%s", err)
+		rw.WriteHeader(http.StatusInternalServerError)
+		return
+	}
+	w.InChan <- parsedRequest
+
+	message := <-parsedRequest.ResponseChannel
+
+	if message.Err != nil {
+		log.Errorf("Error while processing InBAND: %s", err)
+		rw.WriteHeader(http.StatusInternalServerError)
+		return
+	}
+
+	//here we must rely on WaapRuntimeConfig to know what to do
+	if message.Interruption != nil {
+		rw.WriteHeader(http.StatusForbidden)
+		action := message.Interruption.Action
+		if action == "deny" { // bouncers understand "ban" and not "deny"
+			action = "ban"
+		}
+		body, err := json.Marshal(BodyResponse{Action: action})
+		if err != nil {
+			log.Errorf("unable to build response: %s", err)
+		} else {
+			rw.Write(body)
+		}
+		return
+	}
+
+	rw.WriteHeader(http.StatusOK)
+	body, err := json.Marshal(BodyResponse{Action: "allow"})
+	if err != nil {
+		log.Errorf("unable to marshal response: %s", err)
+		rw.WriteHeader(http.StatusInternalServerError)
+	} else {
+		rw.Write(body)
+	}
+
+}

+ 67 - 0
pkg/acquisition/modules/waf/waap_runner.go

@@ -0,0 +1,67 @@
+package wafacquisition
+
+import (
+	"time"
+
+	"github.com/crowdsecurity/coraza/v3"
+	"github.com/crowdsecurity/crowdsec/pkg/types"
+	"github.com/crowdsecurity/crowdsec/pkg/waf"
+	"github.com/prometheus/client_golang/prometheus"
+	log "github.com/sirupsen/logrus"
+	"gopkg.in/tomb.v2"
+)
+
+// that's the runtime structure of the WAAP as seen from the acquis
+type WaapRunner struct {
+	outChan           chan types.Event
+	inChan            chan waf.ParsedRequest
+	UUID              string
+	WaapRuntime       *waf.WaapRuntimeConfig //this holds the actual waap runtime config, rules, remediations, hooks etc.
+	WaapInbandEngine  coraza.WAF
+	WaapOutbandEngine coraza.WAF
+	logger            *log.Entry
+}
+
+func (r *WaapRunner) Run(t *tomb.Tomb) error {
+	r.logger.Infof("Waap Runner ready to process event")
+	for {
+		select {
+		case <-t.Dying():
+			r.logger.Infof("Waf Runner is dying")
+			return nil
+		case request := <-r.inChan:
+			r.logger.Infof("Requests handled by runner %s", request.UUID)
+
+			//tx := waf.NewExtendedTransaction(r.WaapInbandEngine, r.UUID)
+			WafReqCounter.With(prometheus.Labels{"source": request.RemoteAddr}).Inc()
+			//measure the time spent in the WAF
+			startParsing := time.Now()
+
+			//pre eval (expr) rules
+			err := r.WaapRuntime.ProcessPreEvalRules(request)
+			if err != nil {
+				r.logger.Errorf("unable to process PreEval rules: %s", err)
+				continue
+			}
+
+			//inband WAAP rules
+			interrupt, err := r.WaapRuntime.ProcessInBandRules(request)
+			elapsed := time.Since(startParsing)
+			WafInbandParsingHistogram.With(prometheus.Labels{"source": request.RemoteAddr}).Observe(elapsed.Seconds())
+
+			//generate reponse for the remediation component, based on the WAAP config + inband rules evaluation
+			//@tko : this should move in the WaapRuntimeConfig as it knows what to do with the interruption and the expected remediation
+			response := waf.NewResponseRequest(r.WaapRuntime.InBandTx.Tx, interrupt, request.UUID, err)
+
+			err = r.WaapRuntime.ProcessOnMatchRules(request, response)
+			if err != nil {
+				r.logger.Errorf("unable to process OnMatch rules: %s", err)
+				continue
+			}
+
+			// send back the result to the HTTP handler for the InBand part
+			request.ResponseChannel <- response
+
+		}
+	}
+}

+ 0 - 665
pkg/acquisition/modules/waf/waf.go

@@ -1,665 +0,0 @@
-package wafacquisition
-
-import (
-	"context"
-	"encoding/json"
-	"fmt"
-	"net/http"
-	"os"
-	"regexp"
-	"strings"
-	"time"
-
-	"github.com/antonmedv/expr"
-	"github.com/crowdsecurity/coraza/v3"
-	"github.com/crowdsecurity/coraza/v3/experimental"
-	corazatypes "github.com/crowdsecurity/coraza/v3/types"
-	"github.com/crowdsecurity/crowdsec/pkg/acquisition/configuration"
-	"github.com/crowdsecurity/crowdsec/pkg/types"
-	"github.com/crowdsecurity/crowdsec/pkg/waf"
-	"github.com/crowdsecurity/go-cs-lib/trace"
-	"github.com/google/uuid"
-	"github.com/pkg/errors"
-	"github.com/prometheus/client_golang/prometheus"
-	log "github.com/sirupsen/logrus"
-	"gopkg.in/tomb.v2"
-	"gopkg.in/yaml.v2"
-)
-
-const (
-	InBand    = "inband"
-	OutOfBand = "outofband"
-)
-
-type WafRunner struct {
-	outChan           chan types.Event
-	inChan            chan waf.ParsedRequest
-	inBandWaf         coraza.WAF
-	outOfBandWaf      coraza.WAF
-	UUID              string
-	RulesCollections  []*waf.WafRulesCollection
-	logger            *log.Entry
-	VariablesTracking []*regexp.Regexp
-}
-
-type WafSourceConfig struct {
-	ListenAddr                        string   `yaml:"listen_addr"`
-	ListenPort                        int      `yaml:"listen_port"`
-	CertFilePath                      string   `yaml:"cert_file"`
-	KeyFilePath                       string   `yaml:"key_file"`
-	Path                              string   `yaml:"path"`
-	WafRoutines                       int      `yaml:"waf_routines"`
-	Debug                             bool     `yaml:"debug"`
-	VariablesTracking                 []string `yaml:"variables_tracking"`
-	configuration.DataSourceCommonCfg `yaml:",inline"`
-}
-
-type WafSource struct {
-	config  WafSourceConfig
-	logger  *log.Entry
-	mux     *http.ServeMux
-	server  *http.Server
-	addr    string
-	outChan chan types.Event
-	InChan  chan waf.ParsedRequest
-
-	RulesCollections []*waf.WafRulesCollection
-
-	WafRunners []WafRunner
-}
-
-var WafGlobalParsingHistogram = prometheus.NewHistogramVec(
-	prometheus.HistogramOpts{
-		Help:    "Time spent processing a request by the WAF.",
-		Name:    "cs_waf_parsing_time_seconds",
-		Buckets: []float64{0.0005, 0.001, 0.0015, 0.002, 0.0025, 0.003, 0.004, 0.005, 0.0075, 0.01},
-	},
-	[]string{"source"},
-)
-
-var WafInbandParsingHistogram = prometheus.NewHistogramVec(
-	prometheus.HistogramOpts{
-		Help:    "Time spent processing a request by the inband WAF.",
-		Name:    "cs_waf_inband_parsing_time_seconds",
-		Buckets: []float64{0.0005, 0.001, 0.0015, 0.002, 0.0025, 0.003, 0.004, 0.005, 0.0075, 0.01},
-	},
-	[]string{"source"},
-)
-
-var WafOutbandParsingHistogram = prometheus.NewHistogramVec(
-	prometheus.HistogramOpts{
-		Help:    "Time spent processing a request by the WAF.",
-		Name:    "cs_waf_outband_parsing_time_seconds",
-		Buckets: []float64{0.0005, 0.001, 0.0015, 0.002, 0.0025, 0.003, 0.004, 0.005, 0.0075, 0.01},
-	},
-	[]string{"source"},
-)
-
-var WafReqCounter = prometheus.NewCounterVec(
-	prometheus.CounterOpts{
-		Name: "cs_waf_reqs_total",
-		Help: "Total events processed by the WAF.",
-	},
-	[]string{"source"},
-)
-
-var WafRuleHits = prometheus.NewCounterVec(
-	prometheus.CounterOpts{
-		Name: "cs_waf_rule_hits",
-		Help: "Count of triggered rule, by rule_id and type (inband/outofband).",
-	},
-	[]string{"rule_id", "type"},
-)
-
-func (w *WafSource) GetMetrics() []prometheus.Collector {
-	return nil
-}
-
-func (w *WafSource) GetAggregMetrics() []prometheus.Collector {
-	return nil
-}
-
-func (w *WafSource) UnmarshalConfig(yamlConfig []byte) error {
-	wafConfig := WafSourceConfig{}
-	err := yaml.UnmarshalStrict(yamlConfig, &wafConfig)
-	if err != nil {
-		return errors.Wrap(err, "Cannot parse waf configuration")
-	}
-
-	w.config = wafConfig
-
-	if w.config.ListenAddr == "" {
-		return fmt.Errorf("listen_addr cannot be empty")
-	}
-
-	if w.config.ListenPort == 0 {
-		return fmt.Errorf("listen_port cannot be empty")
-	}
-
-	//FIXME: is that really needed ?
-	if w.config.Path == "" {
-		return fmt.Errorf("path cannot be empty")
-	}
-
-	if w.config.Path[0] != '/' {
-		w.config.Path = "/" + w.config.Path
-	}
-
-	if w.config.Mode == "" {
-		w.config.Mode = configuration.TAIL_MODE
-	}
-
-	return nil
-}
-
-func logError(error corazatypes.MatchedRule) {
-	msg := error.ErrorLog()
-	log.Infof("[logError][%s]  %s", error.Rule().Severity(), msg)
-}
-
-func (w *WafSource) Configure(yamlConfig []byte, logger *log.Entry) error {
-	err := w.UnmarshalConfig(yamlConfig)
-	if err != nil {
-		return errors.Wrap(err, "Cannot parse waf configuration")
-	}
-
-	w.logger = logger
-
-	w.logger.Tracef("WAF configuration: %+v", w.config)
-
-	w.addr = fmt.Sprintf("%s:%d", w.config.ListenAddr, w.config.ListenPort)
-
-	w.mux = http.NewServeMux()
-
-	w.server = &http.Server{
-		Addr:    w.addr,
-		Handler: w.mux,
-	}
-
-	RegisterRX()
-
-	ruleLoader := waf.NewWafRuleLoader()
-
-	rulesCollections, err := ruleLoader.LoadWafRules()
-	if err != nil {
-		return fmt.Errorf("cannot load WAF rules: %w", err)
-	}
-
-	w.RulesCollections = rulesCollections
-
-	var inBandRules string
-	var outOfBandRules string
-
-	//spew.Dump(rulesCollections)
-
-	for _, collection := range rulesCollections {
-		if !collection.OutOfBand {
-			inBandRules += collection.String() + "\n"
-		} else {
-			outOfBandRules += collection.String() + "\n"
-		}
-	}
-
-	w.logger.Infof("Loading %d in-band rules", len(strings.Split(inBandRules, "\n")))
-
-	//w.logger.Infof("Loading rules %+v", inBandRules)
-
-	fs := os.DirFS(ruleLoader.Datadir)
-	// always have at least one waf routine
-	if w.config.WafRoutines == 0 {
-		w.config.WafRoutines = 1
-	}
-
-	w.InChan = make(chan waf.ParsedRequest)
-	w.logger.Infof("w.InChan creation: %p", w.InChan)
-	w.WafRunners = make([]WafRunner, w.config.WafRoutines)
-	for nbRoutine := 0; nbRoutine < w.config.WafRoutines; nbRoutine++ {
-		w.logger.Infof("Loading %d in-band rules", len(strings.Split(inBandRules, "\n")))
-
-		//in-band waf : kill on sight
-		inbandwaf, err := coraza.NewWAF(
-			coraza.NewWAFConfig().
-				//WithErrorCallback(logError).
-				WithDirectives(inBandRules).WithRootFS(fs),
-		)
-
-		if err != nil {
-			return errors.Wrap(err, "Cannot create WAF")
-		}
-		w.logger.Infof("Loading %d out-of-band rules", len(strings.Split(outOfBandRules, "\n")))
-		//out-of-band waf : log only
-		outofbandwaf, err := coraza.NewWAF(
-			coraza.NewWAFConfig().
-				//WithErrorCallback(logError).
-				WithDirectives(outOfBandRules).WithRootFS(fs),
-		)
-
-		if err != nil {
-			return errors.Wrap(err, "Cannot create WAF")
-		}
-		wafUUID := uuid.New().String()
-		wafLogger := &log.Entry{}
-		if w.config.Debug {
-			var clog = log.New()
-			if err := types.ConfigureLogger(clog); err != nil {
-				log.Fatalf("While creating bucket-specific logger : %s", err)
-			}
-			clog.SetLevel(log.DebugLevel)
-			wafLogger = clog.WithFields(log.Fields{
-				"uuid": wafUUID,
-			})
-		} else {
-			wafLogger = log.WithFields(log.Fields{
-				"uuid": wafUUID,
-			})
-		}
-
-		var compiledVariableRules []*regexp.Regexp
-
-		for _, variable := range w.config.VariablesTracking {
-			compiledVariableRule, err := regexp.Compile(variable)
-			if err != nil {
-				return fmt.Errorf("cannot compile variable regexp %s: %w", variable, err)
-			}
-			compiledVariableRules = append(compiledVariableRules, compiledVariableRule)
-		}
-
-		runner := WafRunner{
-			outOfBandWaf:      outofbandwaf,
-			inBandWaf:         inbandwaf,
-			inChan:            w.InChan,
-			UUID:              wafUUID,
-			RulesCollections:  rulesCollections,
-			logger:            wafLogger,
-			VariablesTracking: compiledVariableRules,
-		}
-		w.WafRunners[nbRoutine] = runner
-	}
-
-	w.logger.Infof("Created %d waf runners", len(w.WafRunners))
-
-	//We don´t use the wrapper provided by coraza because we want to fully control what happens when a rule match to send the information in crowdsec
-	w.mux.HandleFunc(w.config.Path, w.wafHandler)
-
-	return nil
-}
-
-func (w *WafSource) ConfigureByDSN(dsn string, labels map[string]string, logger *log.Entry, uuid string) error {
-	return fmt.Errorf("WAF datasource does not support command line acquisition")
-}
-
-func (w *WafSource) GetMode() string {
-	return w.config.Mode
-}
-
-func (w *WafSource) GetName() string {
-	return "waf"
-}
-
-func (w *WafSource) OneShotAcquisition(out chan types.Event, t *tomb.Tomb) error {
-	return fmt.Errorf("WAF datasource does not support command line acquisition")
-}
-
-func (w *WafSource) StreamingAcquisition(out chan types.Event, t *tomb.Tomb) error {
-	w.outChan = out
-	t.Go(func() error {
-		defer trace.CatchPanic("crowdsec/acquis/waf/live")
-
-		w.logger.Infof("%d waf runner to start", len(w.WafRunners))
-		for _, runner := range w.WafRunners {
-			runner := runner
-			runner.outChan = out
-			t.Go(func() error {
-				defer trace.CatchPanic("crowdsec/acquis/waf/live/runner")
-				return runner.Run(t)
-			})
-		}
-
-		w.logger.Infof("Starting WAF server on %s:%d%s", w.config.ListenAddr, w.config.ListenPort, w.config.Path)
-		t.Go(func() error {
-			var err error
-			if w.config.CertFilePath != "" && w.config.KeyFilePath != "" {
-				err = w.server.ListenAndServeTLS(w.config.CertFilePath, w.config.KeyFilePath)
-			} else {
-				err = w.server.ListenAndServe()
-			}
-
-			if err != nil && err != http.ErrServerClosed {
-				return errors.Wrap(err, "WAF server failed")
-			}
-			return nil
-		})
-		<-t.Dying()
-		w.logger.Infof("Stopping WAF server on %s:%d%s", w.config.ListenAddr, w.config.ListenPort, w.config.Path)
-		w.server.Shutdown(context.TODO())
-		return nil
-	})
-	return nil
-}
-
-func (w *WafSource) CanRun() error {
-	return nil
-}
-
-func (w *WafSource) GetUuid() string {
-	return w.config.UniqueId
-}
-
-func (w *WafSource) Dump() interface{} {
-	return w
-}
-
-func (r *WafRunner) processReqWithEngine(tx experimental.FullTransaction, parsedRequest waf.ParsedRequest, wafType string) (*corazatypes.Interruption, experimental.FullTransaction, error) {
-	var in *corazatypes.Interruption
-	if tx.IsRuleEngineOff() {
-		r.logger.Printf("engine is off")
-		return nil, nil, nil
-	}
-
-	defer func() {
-		tx.ProcessLogging()
-		//Dont close the transaction here: we still need access to the variables afterwards
-		//tx.Close()
-	}()
-
-	//this method is not exported by coraza, so we have to do it ourselves.
-	//ideally, this would be dealt with by expr code, and we provide helpers to manipulate the transaction object?\
-	//var txx experimental.FullTransaction
-
-	//txx := experimental.ToFullInterface(tx)
-	//txx = tx.(experimental.FullTransaction)
-	//txx.RemoveRuleByID(1)
-	tx.ProcessConnection(parsedRequest.ClientIP, 0, "", 0)
-
-	//tx.ProcessURI(parsedRequest.URL.String(), parsedRequest.Method, parsedRequest.Proto) //FIXME: get it from the headers
-	tx.ProcessURI(parsedRequest.URI, parsedRequest.Method, parsedRequest.Proto) //FIXME: get it from the headers
-
-	for k, vr := range parsedRequest.Headers {
-		for _, v := range vr {
-			tx.AddRequestHeader(k, v)
-		}
-	}
-
-	if parsedRequest.ClientHost != "" {
-		tx.AddRequestHeader("Host", parsedRequest.ClientHost)
-		// This connector relies on the host header (now host field) to populate ServerName
-		tx.SetServerName(parsedRequest.ClientHost)
-	}
-
-	if parsedRequest.TransferEncoding != nil {
-		tx.AddRequestHeader("Transfer-Encoding", parsedRequest.TransferEncoding[0])
-	}
-
-	in = tx.ProcessRequestHeaders()
-
-	//spew.Dump(in)
-	//spew.Dump(tx.MatchedRules())
-
-	/*for _, rule := range tx.MatchedRules() {
-		log.Infof("Rule %d disruptive: %t", rule.Rule().ID(), rule.Disruptive())
-		if rule.Message() == "" {
-			continue
-		}
-	}*/
-
-	//if we're inband, we should stop here, but for outofband go to the end
-	if in != nil && wafType == InBand {
-		return in, tx, nil
-	}
-
-	if parsedRequest.Body != nil && len(parsedRequest.Body) != 0 {
-		it, _, err := tx.WriteRequestBody(parsedRequest.Body)
-		if err != nil {
-			return nil, nil, errors.Wrap(err, "Cannot read request body")
-		}
-
-		if it != nil {
-			//log.Infof("blocking rule id %d", in.RuleID)
-			return it, nil, nil
-		}
-	}
-
-	in, err := tx.ProcessRequestBody()
-	if err != nil {
-		return nil, nil, errors.Wrap(err, "Cannot process request body")
-	}
-	if in != nil && wafType == InBand {
-		//log.Infof("blocking rule id %d", in.RuleID)
-
-		return in, tx, nil
-	}
-
-	return nil, tx, nil
-}
-
-func (r *WafRunner) Run(t *tomb.Tomb) error {
-	r.logger.Infof("Waf Runner ready to process event")
-	for {
-		select {
-		case <-t.Dying():
-			r.logger.Infof("Waf Runner is dying")
-			return nil
-		case request := <-r.inChan:
-			r.logger.Infof("Requests handled by runner %s", r.UUID)
-			var evt *types.Event
-			WafReqCounter.With(prometheus.Labels{"source": request.RemoteAddr}).Inc()
-			//measure the time spent in the WAF
-			startParsing := time.Now()
-			inBoundTx := r.inBandWaf.NewTransactionWithID(request.UUID)
-			expTx := inBoundTx.(experimental.FullTransaction)
-			// we use this internal transaction for the expr helpers
-			tx := waf.NewTransaction(expTx)
-
-			//r.logger.Infof("Processing request %s | tx: %p", request.UUID, tx)
-
-			//Run the pre_eval hooks
-			for _, rules := range r.RulesCollections {
-				if len(rules.CompiledPreEval) == 0 {
-					continue
-				}
-				for _, compiledHook := range rules.CompiledPreEval {
-					if compiledHook.Filter != nil {
-						res, err := expr.Run(compiledHook.Filter, map[string]interface{}{
-							"rules": rules,
-							"req":   request,
-						})
-						if err != nil {
-							log.Errorf("unable to run PreEval filter: %s", err)
-							continue
-						}
-
-						switch t := res.(type) {
-						case bool:
-							if !t {
-								log.Infof("filter didnt match")
-								continue
-							}
-						default:
-							log.Errorf("Filter must return a boolean, can't filter")
-							continue
-						}
-					}
-					// here means there is no filter or the filter matched
-					for _, applyExpr := range compiledHook.Apply {
-						_, err := expr.Run(applyExpr, map[string]interface{}{
-							"rules":          rules,
-							"req":            request,
-							"RemoveRuleByID": tx.RemoveRuleByIDWithError,
-						})
-						if err != nil {
-							log.Errorf("unable to apply filter: %s", err)
-							continue
-						}
-					}
-				}
-			}
-
-			in, expTx, err := r.processReqWithEngine(expTx, request, InBand)
-			request.Tx = expTx
-			//log.Infof("-> %s", spew.Sdump(in))
-
-			//log.Infof("tx variables: %+v", expTx.Collection(variables.TX))
-
-			//foo := expTx.(plugintypes.TransactionState)
-
-			//log.Infof("from tstate: %+v", foo.Variables().TX().FindAll())
-
-			response := waf.NewResponseRequest(expTx, in, request.UUID, err)
-
-			// run the on_match hooks
-			for _, rules := range r.RulesCollections {
-				if len(rules.CompiledOnMatch) == 0 {
-					continue
-				}
-				for _, compiledHook := range rules.CompiledOnMatch {
-					if compiledHook.Filter != nil {
-						res, err := expr.Run(compiledHook.Filter, map[string]interface{}{
-							"rules": rules,
-							"req":   request,
-						})
-						if err != nil {
-							r.logger.Errorf("unable to run PreEval filter: %s", err)
-							continue
-						}
-
-						switch t := res.(type) {
-						case bool:
-							if !t {
-								continue
-							}
-						default:
-							r.logger.Errorf("Filter must return a boolean, can't filter")
-							continue
-						}
-					}
-					// here means there is no filter or the filter matched
-					for _, applyExpr := range compiledHook.Apply {
-						_, err := expr.Run(applyExpr, map[string]interface{}{
-							"rules":              rules,
-							"req":                request,
-							"RemoveRuleByID":     tx.RemoveRuleByIDWithError,
-							"SetRemediation":     response.SetRemediation,
-							"SetRemediationByID": response.SetRemediationByID,
-							"CancelEvent":        response.CancelEvent,
-						})
-						if err != nil {
-							r.logger.Errorf("unable to apply filter: %s", err)
-							continue
-						}
-					}
-				}
-			}
-			logged := false
-			//measure the full time spent in the WAF
-			elapsed := time.Since(startParsing)
-			WafInbandParsingHistogram.With(prometheus.Labels{"source": request.RemoteAddr}).Observe(elapsed.Seconds())
-			// send back the result to the HTTP handler for the InBand part
-			request.ResponseChannel <- response
-			if in != nil && response.SendEvents {
-				evt = &types.Event{}
-				*evt, err = EventFromRequest(request)
-				if err != nil {
-					return fmt.Errorf("cannot create event from waap context : %w", err)
-				}
-				err = r.AccumulateTxToEvent(expTx, InBand, evt)
-				if err != nil {
-					return fmt.Errorf("cannot convert transaction to event : %w", err)
-				}
-				LogWaapEvent(evt, r.logger)
-				logged = true
-				r.outChan <- *evt
-			}
-			expTx.Close()
-
-			outBandStart := time.Now()
-			// Process outBand
-			outBandTx := r.outOfBandWaf.NewTransactionWithID(request.UUID)
-			expTx = outBandTx.(experimental.FullTransaction)
-
-			in, expTx, err = r.processReqWithEngine(expTx, request, OutOfBand)
-			if err != nil { //things went south
-				r.logger.Errorf("Error while processing request : %s", err)
-				continue
-			}
-			request.Tx = expTx
-			if expTx != nil && len(expTx.MatchedRules()) > 0 {
-				//if event was not instantiated after inband processing, do it now
-				if evt == nil {
-					tmpEvt, err := EventFromRequest(request)
-					if err != nil {
-						return fmt.Errorf("cannot create event from waap context : %w", err)
-					}
-					evt = &tmpEvt
-				}
-
-				err = r.AccumulateTxToEvent(expTx, OutOfBand, evt)
-				if err != nil {
-					return fmt.Errorf("cannot convert transaction to event : %w", err)
-				}
-
-				// expTx.MatchedRules() returns also rules that set variables
-				// in evt.Waap.MatchedRules we have filtered those rules
-				if len(evt.Waap.MatchedRules) > 0 {
-					if !logged {
-						LogWaapEvent(evt, r.logger)
-					}
-					r.outChan <- *evt
-				}
-			}
-			expTx.Close()
-			//measure the full time spent in the WAF
-			totalElapsed := time.Since(startParsing)
-			WafGlobalParsingHistogram.With(prometheus.Labels{"source": request.RemoteAddr}).Observe(totalElapsed.Seconds())
-			elapsed = time.Since(outBandStart)
-			WafOutbandParsingHistogram.With(prometheus.Labels{"source": request.RemoteAddr}).Observe(elapsed.Seconds())
-		}
-	}
-}
-
-type BodyResponse struct {
-	Action string `json:"action"`
-}
-
-func (w *WafSource) wafHandler(rw http.ResponseWriter, r *http.Request) {
-	// parse the request only once
-	parsedRequest, err := waf.NewParsedRequestFromRequest(r)
-	if err != nil {
-		log.Errorf("%s", err)
-		rw.WriteHeader(http.StatusInternalServerError)
-		return
-	}
-	w.InChan <- parsedRequest
-
-	message := <-parsedRequest.ResponseChannel
-
-	if message.Err != nil {
-		log.Errorf("Error while processing InBAND: %s", err)
-		rw.WriteHeader(http.StatusInternalServerError)
-		return
-	}
-
-	if message.Interruption != nil {
-		rw.WriteHeader(http.StatusForbidden)
-		action := message.Interruption.Action
-		if action == "deny" { // bouncers understand "ban" and not "deny"
-			action = "ban"
-		}
-		body, err := json.Marshal(BodyResponse{Action: action})
-		if err != nil {
-			log.Errorf("unable to build response: %s", err)
-		} else {
-			rw.Write(body)
-		}
-		return
-	}
-
-	rw.WriteHeader(http.StatusOK)
-	body, err := json.Marshal(BodyResponse{Action: "allow"})
-	if err != nil {
-		log.Errorf("unable to marshal response: %s", err)
-		rw.WriteHeader(http.StatusInternalServerError)
-	} else {
-		rw.Write(body)
-	}
-
-}

+ 17 - 7
pkg/waf/env.go

@@ -1,16 +1,26 @@
 package waf
 
-import "github.com/crowdsecurity/coraza/v3/experimental"
+import (
+	"github.com/crowdsecurity/coraza/v3"
+	"github.com/crowdsecurity/coraza/v3/experimental"
+)
 
-type Transaction struct {
+type ExtendedTransaction struct {
 	Tx experimental.FullTransaction
 }
 
-func NewTransaction(tx experimental.FullTransaction) Transaction {
-	return Transaction{Tx: tx}
+func NewExtendedTransaction(engine coraza.WAF, uuid string) ExtendedTransaction {
+	inBoundTx := engine.NewTransactionWithID(uuid)
+	expTx := inBoundTx.(experimental.FullTransaction)
+	tx := NewTransaction(expTx)
+	return tx
 }
 
-func (t *Transaction) RemoveRuleByIDWithError(id int) error {
+func NewTransaction(tx experimental.FullTransaction) ExtendedTransaction {
+	return ExtendedTransaction{Tx: tx}
+}
+
+func (t *ExtendedTransaction) RemoveRuleByIDWithError(id int) error {
 	t.Tx.RemoveRuleByID(id)
 	return nil
 }
@@ -18,8 +28,8 @@ func (t *Transaction) RemoveRuleByIDWithError(id int) error {
 func GetEnv() map[string]interface{} {
 	ResponseRequest := ResponseRequest{}
 	ParsedRequest := ParsedRequest{}
-	Rules := &WafRulesCollection{}
-	Tx := Transaction{}
+	Rules := &WaapCollection{}
+	Tx := ExtendedTransaction{}
 
 	return map[string]interface{}{
 		"rules":              Rules,

+ 237 - 0
pkg/waf/waap.go

@@ -0,0 +1,237 @@
+package waf
+
+import (
+	"fmt"
+	"regexp"
+
+	"github.com/antonmedv/expr"
+	"github.com/antonmedv/expr/vm"
+	corazatypes "github.com/crowdsecurity/coraza/v3/types"
+	log "github.com/sirupsen/logrus"
+)
+
+type Hook struct {
+	Filter     string      `yaml:"filter"`
+	FilterExpr *vm.Program `yaml:"-"`
+
+	OnSuccess string        `yaml:"on_success"`
+	Apply     []string      `yaml:"apply"`
+	ApplyExpr []*vm.Program `yaml:"-"`
+}
+
+func (h *Hook) Build() error {
+
+	if h.Filter != "" {
+		program, err := expr.Compile(h.Filter) //FIXME: opts
+		if err != nil {
+			return fmt.Errorf("unable to compile filter %s : %w", h.Filter, err)
+		}
+		h.FilterExpr = program
+	}
+	for _, apply := range h.Apply {
+		program, err := expr.Compile(apply, GetExprWAFOptions(GetEnv())...)
+		if err != nil {
+			return fmt.Errorf("unable to compile apply %s : %w", apply, err)
+		}
+		h.ApplyExpr = append(h.ApplyExpr, program)
+	}
+	return nil
+}
+
+// runtime version of WaapConfig
+type WaapRuntimeConfig struct {
+	Name                      string
+	OutOfBandRules            []WaapCollection
+	OutOfBandTx               ExtendedTransaction //is it a good idea ?
+	InBandRules               []WaapCollection
+	InBandTx                  ExtendedTransaction //is it a good idea ?
+	DefaultRemediation        string
+	CompiledOnLoad            []Hook
+	CompiledPreEval           []Hook
+	CompiledOnMatch           []Hook
+	CompiledVariablesTracking []*regexp.Regexp
+}
+
+type WaapConfig struct {
+	Name               string   `yaml:"name"`
+	OutOfBandRules     []string `yaml:"outofband_rules"`
+	InBandRules        []string `yaml:"inband_rules"`
+	DefaultRemediation string   `yaml:"default_remediation"`
+	OnLoad             []Hook   `yaml:"on_load"`
+	PreEval            []Hook   `yaml:"pre_eval"`
+	OnMatch            []Hook   `yaml:"on_match"`
+	VariablesTracking  []string `yaml:"variables_tracking"`
+}
+
+func (wc *WaapConfig) Build() (*WaapRuntimeConfig, error) {
+	ret := &WaapRuntimeConfig{}
+	ret.Name = wc.Name
+	ret.DefaultRemediation = wc.DefaultRemediation
+
+	//load rules
+	for _, rule := range wc.OutOfBandRules {
+		collection, err := LoadCollection(rule)
+		if err != nil {
+			return nil, fmt.Errorf("unable to load outofband rule %s : %s", rule, err)
+		}
+		ret.OutOfBandRules = append(ret.OutOfBandRules, collection)
+	}
+
+	for _, rule := range wc.InBandRules {
+		collection, err := LoadCollection(rule)
+		if err != nil {
+			return nil, fmt.Errorf("unable to load inband rule %s : %s", rule, err)
+		}
+		ret.InBandRules = append(ret.InBandRules, collection)
+	}
+
+	//load hooks
+	for _, hook := range wc.OnLoad {
+		err := hook.Build()
+		if err != nil {
+			return nil, fmt.Errorf("unable to build on_load hook : %s", err)
+		}
+		ret.CompiledOnLoad = append(ret.CompiledOnLoad, hook)
+	}
+
+	for _, hook := range wc.PreEval {
+		err := hook.Build()
+		if err != nil {
+			return nil, fmt.Errorf("unable to build pre_eval hook : %s", err)
+		}
+		ret.CompiledPreEval = append(ret.CompiledPreEval, hook)
+	}
+
+	for _, hook := range wc.OnMatch {
+		err := hook.Build()
+		if err != nil {
+			return nil, fmt.Errorf("unable to build on_match hook : %s", err)
+		}
+		ret.CompiledOnMatch = append(ret.CompiledOnMatch, hook)
+	}
+
+	//variable tracking
+	for _, variable := range wc.VariablesTracking {
+		compiledVariableRule, err := regexp.Compile(variable)
+		if err != nil {
+			return nil, fmt.Errorf("cannot compile variable regexp %s: %w", variable, err)
+		}
+		ret.CompiledVariablesTracking = append(ret.CompiledVariablesTracking, compiledVariableRule)
+	}
+	return ret, nil
+}
+
+func (w *WaapRuntimeConfig) ProcessOnMatchRules(request ParsedRequest, response ResponseRequest) error {
+
+	for _, rule := range w.CompiledOnMatch {
+		if rule.FilterExpr != nil {
+			output, err := expr.Run(rule.FilterExpr, map[string]interface{}{
+				//"rules": rules, //is it still useful ?
+				"req": request,
+			})
+			if err != nil {
+				return fmt.Errorf("unable to run filter %s : %w", rule.Filter, err)
+			}
+			switch t := output.(type) {
+			case bool:
+				if !t {
+					log.Infof("filter didnt match")
+					continue
+				}
+			default:
+				log.Errorf("Filter must return a boolean, can't filter")
+				continue
+			}
+		}
+		for _, applyExpr := range rule.ApplyExpr {
+			_, err := expr.Run(applyExpr, map[string]interface{}{
+				//"rules":                 w.InBandTx.Tx.Rules, //what is it supposed to be ? matched rules ?
+				"req":                   request,
+				"RemoveInbandRuleByID":  w.RemoveInbandRuleByID,
+				"RemoveOutbandRuleByID": w.RemoveOutbandRuleByID,
+				"SetRemediation":        response.SetRemediation,
+				"SetRemediationByID":    response.SetRemediationByID,
+				"CancelEvent":           response.CancelEvent,
+			})
+			if err != nil {
+				log.Errorf("unable to apply filter: %s", err)
+				continue
+			}
+		}
+	}
+	return nil
+}
+
+func (w *WaapRuntimeConfig) ProcessPreEvalRules(request ParsedRequest) error {
+	for _, rule := range w.CompiledPreEval {
+		if rule.FilterExpr != nil {
+			output, err := expr.Run(rule.FilterExpr, map[string]interface{}{
+				//"rules": rules, //is it still useful ?
+				"req": request,
+			})
+			if err != nil {
+				return fmt.Errorf("unable to run filter %s : %w", rule.Filter, err)
+			}
+			switch t := output.(type) {
+			case bool:
+				if !t {
+					log.Infof("filter didnt match")
+					continue
+				}
+			default:
+				log.Errorf("Filter must return a boolean, can't filter")
+				continue
+			}
+		}
+		// here means there is no filter or the filter matched
+		for _, applyExpr := range rule.ApplyExpr {
+			_, err := expr.Run(applyExpr, map[string]interface{}{
+				"inband_rules":          w.InBandRules,
+				"outband_rules":         w.OutOfBandRules,
+				"req":                   request,
+				"RemoveInbandRuleByID":  w.RemoveInbandRuleByID,
+				"RemoveOutbandRuleByID": w.RemoveOutbandRuleByID,
+			})
+			if err != nil {
+				log.Errorf("unable to apply filter: %s", err)
+				continue
+			}
+		}
+	}
+
+	return nil
+}
+
+func (w *WaapRuntimeConfig) RemoveInbandRuleByID(id int) {
+	w.InBandTx.RemoveRuleByIDWithError(id)
+}
+
+func (w *WaapRuntimeConfig) RemoveOutbandRuleByID(id int) {
+	w.OutOfBandTx.RemoveRuleByIDWithError(id)
+}
+
+func (w *WaapRuntimeConfig) ProcessInBandRules(request ParsedRequest) (*corazatypes.Interruption, error) {
+	for _, rule := range w.InBandRules {
+		interrupt, err := rule.Eval(request)
+		if err != nil {
+			return nil, fmt.Errorf("unable to process inband rule %s : %s", rule.GetDisplayName(), err)
+		}
+		if interrupt != nil {
+			return interrupt, nil
+		}
+	}
+	return nil, nil
+}
+
+func (w *WaapRuntimeConfig) ProcessOutOfBandRules(request ParsedRequest) (*corazatypes.Interruption, error) {
+	for _, rule := range w.OutOfBandRules {
+		interrupt, err := rule.Eval(request)
+		if err != nil {
+			return nil, fmt.Errorf("unable to process inband rule %s : %s", rule.GetDisplayName(), err)
+		}
+		if interrupt != nil {
+			return interrupt, nil
+		}
+	}
+	return nil, nil
+}

+ 34 - 0
pkg/waf/waap_rules_collection.go

@@ -0,0 +1,34 @@
+package waf
+
+import corazatypes "github.com/crowdsecurity/coraza/v3/types"
+
+// to be filled w/ seb update
+type WaapCollection struct {
+}
+
+// to be filled w/ seb update
+type WaapCollectionConfig struct {
+	SecLangFilesRules []string `yaml:"seclang_files_rules"`
+	SecLangRules      []string `yaml:"seclang_rules"`
+	MergedRules       []string `yaml:"-"`
+}
+
+func LoadCollection(collection string) (WaapCollection, error) {
+	return WaapCollection{}, nil
+}
+
+func (wcc WaapCollectionConfig) LoadCollection(collection string) (WaapCollection, error) {
+	return WaapCollection{}, nil
+}
+
+func (w WaapCollection) Check() error {
+	return nil
+}
+
+func (w WaapCollection) Eval(req ParsedRequest) (*corazatypes.Interruption, error) {
+	return nil, nil
+}
+
+func (w WaapCollection) GetDisplayName() string {
+	return "rule XX"
+}

+ 0 - 204
pkg/waf/waf.go

@@ -1,204 +0,0 @@
-package waf
-
-import (
-	"fmt"
-	"os"
-	"path/filepath"
-	"strings"
-
-	"github.com/antonmedv/expr"
-	"github.com/antonmedv/expr/vm"
-	"github.com/crowdsecurity/crowdsec/pkg/csconfig"
-	"github.com/crowdsecurity/crowdsec/pkg/cwhub"
-	"github.com/crowdsecurity/crowdsec/pkg/types"
-	log "github.com/sirupsen/logrus"
-	"gopkg.in/yaml.v3"
-)
-
-type Hook struct {
-	Filter     string        `yaml:"filter"`
-	FilterExpr *vm.Program   `yaml:"-"`
-	OnSuccess  string        `yaml:"on_success"`
-	Apply      []string      `yaml:"apply"`
-	ApplyExpr  []*vm.Program `yaml:"-"`
-}
-
-type CompiledHook struct {
-	Filter *vm.Program   `yaml:"-"`
-	Apply  []*vm.Program `yaml:"-"`
-}
-
-/*type WafConfig struct {
-	InbandRules    []WafRule
-	OutOfBandRules []WafRule
-	Datadir        string
-	logger         *log.Entry
-}*/
-
-// This represents one "waf-rule" config
-type WafConfig struct {
-	SecLangFilesRules []string `yaml:"seclang_files_rules"`
-	SecLangRules      []string `yaml:"seclang_rules"`
-	OnLoad            []Hook   `yaml:"on_load"`
-	PreEval           []Hook   `yaml:"pre_eval"`
-	OnMatch           []Hook   `yaml:"on_match"`
-
-	CompiledOnLoad  []CompiledHook `yaml:"-"`
-	CompiledPreEval []CompiledHook `yaml:"-"`
-	CompiledOnMatch []CompiledHook `yaml:"-"`
-
-	MergedRules []string `yaml:"-"`
-	OutOfBand   bool     `yaml:"outofband"`
-}
-
-type WafRuleLoader struct {
-	logger  *log.Entry
-	Datadir string
-}
-
-func buildHook(hook Hook) (CompiledHook, error) {
-	compiledHook := CompiledHook{}
-	if hook.Filter != "" {
-		program, err := expr.Compile(hook.Filter) //FIXME: opts
-		if err != nil {
-			return CompiledHook{}, fmt.Errorf("unable to compile filter %s : %w", hook.Filter, err)
-		}
-		compiledHook.Filter = program
-	}
-	for _, apply := range hook.Apply {
-		program, err := expr.Compile(apply, GetExprWAFOptions(GetEnv())...)
-		if err != nil {
-			return CompiledHook{}, fmt.Errorf("unable to compile apply %s : %w", apply, err)
-		}
-		compiledHook.Apply = append(compiledHook.Apply, program)
-	}
-	return compiledHook, nil
-}
-
-func (w *WafRuleLoader) LoadWafRules() ([]*WafRulesCollection, error) {
-	var wafRulesFiles []string
-	for _, hubWafRuleItem := range cwhub.GetItemMap(cwhub.WAF_RULES) {
-		if hubWafRuleItem.Installed {
-			wafRulesFiles = append(wafRulesFiles, hubWafRuleItem.LocalPath)
-		}
-	}
-
-	if len(wafRulesFiles) == 0 {
-		return nil, fmt.Errorf("no waf rules found in hub")
-	}
-
-	w.logger.Infof("Loading %d waf files", len(wafRulesFiles))
-	wafRulesCollections := []*WafRulesCollection{}
-	for _, wafRulesFile := range wafRulesFiles {
-
-		fileContent, err := os.ReadFile(wafRulesFile)
-		if err != nil {
-			w.logger.Errorf("unable to read file %s : %s", wafRulesFile, err)
-			continue
-		}
-		wafConfig := WafConfig{}
-		err = yaml.Unmarshal(fileContent, &wafConfig)
-		if err != nil {
-			w.logger.Errorf("unable to unmarshal file %s : %s", wafRulesFile, err)
-			continue
-		}
-
-		//spew.Dump(wafConfig)
-
-		collection := &WafRulesCollection{}
-
-		if wafConfig.SecLangFilesRules != nil {
-			for _, rulesFile := range wafConfig.SecLangFilesRules {
-				fullPath := filepath.Join(w.Datadir, rulesFile)
-				c, err := os.ReadFile(fullPath)
-				if err != nil {
-					w.logger.Errorf("unable to read file %s : %s", rulesFile, err)
-					continue
-				}
-				for _, line := range strings.Split(string(c), "\n") {
-					if strings.HasPrefix(line, "#") {
-						continue
-					}
-					if strings.TrimSpace(line) == "" {
-						continue
-					}
-					collection.Rules = append(collection.Rules, WafRule{RawRule: line})
-				}
-			}
-		}
-
-		if wafConfig.SecLangRules != nil {
-			for _, rule := range wafConfig.SecLangRules {
-				collection.Rules = append(collection.Rules, WafRule{RawRule: rule})
-			}
-		}
-
-		//TODO: add our own format
-
-		//compile hooks
-		for _, hook := range wafConfig.OnLoad {
-			compiledHook, err := buildHook(hook)
-			if err != nil {
-				w.logger.Errorf("unable to build on_load hook %s : %s", hook.Filter, err)
-				continue
-			}
-			collection.CompiledOnLoad = append(collection.CompiledOnLoad, compiledHook)
-		}
-
-		for _, hook := range wafConfig.PreEval {
-			compiledHook, err := buildHook(hook)
-			if err != nil {
-				w.logger.Errorf("unable to build pre_eval hook %s : %s", hook.Filter, err)
-				continue
-			}
-			collection.CompiledPreEval = append(collection.CompiledPreEval, compiledHook)
-		}
-
-		for _, hook := range wafConfig.OnMatch {
-			compiledHook, err := buildHook(hook)
-			if err != nil {
-				w.logger.Errorf("unable to build on_match hook %s : %s", hook.Filter, err)
-				continue
-			}
-			collection.CompiledOnMatch = append(collection.CompiledOnMatch, compiledHook)
-		}
-
-		//Run the on_load hooks
-		if len(collection.CompiledOnLoad) > 0 {
-			w.logger.Infof("Running %d on_load hooks", len(collection.CompiledOnLoad))
-			for hookIdx, onLoadHook := range collection.CompiledOnLoad {
-				//Ignore filter for on load ?
-				if onLoadHook.Apply != nil {
-					for exprIdx, applyExpr := range onLoadHook.Apply {
-						_, err := expr.Run(applyExpr, map[string]interface{}{
-							"rules": collection,
-						})
-						if err != nil {
-							w.logger.Errorf("unable to run apply for on_load rule %s : %s", wafConfig.OnLoad[hookIdx].Apply[exprIdx], err)
-							continue
-						}
-					}
-				}
-			}
-		}
-		wafRulesCollections = append(wafRulesCollections, collection)
-	}
-
-	return wafRulesCollections, nil
-}
-
-func NewWafRuleLoader() *WafRuleLoader {
-	//FIXME: find a better way to get the datadir
-	clog := log.New()
-	if err := types.ConfigureLogger(clog); err != nil {
-		//return nil, fmt.Errorf("while configuring datasource logger: %w", err)
-		return nil
-	}
-	logger := clog.WithFields(log.Fields{
-		"type": "waf-config",
-	})
-
-	initWafHelpers()
-
-	return &WafRuleLoader{Datadir: csconfig.DataDir, logger: logger}
-}

+ 0 - 37
pkg/waf/waf_rules_collection.go

@@ -1,37 +0,0 @@
-package waf
-
-import "strings"
-
-type WafRule struct {
-	RawRule string
-	RuleID  string
-	InBand  bool
-}
-
-// This is the "compiled" state of a WafConfig
-type WafRulesCollection struct {
-	Rules           []WafRule
-	CompiledOnLoad  []CompiledHook `yaml:"-"`
-	CompiledPreEval []CompiledHook `yaml:"-"`
-	CompiledOnMatch []CompiledHook `yaml:"-"`
-	OutOfBand       bool
-}
-
-func (w *WafRulesCollection) SetInBand() error {
-	w.OutOfBand = false
-	return nil
-}
-
-func (w *WafRulesCollection) SetOutOfBand() error {
-	w.OutOfBand = true
-	return nil
-}
-
-func (w *WafRulesCollection) String() string {
-	//return strings.Join(w.Rules, "\n")
-	var rules []string
-	for _, rule := range w.Rules {
-		rules = append(rules, rule.RawRule)
-	}
-	return strings.Join(rules, "\n")
-}